Class TestInputTopic<K,V>

java.lang.Object
org.apache.kafka.streams.TestInputTopic<K,V>
Type Parameters:
K - the type of the record key
V - the type of the record value

public class TestInputTopic<K,V> extends Object
TestInputTopic is used to pipe records to topic in TopologyTestDriver. To use TestInputTopic create a new instance via TopologyTestDriver.createInputTopic(String, Serializer, Serializer). In actual test code, you can pipe new record values, keys and values or list of KeyValue pairs. If you have multiple source topics, you need to create a TestInputTopic for each.

Processing messages


     private TestInputTopic<Long, String> inputTopic;
     ...
     inputTopic = testDriver.createInputTopic(INPUT_TOPIC, longSerializer, stringSerializer);
     ...
     inputTopic.pipeInput("Hello");
 
See Also:
  • Method Summary

    Modifier and Type
    Method
    Description
    void
    Advances the internally tracked event time of this input topic.
    void
    pipeInput(K key, V value)
    Send an input record with the given key and value on the topic and then commit the records.
    void
    pipeInput(K key, V value, long timestampMs)
    Send an input record with the given key, value and timestamp on the topic and then commit the records.
    void
    pipeInput(K key, V value, Instant timestamp)
    Send an input record with the given key, value and timestamp on the topic and then commit the records.
    void
    Send an input record with the given record on the topic and then commit the records.
    void
    pipeInput(V value)
    Send an input record with the given value on the topic and then commit the records.
    void
    pipeInput(V value, Instant timestamp)
    Send an input record with the given value and timestamp on the topic and then commit the records.
    void
    Send input records with the given KeyValue list on the topic then commit each record individually.
    void
    pipeKeyValueList(List<KeyValue<K,V>> keyValues, Instant startTimestamp, Duration advance)
    Send input records with the given KeyValue list on the topic then commit each record individually.
    void
    pipeRecordList(List<? extends TestRecord<K,V>> records)
    Send input records with the given KeyValue list on the topic then commit each record individually.
    void
    pipeValueList(List<V> values)
    Send input records with the given value list on the topic then commit each record individually.
    void
    pipeValueList(List<V> values, Instant startTimestamp, Duration advance)
    Send input records with the given value list on the topic then commit each record individually.
     

    Methods inherited from class java.lang.Object

    clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
  • Method Details

    • advanceTime

      public void advanceTime(Duration advance)
      Advances the internally tracked event time of this input topic. Each time a record without explicitly defined timestamp is piped, the current topic event time is used as record timestamp.

      Note: advancing the event time on the input topic, does not advance the tracked stream time in TopologyTestDriver as long as no new input records are piped. Furthermore, it does not advance the wall-clock time of TopologyTestDriver.

      Parameters:
      advance - the duration of time to advance
    • pipeInput

      public void pipeInput(TestRecord<K,V> record)
      Send an input record with the given record on the topic and then commit the records. May auto advance topic time.
      Parameters:
      record - the record to sent
    • pipeInput

      public void pipeInput(V value)
      Send an input record with the given value on the topic and then commit the records. May auto advance topic time.
      Parameters:
      value - the record value
    • pipeInput

      public void pipeInput(K key, V value)
      Send an input record with the given key and value on the topic and then commit the records. May auto advance topic time
      Parameters:
      key - the record key
      value - the record value
    • pipeInput

      public void pipeInput(V value, Instant timestamp)
      Send an input record with the given value and timestamp on the topic and then commit the records. Does not auto advance internally tracked time.
      Parameters:
      value - the record value
      timestamp - the record timestamp
    • pipeInput

      public void pipeInput(K key, V value, long timestampMs)
      Send an input record with the given key, value and timestamp on the topic and then commit the records. Does not auto advance internally tracked time.
      Parameters:
      key - the record key
      value - the record value
      timestampMs - the record timestamp
    • pipeInput

      public void pipeInput(K key, V value, Instant timestamp)
      Send an input record with the given key, value and timestamp on the topic and then commit the records. Does not auto advance internally tracked time.
      Parameters:
      key - the record key
      value - the record value
      timestamp - the record timestamp
    • pipeRecordList

      public void pipeRecordList(List<? extends TestRecord<K,V>> records)
      Send input records with the given KeyValue list on the topic then commit each record individually. The timestamp will be generated based on the constructor provided start time and time will auto advance.
      Parameters:
      records - the list of TestRecord records
    • pipeKeyValueList

      public void pipeKeyValueList(List<KeyValue<K,V>> keyValues)
      Send input records with the given KeyValue list on the topic then commit each record individually. The timestamp will be generated based on the constructor provided start time and time will auto advance based on autoAdvance setting.
      Parameters:
      keyValues - the List of KeyValue records
    • pipeValueList

      public void pipeValueList(List<V> values)
      Send input records with the given value list on the topic then commit each record individually. The timestamp will be generated based on the constructor provided start time and time will auto advance based on autoAdvance setting.
      Parameters:
      values - the List of KeyValue records
    • pipeKeyValueList

      public void pipeKeyValueList(List<KeyValue<K,V>> keyValues, Instant startTimestamp, Duration advance)
      Send input records with the given KeyValue list on the topic then commit each record individually. Does not auto advance internally tracked time.
      Parameters:
      keyValues - the List of KeyValue records
      startTimestamp - the timestamp for the first generated record
      advance - the time difference between two consecutive generated records
    • pipeValueList

      public void pipeValueList(List<V> values, Instant startTimestamp, Duration advance)
      Send input records with the given value list on the topic then commit each record individually. The timestamp will be generated based on the constructor provided start time and time will auto advance based on autoAdvance setting.
      Parameters:
      values - the List of values
      startTimestamp - the timestamp for the first generated record
      advance - the time difference between two consecutive generated records
    • toString

      public String toString()
      Overrides:
      toString in class Object