Package org.apache.kafka.streams.kstream
Class Consumed<K,V>
java.lang.Object
org.apache.kafka.streams.kstream.Consumed<K,V>
- Type Parameters:
K
- type of record keyV
- type of record value
The
Consumed
class is used to define the optional parameters when using StreamsBuilder
to
build instances of KStream
, KTable
, and GlobalKTable
.
For example, you can read a topic as KStream
with a custom timestamp extractor and specify the corresponding
key and value serdes like:
StreamsBuilder builder = new StreamsBuilder();
KStream<String, Long> stream = builder.stream(
"topicName",
Consumed.with(Serdes.String(), Serdes.Long())
.withTimestampExtractor(new LogAndSkipOnInvalidTimestamp()));
Similarly, you can read a topic as KTable
with a custom auto.offset.reset
configuration and force a
state store materialization
to access the content via
interactive queries:
StreamsBuilder builder = new StreamsBuilder();
KTable<Integer, Integer> table = builder.table(
"topicName",
Consumed.with(AutoOffsetReset.LATEST),
Materialized.as("queryable-store-name"));
-
Field Summary
Modifier and TypeFieldDescriptionprotected String
protected Topology.AutoOffsetReset
protected TimestampExtractor
-
Constructor Summary
-
Method Summary
Modifier and TypeMethodDescriptionstatic <K,
V> Consumed<K, V> Create an instance ofConsumed
with provided processor name.boolean
int
hashCode()
static <K,
V> Consumed<K, V> static <K,
V> Consumed<K, V> with
(Serde<K> keySerde, Serde<V> valueSerde, TimestampExtractor timestampExtractor, Topology.AutoOffsetReset resetPolicy) Create an instance ofConsumed
with the supplied arguments.static <K,
V> Consumed<K, V> with
(TimestampExtractor timestampExtractor) Create an instance ofConsumed
with aTimestampExtractor
.static <K,
V> Consumed<K, V> with
(Topology.AutoOffsetReset resetPolicy) Create an instance ofConsumed
with aTopology.AutoOffsetReset
.withKeySerde
(Serde<K> keySerde) Configure the instance ofConsumed
with a processor name.withOffsetResetPolicy
(Topology.AutoOffsetReset resetPolicy) Configure the instance ofConsumed
with aTopology.AutoOffsetReset
.withTimestampExtractor
(TimestampExtractor timestampExtractor) Configure the instance ofConsumed
with aTimestampExtractor
.withValueSerde
(Serde<V> valueSerde)
-
Field Details
-
keySerde
-
valueSerde
-
timestampExtractor
-
resetPolicy
-
processorName
-
-
Constructor Details
-
Method Details
-
with
public static <K,V> Consumed<K,V> with(Serde<K> keySerde, Serde<V> valueSerde, TimestampExtractor timestampExtractor, Topology.AutoOffsetReset resetPolicy) Create an instance ofConsumed
with the supplied arguments.null
values are acceptable.- Type Parameters:
K
- key typeV
- value type- Parameters:
keySerde
- the key serde. Ifnull
the default key serde from config will be usedvalueSerde
- the value serde. Ifnull
the default value serde from config will be usedtimestampExtractor
- the timestamp extractor to used. Ifnull
the default timestamp extractor from config will be usedresetPolicy
- the offset reset policy to be used. Ifnull
the default reset policy from config will be used- Returns:
- a new instance of
Consumed
-
with
- Type Parameters:
K
- key typeV
- value type- Parameters:
keySerde
- the key serde. Ifnull
the default key serde from config will be usedvalueSerde
- the value serde. Ifnull
the default value serde from config will be used- Returns:
- a new instance of
Consumed
-
with
Create an instance ofConsumed
with aTimestampExtractor
.- Type Parameters:
K
- key typeV
- value type- Parameters:
timestampExtractor
- the timestamp extractor to used. Ifnull
the default timestamp extractor from config will be used- Returns:
- a new instance of
Consumed
-
with
Create an instance ofConsumed
with aTopology.AutoOffsetReset
.- Type Parameters:
K
- key typeV
- value type- Parameters:
resetPolicy
- the offset reset policy to be used. Ifnull
the default reset policy from config will be used- Returns:
- a new instance of
Consumed
-
as
Create an instance ofConsumed
with provided processor name.- Type Parameters:
K
- key typeV
- value type- Parameters:
processorName
- the processor name to be used. Ifnull
a default processor name will be generated- Returns:
- a new instance of
Consumed
-
withKeySerde
- Parameters:
keySerde
- the key serde. Ifnull
the default key serde from config will be used- Returns:
- this
-
withValueSerde
- Parameters:
valueSerde
- the value serde. Ifnull
the default value serde from config will be used- Returns:
- this
-
withTimestampExtractor
Configure the instance ofConsumed
with aTimestampExtractor
.- Parameters:
timestampExtractor
- the timestamp extractor to used. Ifnull
the default timestamp extractor from config will be used- Returns:
- this
-
withOffsetResetPolicy
Configure the instance ofConsumed
with aTopology.AutoOffsetReset
.- Parameters:
resetPolicy
- the offset reset policy to be used. Ifnull
the default reset policy from config will be used- Returns:
- this
-
withName
Configure the instance ofConsumed
with a processor name.- Parameters:
processorName
- the processor name to be used. Ifnull
a default processor name will be generated- Returns:
- this
-
equals
-
hashCode
public int hashCode()
-