Class Consumed<K,​V>

java.lang.Object
org.apache.kafka.streams.kstream.Consumed<K,​V>
Type Parameters:
K - type of record key
V - 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 Details

  • Constructor Details

    • Consumed

      protected Consumed​(Consumed<K,​V> consumed)
      Create an instance of Consumed from an existing instance.
      Parameters:
      consumed - the instance of Consumed to copy
  • 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 of Consumed with the supplied arguments. null values are acceptable.
      Type Parameters:
      K - key type
      V - value type
      Parameters:
      keySerde - the key serde. If null the default key serde from config will be used
      valueSerde - the value serde. If null the default value serde from config will be used
      timestampExtractor - the timestamp extractor to used. If null the default timestamp extractor from config will be used
      resetPolicy - the offset reset policy to be used. If null the default reset policy from config will be used
      Returns:
      a new instance of Consumed
    • with

      public static <K,​ V> Consumed<K,​V> with​(Serde<K> keySerde, Serde<V> valueSerde)
      Create an instance of Consumed with key and value Serdes.
      Type Parameters:
      K - key type
      V - value type
      Parameters:
      keySerde - the key serde. If null the default key serde from config will be used
      valueSerde - the value serde. If null the default value serde from config will be used
      Returns:
      a new instance of Consumed
    • with

      public static <K,​ V> Consumed<K,​V> with​(TimestampExtractor timestampExtractor)
      Create an instance of Consumed with a TimestampExtractor.
      Type Parameters:
      K - key type
      V - value type
      Parameters:
      timestampExtractor - the timestamp extractor to used. If null the default timestamp extractor from config will be used
      Returns:
      a new instance of Consumed
    • with

      public static <K,​ V> Consumed<K,​V> with​(Topology.AutoOffsetReset resetPolicy)
      Create an instance of Consumed with a Topology.AutoOffsetReset.
      Type Parameters:
      K - key type
      V - value type
      Parameters:
      resetPolicy - the offset reset policy to be used. If null the default reset policy from config will be used
      Returns:
      a new instance of Consumed
    • as

      public static <K,​ V> Consumed<K,​V> as​(String processorName)
      Create an instance of Consumed with provided processor name.
      Type Parameters:
      K - key type
      V - value type
      Parameters:
      processorName - the processor name to be used. If null a default processor name will be generated
      Returns:
      a new instance of Consumed
    • withKeySerde

      public Consumed<K,​V> withKeySerde​(Serde<K> keySerde)
      Configure the instance of Consumed with a key Serde.
      Parameters:
      keySerde - the key serde. If nullthe default key serde from config will be used
      Returns:
      this
    • withValueSerde

      public Consumed<K,​V> withValueSerde​(Serde<V> valueSerde)
      Configure the instance of Consumed with a value Serde.
      Parameters:
      valueSerde - the value serde. If null the default value serde from config will be used
      Returns:
      this
    • withTimestampExtractor

      public Consumed<K,​V> withTimestampExtractor​(TimestampExtractor timestampExtractor)
      Configure the instance of Consumed with a TimestampExtractor.
      Parameters:
      timestampExtractor - the timestamp extractor to used. If null the default timestamp extractor from config will be used
      Returns:
      this
    • withOffsetResetPolicy

      public Consumed<K,​V> withOffsetResetPolicy​(Topology.AutoOffsetReset resetPolicy)
      Configure the instance of Consumed with a Topology.AutoOffsetReset.
      Parameters:
      resetPolicy - the offset reset policy to be used. If null the default reset policy from config will be used
      Returns:
      this
    • withName

      public Consumed<K,​V> withName​(String processorName)
      Configure the instance of Consumed with a processor name.
      Parameters:
      processorName - the processor name to be used. If null a default processor name will be generated
      Returns:
      this
    • equals

      public boolean equals​(Object o)
      Overrides:
      equals in class Object
    • hashCode

      public int hashCode()
      Overrides:
      hashCode in class Object