K
- type of record keyV
- type of record valuepublic class Consumed<K,V> extends Object
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"));
Modifier and Type | Field and Description |
---|---|
protected Serde<K> |
keySerde |
protected String |
processorName |
protected Topology.AutoOffsetReset |
resetPolicy |
protected TimestampExtractor |
timestampExtractor |
protected Serde<V> |
valueSerde |
Modifier | Constructor and Description |
---|---|
protected |
Consumed(Consumed<K,V> consumed) |
Modifier and Type | Method and Description |
---|---|
static <K,V> Consumed<K,V> |
as(String processorName)
Create an instance of
Consumed with provided processor name. |
boolean |
equals(Object o) |
int |
hashCode() |
static <K,V> Consumed<K,V> |
with(Serde<K> keySerde,
Serde<V> valueSerde)
|
static <K,V> Consumed<K,V> |
with(Serde<K> keySerde,
Serde<V> valueSerde,
TimestampExtractor timestampExtractor,
Topology.AutoOffsetReset resetPolicy)
Create an instance of
Consumed with the supplied arguments. |
static <K,V> Consumed<K,V> |
with(TimestampExtractor timestampExtractor)
Create an instance of
Consumed with a TimestampExtractor . |
static <K,V> Consumed<K,V> |
with(Topology.AutoOffsetReset resetPolicy)
Create an instance of
Consumed with a Topology.AutoOffsetReset . |
Consumed<K,V> |
withKeySerde(Serde<K> keySerde)
|
Consumed<K,V> |
withName(String processorName)
Configure the instance of
Consumed with a processor name. |
Consumed<K,V> |
withOffsetResetPolicy(Topology.AutoOffsetReset resetPolicy)
Configure the instance of
Consumed with a Topology.AutoOffsetReset . |
Consumed<K,V> |
withTimestampExtractor(TimestampExtractor timestampExtractor)
Configure the instance of
Consumed with a TimestampExtractor . |
Consumed<K,V> |
withValueSerde(Serde<V> valueSerde)
|
protected TimestampExtractor timestampExtractor
protected Topology.AutoOffsetReset resetPolicy
protected String processorName
public static <K,V> Consumed<K,V> with(Serde<K> keySerde, Serde<V> valueSerde, TimestampExtractor timestampExtractor, Topology.AutoOffsetReset resetPolicy)
Consumed
with the supplied arguments. null
values are acceptable.K
- key typeV
- value typekeySerde
- the key serde. If null
the default key serde from config will be usedvalueSerde
- the value serde. If null
the default value serde from config will be usedtimestampExtractor
- the timestamp extractor to used. If null
the default timestamp extractor from config will be usedresetPolicy
- the offset reset policy to be used. If null
the default reset policy from config will be usedConsumed
public static <K,V> Consumed<K,V> with(Serde<K> keySerde, Serde<V> valueSerde)
K
- key typeV
- value typekeySerde
- the key serde. If null
the default key serde from config will be usedvalueSerde
- the value serde. If null
the default value serde from config will be usedConsumed
public static <K,V> Consumed<K,V> with(TimestampExtractor timestampExtractor)
Consumed
with a TimestampExtractor
.K
- key typeV
- value typetimestampExtractor
- the timestamp extractor to used. If null
the default timestamp extractor from config will be usedConsumed
public static <K,V> Consumed<K,V> with(Topology.AutoOffsetReset resetPolicy)
Consumed
with a Topology.AutoOffsetReset
.K
- key typeV
- value typeresetPolicy
- the offset reset policy to be used. If null
the default reset policy from config will be usedConsumed
public static <K,V> Consumed<K,V> as(String processorName)
Consumed
with provided processor name.K
- key typeV
- value typeprocessorName
- the processor name to be used. If null
a default processor name will be generatedConsumed
public Consumed<K,V> withKeySerde(Serde<K> keySerde)
keySerde
- the key serde. If null
the default key serde from config will be usedConsumed
public Consumed<K,V> withValueSerde(Serde<V> valueSerde)
valueSerde
- the value serde. If null
the default value serde from config will be usedConsumed
public Consumed<K,V> withTimestampExtractor(TimestampExtractor timestampExtractor)
Consumed
with a TimestampExtractor
.timestampExtractor
- the timestamp extractor to used. If null
the default timestamp extractor from config will be usedConsumed
public Consumed<K,V> withOffsetResetPolicy(Topology.AutoOffsetReset resetPolicy)
Consumed
with a Topology.AutoOffsetReset
.resetPolicy
- the offset reset policy to be used. If null
the default reset policy from config will be usedConsumed
public Consumed<K,V> withName(String processorName)
Consumed
with a processor name.processorName
- the processor name to be used. If null
a default processor name will be generatedConsumed