K
- Type of primary keysV
- Type of value changes@InterfaceStability.Unstable
public interface KGroupedTable<K,V>
KGroupedTable
is an abstraction of a grouped changelog stream from a primary-keyed table,
usually on a different grouping key than the original primary key.
It is an intermediate representation after a re-grouping of a KTable
before an aggregation is applied
to the new partitions resulting in a new KTable
.
Modifier and Type | Method and Description |
---|---|
<T> KTable<K,T> |
aggregate(Initializer<T> initializer,
Aggregator<K,V,T> adder,
Aggregator<K,V,T> subtractor,
Serde<T> aggValueSerde,
String storeName)
Aggregate updating values of this stream by the selected key into a new instance of
KTable . |
<T> KTable<K,T> |
aggregate(Initializer<T> initializer,
Aggregator<K,V,T> adder,
Aggregator<K,V,T> subtractor,
String storeName)
Aggregate updating values of this stream by the selected key into a new instance of
KTable
using default serializers and deserializers. |
KTable<K,Long> |
count(String storeName)
Count number of records of this stream by the selected key into a new instance of
KTable . |
KTable<K,V> |
reduce(Reducer<V> adder,
Reducer<V> subtractor,
String storeName)
Combine updating values of this stream by the selected key into a new instance of
KTable . |
KTable<K,V> reduce(Reducer<V> adder, Reducer<V> subtractor, String storeName)
KTable
.
The resulting KTable
will be materialized in a local state
store with the given store name. Also a changelog topic named "${applicationId}-${storeName}-changelog"
will be automatically created in Kafka for failure recovery, where "applicationID"
is specified by the user in StreamsConfig
.adder
- the instance of Reducer
for additionsubtractor
- the instance of Reducer
for subtractionstoreName
- the name of the underlying KTable
state storeKTable
with the same key and value types as this KGroupedTable
,
containing aggregated values for each key<T> KTable<K,T> aggregate(Initializer<T> initializer, Aggregator<K,V,T> adder, Aggregator<K,V,T> subtractor, Serde<T> aggValueSerde, String storeName)
KTable
.
The resulting KTable
will be materialized in a local state
store with the given store name. Also a changelog topic named "${applicationId}-${storeName}-changelog"
will be automatically created in Kafka for failure recovery, where "applicationID"
is specified by the user in StreamsConfig
.T
- the value type of the aggregated KTable
initializer
- the instance of Initializer
adder
- the instance of Aggregator
for additionsubtractor
- the instance of Aggregator
for subtractionaggValueSerde
- value serdes for materializing the aggregated table,
if not specified the default serdes defined in the configs will be usedstoreName
- the name of the underlying KTable
state storeKTable
with same key and aggregated value type T
,
containing aggregated values for each key<T> KTable<K,T> aggregate(Initializer<T> initializer, Aggregator<K,V,T> adder, Aggregator<K,V,T> subtractor, String storeName)
KTable
using default serializers and deserializers.
The resulting KTable
will be materialized in a local state
store with the given store name. Also a changelog topic named "${applicationId}-${storeName}-changelog"
will be automatically created in Kafka for failure recovery, where "applicationID"
is specified by the user in StreamsConfig
.T
- the value type of the aggregated KTable
initializer
- the instance of Initializer
adder
- the instance of Aggregator
for additionsubtractor
- the instance of Aggregator
for subtractionstoreName
- the name of the underlying KTable
state storeKTable
with same key and aggregated value type T
,
containing aggregated values for each keyKTable<K,Long> count(String storeName)
KTable
.
The resulting KTable
will be materialized in a local state
store with the given store name. Also a changelog topic named "${applicationId}-${storeName}-changelog"
will be automatically created in Kafka for failure recovery, where "applicationID"
is specified by the user in StreamsConfig
.storeName
- the name of the underlying KTable
state storeKTable
with same key and Long
value type as this KGroupedTable
,
containing the number of values for each key