Class TestOutputTopic<K,V>

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

public class TestOutputTopic<K,V> extends Object
TestOutputTopic is used to read records from a topic in TopologyTestDriver. To use TestOutputTopic create a new instance via TopologyTestDriver.createOutputTopic(String, Deserializer, Deserializer). In actual test code, you can read record values, keys, KeyValue or TestRecord If you have multiple source topics, you need to create a TestOutputTopic for each.

If you need to test key, value and headers, use readRecord() methods. Using readKeyValue() you get a KeyValue pair, and thus, don't get access to the record's timestamp or headers. Similarly using readValue() you only get the value of a record.

Processing records


     private TestOutputTopic<String, Long> outputTopic;
      ...
     outputTopic = testDriver.createOutputTopic(OUTPUT_TOPIC, stringDeserializer, longDeserializer);
     ...
     assertThat(outputTopic.readValue()).isEqual(1);
 
See Also:
  • Method Details

    • readValue

      public V readValue()
      Read one record from the output topic and return record's value.
      Returns:
      Next value for output topic.
    • readKeyValue

      public KeyValue<K,V> readKeyValue()
      Read one record from the output topic and return its key and value as pair.
      Returns:
      Next output as KeyValue.
    • readRecord

      public TestRecord<K,V> readRecord()
      Read one Record from output topic.
      Returns:
      Next output as TestRecord.
    • readRecordsToList

      public List<TestRecord<K,V>> readRecordsToList()
      Read output to List. This method can be used if the result is considered a stream. If the result is considered a table, the list will contain all updated, ie, a key might be contained multiple times. If you are only interested in the last table update (ie, the final table state), you can use readKeyValuesToMap() instead.
      Returns:
      List of output.
    • readKeyValuesToMap

      public Map<K,V> readKeyValuesToMap()
      Read output to map. This method can be used if the result is considered a table, when you are only interested in the last table update (ie, the final table state). If the result is considered a stream, you can use readRecordsToList() instead. The list will contain all updated, ie, a key might be contained multiple times. If the last update to a key is a delete/tombstone, the key will still be in the map (with null-value).
      Returns:
      Map of output by key.
    • readKeyValuesToList

      public List<KeyValue<K,V>> readKeyValuesToList()
      Read all KeyValues from topic to List.
      Returns:
      List of output KeyValues.
    • readValuesToList

      public List<V> readValuesToList()
      Read all values from topic to List.
      Returns:
      List of output values.
    • getQueueSize

      public final long getQueueSize()
      Get size of unread record in the topic queue.
      Returns:
      size of topic queue.
    • isEmpty

      public final boolean isEmpty()
      Verify if the topic queue is empty.
      Returns:
      true if no more record in the topic queue.
    • toString

      public String toString()
      Overrides:
      toString in class Object