Interface GlobalKTable<K,​V>

Type Parameters:
K - Type of primary keys
V - Type of value changes

public interface GlobalKTable<K,​V>
GlobalKTable is an abstraction of a changelog stream from a primary-keyed table. Each record in this changelog stream is an update on the primary-keyed table with the record key as the primary key.

GlobalKTable can only be used as right-hand side input for stream-table joins.

In contrast to a KTable that is partitioned over all KafkaStreams instances, a GlobalKTable is fully replicated per KafkaStreams instance. Every partition of the underlying topic is consumed by each GlobalKTable, such that the full set of data is available in every KafkaStreams instance. This provides the ability to perform joins with KStream without having to repartition the input stream. All joins with the GlobalKTable require that a KeyValueMapper is provided that can map from the KeyValue of the left hand side KStream to the key of the right hand side GlobalKTable.

A GlobalKTable is created via a StreamsBuilder. For example:


 builder.globalTable("topic-name", "queryable-store-name");
 
all GlobalKTables are backed by a ReadOnlyKeyValueStore and are therefore queryable via the interactive queries API. For example:

 final GlobalKTable globalOne = builder.globalTable("g1", "g1-store");
 final GlobalKTable globalTwo = builder.globalTable("g2", "g2-store");
 ...
 final KafkaStreams streams = ...;
 streams.start()
 ...
 ReadOnlyKeyValueStore view = streams.store("g1-store", QueryableStoreTypes.timestampedKeyValueStore());
 view.get(key); // can be done on any key, as all keys are present
Note that in contrast to KTable a GlobalKTable's state holds a full copy of the underlying topic, thus all keys can be queried locally.

Records from the source topic that have null keys are dropped.

See Also:
KTable, StreamsBuilder.globalTable(String), KStream.join(GlobalKTable, KeyValueMapper, ValueJoiner), KStream.leftJoin(GlobalKTable, KeyValueMapper, ValueJoiner)
  • Method Summary

    Modifier and Type Method Description
    String queryableStoreName()
    Get the name of the local state store that can be used to query this GlobalKTable.
  • Method Details

    • queryableStoreName

      String queryableStoreName()
      Get the name of the local state store that can be used to query this GlobalKTable.
      Returns:
      the underlying state store name, or null if this GlobalKTable cannot be queried.