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
public class Consumed<K,V> extends Object
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
Fields Modifier and Type Field Description protected Serde<K>keySerdeprotected StringprocessorNameprotected Topology.AutoOffsetResetresetPolicyprotected TimestampExtractortimestampExtractorprotected Serde<V>valueSerde -
Constructor Summary
-
Method Summary
Modifier and Type Method Description static <K, V> Consumed<K,V>as(String processorName)Create an instance ofConsumedwith provided processor name.booleanequals(Object o)inthashCode()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 ofConsumedwith the supplied arguments.static <K, V> Consumed<K,V>with(TimestampExtractor timestampExtractor)Create an instance ofConsumedwith aTimestampExtractor.static <K, V> Consumed<K,V>with(Topology.AutoOffsetReset resetPolicy)Create an instance ofConsumedwith aTopology.AutoOffsetReset.Consumed<K,V>withKeySerde(Serde<K> keySerde)Consumed<K,V>withName(String processorName)Configure the instance ofConsumedwith a processor name.Consumed<K,V>withOffsetResetPolicy(Topology.AutoOffsetReset resetPolicy)Configure the instance ofConsumedwith aTopology.AutoOffsetReset.Consumed<K,V>withTimestampExtractor(TimestampExtractor timestampExtractor)Configure the instance ofConsumedwith aTimestampExtractor.Consumed<K,V>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 ofConsumedwith the supplied arguments.nullvalues are acceptable.- Type Parameters:
K- key typeV- value type- Parameters:
keySerde- the key serde. Ifnullthe default key serde from config will be usedvalueSerde- the value serde. Ifnullthe default value serde from config will be usedtimestampExtractor- the timestamp extractor to used. Ifnullthe default timestamp extractor from config will be usedresetPolicy- the offset reset policy to be used. Ifnullthe 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. Ifnullthe default key serde from config will be usedvalueSerde- the value serde. Ifnullthe default value serde from config will be used- Returns:
- a new instance of
Consumed
-
with
Create an instance ofConsumedwith aTimestampExtractor.- Type Parameters:
K- key typeV- value type- Parameters:
timestampExtractor- the timestamp extractor to used. Ifnullthe default timestamp extractor from config will be used- Returns:
- a new instance of
Consumed
-
with
Create an instance ofConsumedwith aTopology.AutoOffsetReset.- Type Parameters:
K- key typeV- value type- Parameters:
resetPolicy- the offset reset policy to be used. Ifnullthe default reset policy from config will be used- Returns:
- a new instance of
Consumed
-
as
Create an instance ofConsumedwith provided processor name.- Type Parameters:
K- key typeV- value type- Parameters:
processorName- the processor name to be used. Ifnulla default processor name will be generated- Returns:
- a new instance of
Consumed
-
withKeySerde
- Parameters:
keySerde- the key serde. Ifnullthe default key serde from config will be used- Returns:
- this
-
withValueSerde
- Parameters:
valueSerde- the value serde. Ifnullthe default value serde from config will be used- Returns:
- this
-
withTimestampExtractor
Configure the instance ofConsumedwith aTimestampExtractor.- Parameters:
timestampExtractor- the timestamp extractor to used. Ifnullthe default timestamp extractor from config will be used- Returns:
- this
-
withOffsetResetPolicy
Configure the instance ofConsumedwith aTopology.AutoOffsetReset.- Parameters:
resetPolicy- the offset reset policy to be used. Ifnullthe default reset policy from config will be used- Returns:
- this
-
withName
Configure the instance ofConsumedwith a processor name.- Parameters:
processorName- the processor name to be used. Ifnulla default processor name will be generated- Returns:
- this
-
equals
-
hashCode
public int hashCode()
-