Streams DSL
The Kafka Streams DSL (Domain Specific Language) is built on top of the Streams Processor API. It is the recommended for most users, especially beginners. Most data processing operations can be expressed in just a few lines of DSL code.
Table of Contents
- Overview
- Creating source streams from Kafka
- Transform a stream
- Naming Operators in a Streams DSL application
- Controlling KTable update rate
- Using timestamp-based semantics for table processors
- Writing streams back to Kafka
- Testing a Streams application
- Kafka Streams DSL for Scala
Overview
In comparison to the Processor API, only the DSL supports:
- Built-in abstractions for streams and tables in the form of KStream, KTable, and GlobalKTable. Having first-class support for streams and tables is crucial because, in practice, most use cases require not just either streams or databases/tables, but a combination of both. For example, if your use case is to create a customer 360-degree view that is updated in real-time, what your application will be doing is transforming many input streams of customer-related events into an output table that contains a continuously updated 360-degree view of your customers.
- Declarative, functional programming style with
stateless transformations (e.g.
map
andfilter
) as well as stateful transformations such as aggregations (e.g.count
andreduce
), joins (e.g.leftJoin
), and windowing (e.g. session windows).
With the DSL, you can define processor topologies (i.e., the logical processing plan) in your application. The steps to accomplish this are:
- Specify one or more input streams that are read from Kafka topics.
- Compose transformations on these streams.
- Write the resulting output streams back to Kafka topics, or expose the processing results of your application directly to other applications through interactive queries (e.g., via a REST API).
After the application is run, the defined processor topologies are continuously executed (i.e., the processing plan is put into action). A step-by-step guide for writing a stream processing application using the DSL is provided below.
For a complete list of available API functionality, see also the Streams API docs.
KStream
Only the Kafka Streams DSL has the notion of a KStream
.
A KStream is an abstraction of a record stream, where each data record represents a self-contained datum in the unbounded data set. Using the table analogy, data records in a record stream are always interpreted as an "INSERT" -- think: adding more entries to an append-only ledger -- because no record replaces an existing row with the same key. Examples are a credit card transaction, a page view event, or a server log entry.
To illustrate, let's imagine the following two data records are being sent to the stream:
("alice", 1) --> ("alice", 3)
If your stream processing application were to sum the values per user, it would return 4
for alice
. Why? Because the second data record would not be considered an update of the previous record. Compare this behavior of KStream to KTable
below,
which would return 3
for alice
.
KTable
Only the Kafka Streams DSL has the notion of a KTable
.
A KTable is an abstraction of a changelog stream, where each data record represents an update. More precisely, the value in a data record is interpreted as an "UPDATE" of the last value for the same record key, if any (if a corresponding key doesn't exist yet, the update will be considered an INSERT). Using the table analogy, a data record in a changelog stream is interpreted as an UPSERT aka INSERT/UPDATE because any existing row with the same key is overwritten. Also, null
values are interpreted in a special way: a record with a null
value represents a "DELETE" or tombstone for the record's key.
To illustrate, let's imagine the following two data records are being sent to the stream:
("alice", 1) --> ("alice", 3)
If your stream processing application were to sum the values per user, it would return 3
for alice
. Why? Because the second data record would be considered an update of the previous record.
Effects of Kafka's log compaction: Another way of thinking about KStream and KTable is as follows: If you were to store a KTable into a Kafka topic, you'd probably want to enable Kafka's log compaction feature, e.g. to save storage space.
However, it would not be safe to enable log compaction in the case of a KStream because, as soon as log compaction would begin purging older data records of the same key, it would break the semantics of the data. To pick up the illustration example again, you'd suddenly get a 3
for alice
instead of a 4
because log compaction would have removed the ("alice", 1)
data record. Hence log compaction is perfectly safe for a KTable (changelog stream) but it is a mistake for a KStream (record stream).
We have already seen an example of a changelog stream in the section streams and tables. Another example are change data capture (CDC) records in the changelog of a relational database, representing which row in a database table was inserted, updated, or deleted.
KTable also provides an ability to look up current values of data records by keys. This table-lookup functionality is available through join operations (see also Joining in the Developer Guide) as well as through Interactive Queries.
GlobalKTable
Only the Kafka Streams DSL has the notion of a GlobalKTable.
Like a KTable, a GlobalKTable is an abstraction of a changelog stream, where each data record represents an update.
A GlobalKTable differs from a KTable in the data that they are being populated with, i.e. which data from the underlying Kafka topic is being read into the respective table. Slightly simplified, imagine you have an input topic with 5 partitions. In your application, you want to read this topic into a table. Also, you want to run your application across 5 application instances for maximum parallelism.
- If you read the input topic into a KTable, then the "local" KTable instance of each application instance will be populated with data from only 1 partition of the topic's 5 partitions.
- If you read the input topic into a GlobalKTable, then the local GlobalKTable instance of each application instance will be populated with data from all partitions of the topic.
GlobalKTable provides the ability to look up current values of data records by keys. This table-lookup functionality is available through join operations
.
Note that a GlobalKTable has no notion of time in contrast to a KTable.
Benefits of global tables:
- More convenient and/or efficient joins: Notably, global tables allow you to perform star joins, they support "foreign-key" lookups (i.e., you can lookup data in the table not just by record key, but also by data in the record values), and they are more efficient when chaining multiple joins. Also, when joining against a global table, the input data does not need to be co-partitioned.
- Can be used to "broadcast" information to all the running instances of your application.
Downsides of global tables:
- Increased local storage consumption compared to the (partitioned) KTable because the entire topic is tracked.
- Increased network and Kafka broker load compared to the (partitioned) KTable because the entire topic is read.
Creating source streams from Kafka
You can easily read data from Kafka topics into your application. The following operations are supported.
Reading from Kafka | Description |
---|---|
Stream
|
Creates a KStream from the specified Kafka input topics and interprets the data
as a record stream.
A In the case of a KStream, the local KStream instance of every application instance will be populated with data from only a subset of the partitions of the input topic. Collectively, across all application instances, all input topic partitions are read and processed.
If you do not specify Serdes explicitly, the default Serdes from the configuration are used. You must specify Serdes explicitly if the key or value types of the records in the Kafka input topics do not match the configured default Serdes. For information about configuring default Serdes, available Serdes, and implementing your own custom Serdes see Data Types and Serialization. Several variants of |
Table
|
Reads the specified Kafka input topic into a KTable. The topic is
interpreted as a changelog stream, where records with the same key are interpreted as UPSERT aka INSERT/UPDATE
(when the record value is not In the case of a KTable, the local KTable instance of every application instance will be populated with data from only a subset of the partitions of the input topic. Collectively, across all application instances, all input topic partitions are read and processed. You must provide a name for the table (more precisely, for the internal state store that backs the table). This is required for supporting interactive queries against the table. When a name is not provided the table will not be queryable and an internal name will be provided for the state store. If you do not specify Serdes explicitly, the default Serdes from the configuration are used. You must specify Serdes explicitly if the key or value types of the records in the Kafka input topics do not match the configured default Serdes. For information about configuring default Serdes, available Serdes, and implementing your own custom Serdes see Data Types and Serialization. Several variants of |
Global Table
|
Reads the specified Kafka input topic into a GlobalKTable. The topic is
interpreted as a changelog stream, where records with the same key are interpreted as UPSERT aka INSERT/UPDATE
(when the record value is not In the case of a GlobalKTable, the local GlobalKTable instance of every application instance will be populated with data from all the partitions of the input topic. You must provide a name for the table (more precisely, for the internal state store that backs the table). This is required for supporting interactive queries against the table. When a name is not provided the table will not be queryable and an internal name will be provided for the state store.
You must specify Serdes explicitly if the key or value types of the records in the Kafka input topics do not match the configured default Serdes. For information about configuring default Serdes, available Serdes, and implementing your own custom Serdes see Data Types and Serialization. Several variants of |
Transform a stream
The KStream and KTable interfaces support a variety of transformation operations. Each of these operations can be translated into one or more connected processors into the underlying processor topology. Since KStream and KTable are strongly typed, all of these transformation operations are defined as generic functions where users could specify the input and output data types.
Some KStream transformations may generate one or more KStream objects, for example:
- filter
and map
on a KStream will generate another KStream
- split
on KStream can generate multiple KStreams
Some others may generate a KTable object, for example an aggregation of a KStream also yields a KTable. This allows Kafka Streams to continuously update the computed value upon arrivals of out-of-order records after it has already been produced to the downstream transformation operators.
All KTable transformation operations can only generate another KTable. However, the Kafka Streams DSL does provide a special function that converts a KTable representation into a KStream. All of these transformation methods can be chained together to compose a complex processor topology.
These transformation operations are described in the following subsections:
Stateless transformations
Stateless transformations do not require state for processing and they do not require a state store associated with
the stream processor. Kafka 0.11.0 and later allows you to materialize the result from a stateless KTable
transformation. This allows the result to be queried through interactive queries. To materialize a KTable
, each of the below stateless operations can be augmented with an optional queryableStoreName
argument.
Transformation | Description |
---|---|
Branch
|
Branch (or split) a Predicates are evaluated in order. A record is placed to one and only one output stream on the first match: if the n-th predicate evaluates to true, the record is placed to n-th stream. If a record does not match any predicates, it will be routed to the default branch, or dropped if no default branch is created. Branching is useful, for example, to route records to different downstream topics.
|
Filter
|
Evaluates a boolean function for each element and retains those for which the function returns true. (KStream details, KTable details)
|
Inverse Filter
|
Evaluates a boolean function for each element and drops those for which the function returns true. (KStream details, KTable details)
|
FlatMap
|
Takes one record and produces zero, one, or more records. You can modify the record keys and values, including their types. (details) Marks the stream for data re-partitioning:
Applying a grouping or a join after
|
FlatMapValues
|
Takes one record and produces zero, one, or more records, while retaining the key of the original record. You can modify the record values and the value type. (details)
|
Foreach
|
Terminal operation. Performs a stateless action on each record. (details) You would use Note on processing guarantees: Any side effects of an action (such as writing to external systems) are not trackable by Kafka, which means they will typically not benefit from Kafka’s processing guarantees.
|
GroupByKey
|
Groups the records by the existing key. (details) Grouping is a prerequisite for aggregating a stream or a table and ensures that data is properly partitioned (“keyed”) for subsequent operations. When to set explicit Serdes:
Variants of Note Grouping vs. Windowing: A related operation is windowing, which lets you control how to “sub-group” the grouped records of the same key into so-called windows for stateful operations such as windowed aggregations or windowed joins. Causes data re-partitioning if and only if the stream was marked for re-partitioning.
|
GroupBy
|
Groups the records by a new key, which may be of a different key type.
When grouping a table, you may also specify a new value and value type.
Grouping is a prerequisite for aggregating a stream or a table and ensures that data is properly partitioned (“keyed”) for subsequent operations. When to set explicit Serdes:
Variants of Note Grouping vs. Windowing: A related operation is windowing, which lets you control how to “sub-group” the grouped records of the same key into so-called windows for stateful operations such as windowed aggregations or windowed joins. Always causes data re-partitioning:
|
Cogroup
|
Cogrouping allows to aggregate multiple input streams in a single operation.
The different (already grouped) input streams must have the same key type and may have different values types.
KGroupedStream#cogroup() creates a new cogrouped stream with a single input stream, while CogroupedKStream#cogroup() adds a grouped stream to an existing cogrouped stream.
A Cogroup does not cause a repartition as it has the prerequisite that the input streams are grouped. In the process of creating these groups they will have already been repartitioned if the stream was already marked for repartitioning.
|
Map
|
Takes one record and produces one record. You can modify the record key and value, including their types. (details) Marks the stream for data re-partitioning:
Applying a grouping or a join after
|
Map (values only)
|
Takes one record and produces one record, while retaining the key of the original record. You can modify the record value and the value type. (KStream details, KTable details)
|
Merge
|
Merges records of two streams into one larger stream. (details) There is no ordering guarantee between records from different streams in the merged stream. Relative order is preserved within each input stream though (ie, records within the same input stream are processed in order)
|
Peek
|
Performs a stateless action on each record, and returns an unchanged stream. (details) You would use
Note on processing guarantees: Any side effects of an action (such as writing to external systems) are not trackable by Kafka, which means they will typically not benefit from Kafka’s processing guarantees.
|
|
Terminal operation. Prints the records to Calling
|
SelectKey
|
Assigns a new key – possibly of a new key type – to each record. (details) Calling Marks the stream for data re-partitioning:
Applying a grouping or a join after
|
Table to Stream
|
Get the changelog stream of this table. (details)
|
Stream to Table
|
Convert an event stream into a table, or say a changelog stream. (details)
|
Repartition
|
Manually trigger repartitioning of the stream with desired number of partitions. (details) repartition() is similar to through() however Kafka Streams will manage the topic for you.
Generated topic is treated as internal topic, as a result data will be purged automatically as any other internal repartition topic.
In addition, you can specify the desired number of partitions, which allows to easily scale in/out downstream sub-topologies.
repartition() operation always triggers repartitioning of the stream, as a result it can be used with embedded Processor API methods (like transform() et al.) that do not trigger auto repartitioning when key changing operation is performed beforehand.
|
Stateful transformations
Stateful transformations depend on state for processing inputs and producing outputs and require a state store associated with the stream processor. For example, in aggregating operations, a windowing state store is used to collect the latest aggregation results per window. In join operations, a windowing state store is used to collect all of the records received so far within the defined window boundary.
Note: Following store types are used regardless of the possibly specified type (via the parameter materialized
):
- non-windowed aggregations and non-windowed KTables use TimestampedKeyValueStores
or VersionedKeyValueStores,
depending on whether the parameter
materialized
is versioned - time-windowed aggregations and KStream-KStream joins use TimestampedWindowStores
- session windowed aggregations use SessionStores (there is no timestamped session store as of now)
Note, that state stores are fault-tolerant. In case of failure, Kafka Streams guarantees to fully restore all state stores prior to resuming the processing. See Fault Tolerance for further information.
Available stateful transformations in the DSL include:
- Aggregating
- Joining
- Windowing (as part of aggregations and joins)
- Applying custom processors and transformers, which may be stateful, for Processor API integration
The following diagram shows their relationships:
Here is an example of a stateful application: the WordCount algorithm.
WordCount example in Java 8+, using lambda expressions:
// Assume the record values represent lines of text. For the sake of this example, you can ignore
// whatever may be stored in the record keys.
KStream<String, String> textLines = ...;
KStream<String, Long> wordCounts = textLines
// Split each text line, by whitespace, into words. The text lines are the record
// values, i.e. you can ignore whatever data is in the record keys and thus invoke
// `flatMapValues` instead of the more generic `flatMap`.
.flatMapValues(value -> Arrays.asList(value.toLowerCase().split("\\W+")))
// Group the stream by word to ensure the key of the record is the word.
.groupBy((key, word) -> word)
// Count the occurrences of each word (record key).
//
// This will change the stream type from `KGroupedStream<String, String>` to
// `KTable<String, Long>` (word -> count).
.count()
// Convert the `KTable<String, Long>` into a `KStream<String, Long>`.
.toStream();
Aggregating
After records are grouped by key via groupByKey
or
groupBy
– and thus represented as either a KGroupedStream
or a KGroupedTable
, they can be aggregated
via an operation such as reduce
. Aggregations are key-based operations, which means that they always operate over records
(notably record values) of the same key.
You can perform aggregations on windowed or non-windowed data.
Transformation | Description |
---|---|
Aggregate
|
Rolling aggregation. Aggregates the values of (non-windowed) records by the grouped key or cogrouped.
Aggregating is a generalization of When aggregating a grouped stream, you must provide an initializer (e.g., When aggregating a cogrouped stream, the actual aggregators are provided for each input stream in the prior Several variants of
Detailed behavior of
Detailed behavior of
See the example at the bottom of this section for a visualization of the aggregation semantics. |
Aggregate (windowed)
|
Windowed aggregation.
Aggregates the values of records, per window, by the grouped key.
Aggregating is a generalization of You must provide an initializer (e.g., The windowed Several variants of
Detailed behavior:
See the example at the bottom of this section for a visualization of the aggregation semantics. |
Count
|
Rolling aggregation. Counts the number of records by the grouped key. (KGroupedStream details, KGroupedTable details) Several variants of
Detailed behavior for
Detailed behavior for
|
Count (windowed)
|
Windowed aggregation. Counts the number of records, per window, by the grouped key. (TimeWindowedKStream details, SessionWindowedKStream details) The windowed Several variants of
Detailed behavior:
|
Reduce
|
Rolling aggregation. Combines the values of (non-windowed) records by the grouped key.
The current record value is combined with the last reduced value, and a new reduced value is returned.
The result value type cannot be changed, unlike When reducing a grouped stream, you must provide an “adder” reducer (e.g., Several variants of
Detailed behavior for
Detailed behavior for
See the example at the bottom of this section for a visualization of the aggregation semantics. |
Reduce (windowed)
|
Windowed aggregation.
Combines the values of records, per window, by the grouped key.
The current record value is combined with the last reduced value, and a new reduced value is returned.
Records with The windowed Several variants of
Detailed behavior:
See the example at the bottom of this section for a visualization of the aggregation semantics. |
Example of semantics for stream aggregations:
A KGroupedStream
→ KTable
example is shown below. The streams and the table are initially empty. Bold
font is used in the column for “KTable aggregated
” to highlight changed state. An entry such as (hello, 1)
denotes a
record with key hello
and value 1
. To improve the readability of the semantics table you can assume that all records
are processed in timestamp order.
// Key: word, value: count
KStream<String, Integer> wordCounts = ...;
KGroupedStream<String, Integer> groupedStream = wordCounts
.groupByKey(Grouped.with(Serdes.String(), Serdes.Integer()));
KTable<String, Integer> aggregated = groupedStream.aggregate(
() -> 0, /* initializer */
(aggKey, newValue, aggValue) -> aggValue + newValue, /* adder */
Materialized.<String, Long, KeyValueStore<Bytes, byte[]>as("aggregated-stream-store" /* state store name */)
.withKeySerde(Serdes.String()) /* key serde */
.withValueSerde(Serdes.Integer()); /* serde for aggregate value */
Note
Impact of record caches:
For illustration purposes, the column “KTable aggregated
” below shows the table’s state changes over time in a
very granular way. In practice, you would observe state changes in such a granular way only when
record caches are disabled (default: enabled).
When record caches are enabled, what might happen for example is that the output results of the rows with timestamps
4 and 5 would be compacted, and there would only be
a single state update for the key kafka
in the KTable (here: from (kafka 1)
directly to (kafka, 3)
.
Typically, you should only disable record caches for testing or debugging purposes – under normal circumstances it
is better to leave record caches enabled.
KStream wordCounts |
KGroupedStream groupedStream |
KTable aggregated |
|||
---|---|---|---|---|---|
Timestamp | Input record | Grouping | Initializer | Adder | State |
1 | (hello, 1) | (hello, 1) | 0 (for hello) | (hello, 0 + 1) | (hello, 1)
|
2 | (kafka, 1) | (kafka, 1) | 0 (for kafka) | (kafka, 0 + 1) | (hello, 1)
(kafka, 1)
|
3 | (streams, 1) | (streams, 1) | 0 (for streams) | (streams, 0 + 1) | (hello, 1)
(kafka, 1)
(streams, 1)
|
4 | (kafka, 1) | (kafka, 1) | (kafka, 1 + 1) | (hello, 1)
(kafka, 2)
(streams, 1)
|
|
5 | (kafka, 1) | (kafka, 1) | (kafka, 2 + 1) | (hello, 1)
(kafka, 3)
(streams, 1)
|
|
6 | (streams, 1) | (streams, 1) | (streams, 1 + 1) | (hello, 1)
(kafka, 3)
(streams, 2)
|
Example of semantics for table aggregations:
A KGroupedTable
→ KTable
example is shown below. The tables are initially empty. Bold font is used in the column
for “KTable aggregated
” to highlight changed state. An entry such as (hello, 1)
denotes a record with key
hello
and value 1
. To improve the readability of the semantics table you can assume that all records are processed
in timestamp order.
// Key: username, value: user region (abbreviated to "E" for "Europe", "A" for "Asia")
KTable<String, String> userProfiles = ...;
// Re-group `userProfiles`. Don't read too much into what the grouping does:
// its prime purpose in this example is to show the *effects* of the grouping
// in the subsequent aggregation.
KGroupedTable<String, Integer> groupedTable = userProfiles
.groupBy((user, region) -> KeyValue.pair(region, user.length()), Serdes.String(), Serdes.Integer());
KTable<String, Integer> aggregated = groupedTable.aggregate(
() -> 0, /* initializer */
(aggKey, newValue, aggValue) -> aggValue + newValue, /* adder */
(aggKey, oldValue, aggValue) -> aggValue - oldValue, /* subtractor */
Materialized.<String, Long, KeyValueStore<Bytes, byte[]>as("aggregated-table-store" /* state store name */)
.withKeySerde(Serdes.String()) /* key serde */
.withValueSerde(Serdes.Integer()); /* serde for aggregate value */
Note
Impact of record caches:
For illustration purposes, the column “KTable aggregated
” below shows the table’s state changes over time in a
very granular way. In practice, you would observe state changes in such a granular way only when
record caches are disabled (default: enabled).
When record caches are enabled, what might happen for example is that the output results of the rows with timestamps
4 and 5 would be compacted, and there would only be
a single state update for the key kafka
in the KTable (here: from (kafka 1)
directly to (kafka, 3)
.
Typically, you should only disable record caches for testing or debugging purposes – under normal circumstances it
is better to leave record caches enabled.
KTable userProfiles |
KGroupedTable groupedTable |
KTable aggregated |
|||||
---|---|---|---|---|---|---|---|
Timestamp | Input record | Interpreted as | Grouping | Initializer | Adder | Subtractor | State |
1 | (alice, E) | INSERT alice | (E, 5) | 0 (for E) | (E, 0 + 5) | (E, 5)
|
|
2 | (bob, A) | INSERT bob | (A, 3) | 0 (for A) | (A, 0 + 3) | (A, 3)
(E, 5)
|
|
3 | (charlie, A) | INSERT charlie | (A, 7) | (A, 3 + 7) | (A, 10)
(E, 5)
|
||
4 | (alice, A) | UPDATE alice | (A, 5) | (A, 10 + 5) | (E, 5 - 5) | (A, 15)
(E, 0)
|
|
5 | (charlie, null) | DELETE charlie | (null, 7) | (A, 15 - 7) | (A, 8)
(E, 0)
|
||
6 | (null, E) | ignored | (A, 8)
(E, 0)
|
||||
7 | (bob, E) | UPDATE bob | (E, 3) | (E, 0 + 3) | (A, 8 - 3) | (A, 5)
(E, 3)
|
Joining
Streams and tables can also be joined. Many stream processing applications in practice are coded as streaming joins. For example, applications backing an online shop might need to access multiple, updating database tables (e.g. sales prices, inventory, customer information) in order to enrich a new data record (e.g. customer transaction) with context information. That is, scenarios where you need to perform table lookups at very large scale and with a low processing latency. Here, a popular pattern is to make the information in the databases available in Kafka through so-called change data capture in combination with Kafka’s Connect API, and then implementing applications that leverage the Streams API to perform very fast and efficient local joins of such tables and streams, rather than requiring the application to make a query to a remote database over the network for each record. In this example, the KTable concept in Kafka Streams would enable you to track the latest state (e.g., snapshot) of each table in a local state store, thus greatly reducing the processing latency as well as reducing the load of the remote databases when doing such streaming joins.
The following join operations are supported, see also the diagram in the overview section of Stateful Transformations. Depending on the operands, joins are either windowed joins or non-windowed joins.
Join operands | Type | (INNER) JOIN | LEFT JOIN | OUTER JOIN |
---|---|---|---|---|
KStream-to-KStream | Windowed | Supported | Supported | Supported |
KTable-to-KTable | Non-windowed | Supported | Supported | Supported |
KTable-to-KTable Foreign-Key Join | Non-windowed | Supported | Supported | Not Supported |
KStream-to-KTable | Non-windowed | Supported | Supported | Not Supported |
KStream-to-GlobalKTable | Non-windowed | Supported | Supported | Not Supported |
KTable-to-GlobalKTable | N/A | Not Supported | Not Supported | Not Supported |
Each case is explained in more detail in the subsequent sections.
Join co-partitioning requirements
For equi-joins, input data must be co-partitioned when joining. This ensures that input records with the same key from both sides of the join, are delivered to the same stream task during processing. It is your responsibility to ensure data co-partitioning when joining. Co-partitioning is not required when performing KTable-KTable Foreign-Key joins and Global KTable joins.
The requirements for data co-partitioning are:
- The input topics of the join (left side and right side) must have the same number of partitions.
- All applications that write to the input topics must have the same partitioning strategy so that records with
the same key are delivered to same partition number. In other words, the keyspace of the input data must be
distributed across partitions in the same manner.
This means that, for example, applications that use Kafka’s Java Producer API must use the
same partitioner (cf. the producer setting
"partitioner.class"
akaProducerConfig.PARTITIONER_CLASS_CONFIG
), and applications that use the Kafka’s Streams API must use the sameStreamPartitioner
for operations such asKStream#to()
. The good news is that, if you happen to use the default partitioner-related settings across all applications, you do not need to worry about the partitioning strategy.
Why is data co-partitioning required? Because
KStream-KStream,
KTable-KTable, and
KStream-KTable joins
are performed based on the keys of records (e.g., leftRecord.key == rightRecord.key
), it is required that the
input streams/tables of a join are co-partitioned by key.
There are two exceptions where co-partitioning is not required. For
KStream-GlobalKTable joins joins, co-partitioning is
not required because all partitions of the GlobalKTable
‘s underlying changelog stream are made available to
each KafkaStreams
instance. That is, each instance has a full copy of the changelog stream. Further, a
KeyValueMapper
allows for non-key based joins from the KStream
to the GlobalKTable
.
KTable-KTable Foreign-Key joins also do not require co-partitioning. Kafka Streams internally ensures co-partitioning for Foreign-Key joins.
Note
Kafka Streams partly verifies the co-partitioning requirement:
During the partition assignment step, i.e. at runtime, Kafka Streams verifies whether the number of partitions for
both sides of a join are the same. If they are not, a TopologyBuilderException
(runtime exception) is being
thrown. Note that Kafka Streams cannot verify whether the partitioning strategy matches between the input
streams/tables of a join – it is up to the user to ensure that this is the case.
Ensuring data co-partitioning: If the inputs of a join are not co-partitioned yet, you must ensure this manually. You may follow a procedure such as outlined below. It is recommended to repartition the topic with fewer partitions to match the larger partition number of avoid bottlenecks. Technically it would also be possible to repartition the topic with more partitions to the smaller partition number. For stream-table joins, it's recommended to repartition the KStream because repartitioning a KTable might result in a second state store. For table-table joins, you might also consider to size of the KTables and repartition the smaller KTable.
Identify the input KStream/KTable in the join whose underlying Kafka topic has the smaller number of partitions. Let’s call this stream/table “SMALLER”, and the other side of the join “LARGER”. To learn about the number of partitions of a Kafka topic you can use, for example, the CLI tool
bin/kafka-topics
with the--describe
option.Within your application, re-partition the data of “SMALLER”. You must ensure that, when repartitioning the data with
repartition
, the same partitioner is used as for “LARGER”.- If “SMALLER” is a KStream:
KStream#repartition(Repartitioned.numberOfPartitions(...))
. - If “SMALLER” is a KTable:
KTable#toStream#repartition(Repartitioned.numberOfPartitions(...).toTable())
.
- If “SMALLER” is a KStream:
Within your application, perform the join between “LARGER” and the new stream/table.
KStream-KStream Join
KStream-KStream joins are always windowed joins, because otherwise the size of the internal state store used to perform the join – e.g., a sliding window or “buffer” – would grow indefinitely. For stream-stream joins it’s important to highlight that a new input record on one side will produce a join output for each matching record on the other side, and there can be multiple such matching records in a given join window (cf. the row with timestamp 15 in the join semantics table below, for example).
Join output records are effectively created as follows, leveraging the user-supplied ValueJoiner
:
KeyValue<K, LV> leftRecord = ...;
KeyValue<K, RV> rightRecord = ...;
ValueJoiner<LV, RV, JV> joiner = ...;
KeyValue<K, JV> joinOutputRecord = KeyValue.pair(
leftRecord.key, /* by definition, leftRecord.key == rightRecord.key */
joiner.apply(leftRecord.value, rightRecord.value)
);
Transformation | Description |
---|---|
Inner Join (windowed)
|
Performs an INNER JOIN of this stream with another stream.
Even though this operation is windowed, the joined stream will be of type Data must be co-partitioned: The input data for both sides must be co-partitioned. Causes data re-partitioning of a stream if and only if the stream was marked for re-partitioning (if both are marked, both are re-partitioned). Several variants of
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Left Join (windowed)
|
Performs a LEFT JOIN of this stream with another stream.
Even though this operation is windowed, the joined stream will be of type Data must be co-partitioned: The input data for both sides must be co-partitioned. Causes data re-partitioning of a stream if and only if the stream was marked for re-partitioning (if both are marked, both are re-partitioned). Several variants of
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Outer Join (windowed)
|
Performs an OUTER JOIN of this stream with another stream.
Even though this operation is windowed, the joined stream will be of type Data must be co-partitioned: The input data for both sides must be co-partitioned. Causes data re-partitioning of a stream if and only if the stream was marked for re-partitioning (if both are marked, both are re-partitioned). Several variants of
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Semantics of stream-stream joins: The semantics of the various stream-stream join variants are explained below. To improve the readability of the table, assume that (1) all records have the same key (and thus the key in the table is omitted), and (2) all records are processed in timestamp order. We assume a join window size of 10 seconds with a grace period of 5 seconds.
Note: If you use the old and now deprecated API to specify the grace period, i.e., JoinWindows.of(...).grace(...)
,
left/outer join results are emitted eagerly, and the observed result might differ from the result shown below.
The columns INNER JOIN, LEFT JOIN, and OUTER JOIN denote what is passed as arguments to the user-supplied
ValueJoiner for the join
, leftJoin
, and
outerJoin
methods, respectively, whenever a new input record is received on either side of the join. An empty table
cell denotes that the ValueJoiner
is not called at all.
Timestamp | Left (KStream) | Right (KStream) | (INNER) JOIN | LEFT JOIN | OUTER JOIN |
---|---|---|---|---|---|
1 | null | ||||
2 | null | ||||
3 | A | ||||
4 | a | [A, a] | [A, a] | [A, a] | |
5 | B | [B, a] | [B, a] | [B, a] | |
6 | b | [A, b], [B, b] | [A, b], [B, b] | [A, b], [B, b] | |
7 | null | ||||
8 | null | ||||
9 | C | [C, a], [C, b] | [C, a], [C, b] | [C, a], [C, b] | |
10 | c | [A, c], [B, c], [C, c] | [A, c], [B, c], [C, c] | [A, c], [B, c], [C, c] | |
11 | null | ||||
12 | null | ||||
13 | null | ||||
14 | d | [A, d], [B, d], [C, d] | [A, d], [B, d], [C, d] | [A, d], [B, d], [C, d] | |
15 | D | [D, a], [D, b], [D, c], [D, d] | [D, a], [D, b], [D, c], [D, d] | [D, a], [D, b], [D, c], [D, d] | |
... | |||||
40 | E | ||||
... | |||||
60 | F | [E, null] | [E, null] | ||
... | |||||
80 | f | [F, null] | [F, null] | ||
... | |||||
100 | G | [null, f] |
KTable-KTable Equi-Join
KTable-KTable equi-joins are always non-windowed joins. They are designed to be consistent with their counterparts in relational databases. The changelog streams of both KTables are materialized into local state stores to represent the latest snapshot of their table duals. The join result is a new KTable that represents the changelog stream of the join operation.
Join output records are effectively created as follows, leveraging the user-supplied ValueJoiner
:
KeyValue<K, LV> leftRecord = ...;
KeyValue<K, RV> rightRecord = ...;
ValueJoiner<LV, RV, JV> joiner = ...;
KeyValue<K, JV> joinOutputRecord = KeyValue.pair(
leftRecord.key, /* by definition, leftRecord.key == rightRecord.key */
joiner.apply(leftRecord.value, rightRecord.value)
);
Transformation | Description |
---|---|
Inner Join
|
Performs an INNER JOIN of this table with another table. The result is an ever-updating KTable that represents the "current" result of the join. (details) Data must be co-partitioned: The input data for both sides must be co-partitioned.
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Left Join
|
Performs a LEFT JOIN of this table with another table. (details) Data must be co-partitioned: The input data for both sides must be co-partitioned.
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Outer Join
|
Performs an OUTER JOIN of this table with another table. (details) Data must be co-partitioned: The input data for both sides must be co-partitioned.
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Semantics of table-table equi-joins:
The semantics of the various table-table equi-join variants are explained below.
To improve the readability of the table, you can assume that (1) all records have the same key (and thus the key in the table is omitted) and that (2) all records are processed in timestamp order.
The columns INNER JOIN, LEFT JOIN, and OUTER JOIN denote what is passed as arguments to the user-supplied
ValueJoiner for the join
, leftJoin
, and
outerJoin
methods, respectively, whenever a new input record is received on either side of the join. An empty table
cell denotes that the ValueJoiner
is not called at all.
Timestamp | Left (KTable) | Right (KTable) | (INNER) JOIN | LEFT JOIN | OUTER JOIN |
---|---|---|---|---|---|
1 | null | ||||
2 | null | ||||
3 | A | [A, null] | [A, null] | ||
4 | a | [A, a] | [A, a] | [A, a] | |
5 | B | [B, a] | [B, a] | [B, a] | |
6 | b | [B, b] | [B, b] | [B, b] | |
7 | null | null | null | [null, b] | |
8 | null | null | |||
9 | C | [C, null] | [C, null] | ||
10 | c | [C, c] | [C, c] | [C, c] | |
11 | null | null | [C, null] | [C, null] | |
12 | null | null | null | ||
13 | null | ||||
14 | d | [null, d] | |||
15 | D | [D, d] | [D, d] | [D, d] |
KTable-KTable Foreign-Key Join
KTable-KTable foreign-key joins are always non-windowed joins. Foreign-key joins are analogous to joins in SQL. As a rough example:
SELECT ... FROM {this KTable}
JOIN {other KTable}
ON {other.key} = {result of foreignKeyExtractor(this.value)} ...
The changelog streams of both KTables are materialized into local state stores to represent the latest snapshot of their table duals. A foreign-key extractor function is applied to the left record, with a new intermediate record created and is used to lookup and join with the corresponding primary key on the right hand side table. The result is a new KTable that represents the changelog stream of the join operation.
The left KTable can have multiple records which map to the same key on the right KTable. An update to a single left KTable entry may result in a single output event, provided the corresponding key exists in the right KTable. Consequently, a single update to a right KTable entry will result in an update for each record in the left KTable that has the same foreign key.Transformation | Description |
---|---|
Inner Join
|
Performs a foreign-key INNER JOIN of this table with another table. The result is an ever-updating KTable that represents the "current" result of the join. (details)
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Left Join
|
Performs a foreign-key LEFT JOIN of this table with another table. (details)
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Semantics of table-table foreign-key joins:
The semantics of the table-table foreign-key INNER and LEFT JOIN
variants are demonstrated below.
The key is shown alongside the value for each record.
Records are processed in incrementing offset order.
The columns INNER JOIN and LEFT JOIN denote what is
passed as arguments to the user-supplied ValueJoiner
for the join
and leftJoin
methods, respectively, whenever a new input record is received
on either side of the join. An empty table cell denotes that the
ValueJoiner
is not called at all. For the purpose of this example, Function
foreignKeyExtractor
simply uses the left-value
as the output.
Record Offset | Left KTable (K, extracted-FK) | Right KTable (FK, VR) | (INNER) JOIN | LEFT JOIN |
---|---|---|---|---|
1 | (k,1) | (1,foo) |
(k,1,foo) |
(k,1,foo) |
2 | (k,2) |
|
(k,null) | (k,2,null) |
3 | (k,3) |
(k,null) | (k,3,null) |
|
4 | (3,bar) |
(k,3,bar) |
(k,3,bar) |
|
5 | (k,null) |
(k,null) |
(k,null,null) | |
6 | (k,1) | (k,1,foo) |
(k,1,foo) |
|
7 | (q,10) |
(q,10,null) | ||
8 | (r,10) | (r,10,null) | ||
9 | (10,baz) | (q,10,baz), (r,10,baz) | (q,10,baz), (r,10,baz) |
KStream-KTable Join
KStream-KTable joins are always non-windowed joins. They allow you to perform table lookups against a KTable (changelog stream) upon receiving a new record from the KStream (record stream). An example use case would be to enrich a stream of user activities (KStream) with the latest user profile information (KTable).
Join output records are effectively created as follows, leveraging the user-supplied ValueJoiner
:
KeyValue<K, LV> leftRecord = ...;
KeyValue<K, RV> rightRecord = ...;
ValueJoiner<LV, RV, JV> joiner = ...;
KeyValue<K, JV> joinOutputRecord = KeyValue.pair(
leftRecord.key, /* by definition, leftRecord.key == rightRecord.key */
joiner.apply(leftRecord.value, rightRecord.value)
);
Transformation | Description |
---|---|
Inner Join
|
Performs an INNER JOIN of this stream with the table, effectively doing a table lookup. (details) Data must be co-partitioned: The input data for both sides must be co-partitioned. Causes data re-partitioning of the stream if and only if the stream was marked for re-partitioning. Several variants of
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Left Join
|
Performs a LEFT JOIN of this stream with the table, effectively doing a table lookup. (details) Data must be co-partitioned: The input data for both sides must be co-partitioned. Causes data re-partitioning of the stream if and only if the stream was marked for re-partitioning. Several variants of
Detailed behavior:
See the semantics overview at the bottom of this section for a detailed description. |
Semantics of stream-table joins:
The semantics of the various stream-table join variants are explained below.
To improve the readability of the table we assume that (1) all records have the same key (and thus we omit the key in
the table) and that (2) all records are processed in timestamp order.
The columns INNER JOIN and LEFT JOIN denote what is passed as arguments to the user-supplied
ValueJoiner for the join
and leftJoin
methods, respectively, whenever a new input record is received on either side of the join. An empty table
cell denotes that the ValueJoiner
is not called at all.
Timestamp | Left (KStream) | Right (KTable) | (INNER) JOIN | LEFT JOIN |
---|---|---|---|---|
1 | null | |||
2 | null | |||
3 | A | [A, null] | ||
4 | a | |||
5 | B | [B, a] | [B, a] | |
6 | b | |||
7 | null | |||
8 | null | |||
9 | C | [C, null] | ||
10 | c | |||
11 | null | |||
12 | null | |||
13 | null | |||
14 | d | |||
15 | D | [D, d] | [D, d] |
KStream-GlobalKTable Join
KStream-GlobalKTable joins are always non-windowed joins. They allow you to perform table lookups against a GlobalKTable (entire changelog stream) upon receiving a new record from the KStream (record stream). An example use case would be “star queries” or “star joins”, where you would enrich a stream of user activities (KStream) with the latest user profile information (GlobalKTable) and further context information (further GlobalKTables). However, because GlobalKTables have no notion of time, a KStream-GlobalKTable join is not a temporal join, and there is no event-time synchronization between updates to a GlobalKTable and processing of KStream records.
At a high-level, KStream-GlobalKTable joins are very similar to KStream-KTable joins. However, global tables provide you with much more flexibility at the some expense when compared to partitioned tables:
- They do not require data co-partitioning.
- They allow for efficient “star joins”; i.e., joining a large-scale “facts” stream against “dimension” tables
- They allow for joining against foreign keys; i.e., you can lookup data in the table not just by the keys of records in the stream, but also by data in the record values.
- They make many use cases feasible where you must work on heavily skewed data and thus suffer from hot partitions.
- They are often more efficient than their partitioned KTable counterpart when you need to perform multiple joins in succession.
Join output records are effectively created as follows, leveraging the user-supplied ValueJoiner
:
KeyValue<K, LV> leftRecord = ...;
KeyValue<K, RV> rightRecord = ...;
ValueJoiner<LV, RV, JV> joiner = ...;
KeyValue<K, JV> joinOutputRecord = KeyValue.pair(
leftRecord.key, /* by definition, leftRecord.key == rightRecord.key */
joiner.apply(leftRecord.value, rightRecord.value)
);
Transformation | Description |
---|---|
Inner Join
|
Performs an INNER JOIN of this stream with the global table, effectively doing a table lookup. (details) The Causes data re-partitioning of the stream if and only if the stream was marked for re-partitioning.
Detailed behavior:
|
Left Join
|
Performs a LEFT JOIN of this stream with the global table, effectively doing a table lookup. (details) The Causes data re-partitioning of the stream if and only if the stream was marked for re-partitioning.
Detailed behavior:
|
Semantics of stream-global-table joins:
The join semantics are different to KStream-KTable joins because it's not a temporal join.
Another difference is that, for KStream-GlobalKTable joins, the left input record is first “mapped” with
a user-supplied KeyValueMapper
into the table’s keyspace prior to the table lookup.
Windowing
Windowing lets you control how to group records that have the same key for stateful operations such as aggregations or joins into so-called windows. Windows are tracked per record key.
Note
A related operation is grouping, which groups all records that have the same key to ensure that data is properly partitioned (“keyed”) for subsequent operations. Once grouped, windowing allows you to further sub-group the records of a key.
For example, in join operations, a windowing state store is used to store all the records received so far within the
defined window boundary. In aggregating operations, a windowing state store is used to store the latest aggregation
results per window.
Old records in the state store are purged after the specified
window retention period.
Kafka Streams guarantees to keep a window for at least this specified time; the default value is one day and can be
changed via Materialized#withRetention()
.
The DSL supports the following types of windows:
Window name | Behavior | Short description |
---|---|---|
Hopping time window | Time-based | Fixed-size, overlapping windows |
Tumbling time window | Time-based | Fixed-size, non-overlapping, gap-less windows |
Sliding time window | Time-based | Fixed-size, overlapping windows that work on differences between record timestamps |
Session window | Session-based | Dynamically-sized, non-overlapping, data-driven windows |
Hopping time windows
Hopping time windows are windows based on time intervals. They model fixed-sized, (possibly) overlapping windows. A hopping window is defined by two properties: the window’s size and its advance interval (aka “hop”). The advance interval specifies by how much a window moves forward relative to the previous one. For example, you can configure a hopping window with a size 5 minutes and an advance interval of 1 minute. Since hopping windows can overlap – and in general they do – a data record may belong to more than one such windows.
Note
Hopping windows vs. sliding windows: Hopping windows are sometimes called “sliding windows” in other stream processing tools. Kafka Streams follows the terminology in academic literature, where the semantics of sliding windows are different to those of hopping windows.
The following code defines a hopping window with a size of 5 minutes and an advance interval of 1 minute:
import java.time.Duration;
import org.apache.kafka.streams.kstream.TimeWindows;
// A hopping time window with a size of 5 minutes and an advance interval of 1 minute.
// The window's name -- the string parameter -- is used to e.g. name the backing state store.
Duration windowSize = Duration.ofMinutes(5);
Duration advance = Duration.ofMinutes(1);
TimeWindows.ofSizeWithNoGrace(windowSize).advanceBy(advance);
Hopping time windows are aligned to the epoch, with the lower interval bound being inclusive and the upper bound
being exclusive. “Aligned to the epoch” means that the first window starts at timestamp zero. For example, hopping
windows with a size of 5000ms and an advance interval (“hop”) of 3000ms have predictable window boundaries
[0;5000),[3000;8000),...
— and not [1000;6000),[4000;9000),...
or even something “random” like
[1452;6452),[4452;9452),...
.
Unlike non-windowed aggregates that we have seen previously, windowed aggregates return a windowed KTable whose keys
type is Windowed<K>
. This is to differentiate aggregate values with the same key from different windows. The
corresponding window instance and the embedded key can be retrieved as Windowed#window()
and Windowed#key()
,
respectively.
Tumbling time windows
Tumbling time windows are a special case of hopping time windows and, like the latter, are windows based on time intervals. They model fixed-size, non-overlapping, gap-less windows. A tumbling window is defined by a single property: the window’s size. A tumbling window is a hopping window whose window size is equal to its advance interval. Since tumbling windows never overlap, a data record will belong to one and only one window.
Tumbling time windows are aligned to the epoch, with the lower interval bound being inclusive and the upper bound
being exclusive. “Aligned to the epoch” means that the first window starts at timestamp zero. For example, tumbling
windows with a size of 5000ms have predictable window boundaries [0;5000),[5000;10000),...
— and not
[1000;6000),[6000;11000),...
or even something “random” like [1452;6452),[6452;11452),...
.
The following code defines a tumbling window with a size of 5 minutes:
import java.time.Duration;
import org.apache.kafka.streams.kstream.TimeWindows;
// A tumbling time window with a size of 5 minutes (and, by definition, an implicit
// advance interval of 5 minutes), and grace period of 1 minute.
Duration windowSize = Duration.ofMinutes(5);
Duration gracePeriod = Duration.ofMinutes(1);
TimeWindows.ofSizeAndGrace(windowSize, gracePeriod);
// The above is equivalent to the following code:
TimeWindows.ofSizeAndGrace(windowSize, gracePeriod).advanceBy(windowSize);
Sliding time windows
Sliding windows are actually quite different from hopping and tumbling windows. In Kafka Streams, sliding windows
are used for join operations, specified by using the
JoinWindows
class, and windowed aggregations, specified by using the SlidingWindows
class.
A sliding window models a fixed-size window that slides continuously over the time axis. In this model, two data records are said to be included in the same window if (in the case of symmetric windows) the difference of their timestamps is within the window size. As a sliding window moves along the time axis, records may fall into multiple snapshots of the sliding window, but each unique combination of records appears only in one sliding window snapshot.
The following code defines a sliding window with a time difference of 10 minutes and a grace period of 30 minutes:
import org.apache.kafka.streams.kstream.SlidingWindows;
// A sliding time window with a time difference of 10 minutes and grace period of 30 minutes
Duration timeDifference = Duration.ofMinutes(10);
Duration gracePeriod = Duration.ofMinutes(30);
SlidingWindows.ofTimeDifferenceAndGrace(timeDifference, gracePeriod);
Sliding windows are aligned to the data record timestamps, not to the epoch. In contrast to hopping and tumbling windows, the lower and upper window time interval bounds of sliding windows are both inclusive.
Session Windows
Session windows are used to aggregate key-based events into so-called sessions, the process of which is referred to as sessionization. Sessions represent a period of activity separated by a defined gap of inactivity (or “idleness”). Any events processed that fall within the inactivity gap of any existing sessions are merged into the existing sessions. If an event falls outside of the session gap, then a new session will be created.
Session windows are different from the other window types in that:
- all windows are tracked independently across keys – e.g. windows of different keys typically have different start and end times
- their window sizes sizes vary – even windows for the same key typically have different sizes
The prime area of application for session windows is user behavior analysis. Session-based analyses can range from simple metrics (e.g. count of user visits on a news website or social platform) to more complex metrics (e.g. customer conversion funnel and event flows).
The following code defines a session window with an inactivity gap of 5 minutes:
import java.time.Duration;
import org.apache.kafka.streams.kstream.SessionWindows;
// A session window with an inactivity gap of 5 minutes.
SessionWindows.ofInactivityGapWithNoGrace(Duration.ofMinutes(5));
Given the previous session window example, here’s what would happen on an input stream of six records. When the first three records arrive (upper part of in the diagram below), we’d have three sessions (see lower part) after having processed those records: two for the green record key, with one session starting and ending at the 0-minute mark (only due to the illustration it looks as if the session goes from 0 to 1), and another starting and ending at the 6-minute mark; and one session for the blue record key, starting and ending at the 2-minute mark.
If we then receive three additional records (including two out-of-order records), what would happen is that the two existing sessions for the green record key will be merged into a single session starting at time 0 and ending at time 6, consisting of a total of three records. The existing session for the blue record key will be extended to end at time 5, consisting of a total of two records. And, finally, there will be a new session for the blue key starting and ending at time 11.
Window Final Results
In Kafka Streams, windowed computations update their results continuously. As new data arrives for a window, freshly computed results are emitted downstream. For many applications, this is ideal, since fresh results are always available. and Kafka Streams is designed to make programming continuous computations seamless. However, some applications need to take action only on the final result of a windowed computation. Common examples of this are sending alerts or delivering results to a system that doesn't support updates.
Suppose that you have an hourly windowed count of events per user. If you want to send an alert when a user has less than three events in an hour, you have a real challenge. All users would match this condition at first, until they accrue enough events, so you cannot simply send an alert when someone matches the condition; you have to wait until you know you won't see any more events for a particular window and then send the alert.
Kafka Streams offers a clean way to define this logic: after defining your windowed computation, you can suppress the intermediate results, emitting the final count for each user when the window is closed.
For example:
KGroupedStream<UserId, Event> grouped = ...;
grouped
.windowedBy(TimeWindows.ofSizeAndGrace(Duration.ofHours(1), Duration.ofMinutes(10)))
.count()
.suppress(Suppressed.untilWindowCloses(unbounded()))
.filter((windowedUserId, count) -> count < 3)
.toStream()
.foreach((windowedUserId, count) -> sendAlert(windowedUserId.window(), windowedUserId.key(), count));
The key parts of this program are:
ofSizeAndGrace(Duration.ofHours(1), Duration.ofMinutes(10))
- The specified grace period of 10 minutes (i.e., the
Duration.ofMinutes(10)
argument) allows us to bound the lateness of events the window will accept. For example, the 09:00 to 10:00 window will accept out-of-order records until 10:10, at which point, the window is closed. .suppress(Suppressed.untilWindowCloses(...))
- This configures the suppression operator to emit nothing for a window until it closes, and then emit the final result.
For example, if user
U
gets 10 events between 09:00 and 10:10, thefilter
downstream of the suppression will get no events for the windowed keyU@09:00-10:00
until 10:10, and then it will get exactly one with the value10
. This is the final result of the windowed count. unbounded()
- This configures the buffer used for storing events until their windows close. Production code is able to put a cap on the amount of memory to use for the buffer, but this simple example creates a buffer with no upper bound.
One thing to note is that suppression is just like any other
Kafka Streams operator, so you can build a topology with two
branches emerging from the count
,
one suppressed, and one not, or even multiple differently
configured suppressions.
This allows you to apply suppressions where they are needed
and otherwise rely on the default continuous update behavior.
For more detailed information, see the JavaDoc on the Suppressed
config object
and KIP-328.
Applying processors and transformers (Processor API integration)
Beyond the aforementioned stateless and stateful transformations, you may also leverage the Processor API from the DSL. There are a number of scenarios where this may be helpful:
- Customization: You need to implement special, customized logic that is not or not yet available in the DSL.
- Combining ease-of-use with full flexibility where it’s needed: Even though you generally prefer to use the expressiveness of the DSL, there are certain steps in your processing that require more flexibility and tinkering than the DSL provides. For example, only the Processor API provides access to a record’s metadata such as its topic, partition, and offset information. However, you don’t want to switch completely to the Processor API just because of that.
- Migrating from other tools: You are migrating from other stream processing technologies that provide an imperative API, and migrating some of your legacy code to the Processor API was faster and/or easier than to migrate completely to the DSL right away.
Transformation | Description |
---|---|
Process
|
Terminal operation. Applies a This is essentially equivalent to adding the An example is available in the javadocs. |
Transform
|
Applies a Each input record is transformed into zero, one, or more output records (similar to the stateless Marks the stream for data re-partitioning:
Applying a grouping or a join after
An example is available in the javadocs. |
Transform (values only)
|
Applies a Each input record is transformed into exactly one output record (zero output records or multiple output records are not possible).
The
An example is available in the javadocs. |
The following example shows how to leverage, via the KStream#process()
method, a custom Processor
that sends an
email notification whenever a page view count reaches a predefined threshold.
First, we need to implement a custom stream processor, PopularPageEmailAlert
, that implements the Processor
interface:
// A processor that sends an alert message about a popular page to a configurable email address
public class PopularPageEmailAlert implements Processor<PageId, Long, Void, Void> {
private final String emailAddress;
private ProcessorContext<Void, Void> context;
public PopularPageEmailAlert(String emailAddress) {
this.emailAddress = emailAddress;
}
@Override
public void init(ProcessorContext<Void, Void> context) {
this.context = context;
// Here you would perform any additional initializations such as setting up an email client.
}
@Override
void process(Record<PageId, Long> record) {
// Here you would format and send the alert email.
//
// In this specific example, you would be able to include
// information about the page's ID and its view count
}
@Override
void close() {
// Any code for clean up would go here, for example tearing down the email client and anything
// else you created in the init() method
// This processor instance will not be used again after this call.
}
}
Tip
Even though we do not demonstrate it in this example, a stream processor can access any available state stores by
calling ProcessorContext#getStateStore()
.
State stores are only available if they have been connected to the processor, or if they are global stores. While global stores do not need to be connected explicitly, they only allow for read-only access.
There are two ways to connect state stores to a processor:
- By passing the name of a store that has already been added via
Topology#addStateStore()
to the correspondingKStream#process()
method call. - Implementing
ConnectedStoreProvider#stores()
on theProcessorSupplier
passed toKStream#process()
. In this case there is no need to callStreamsBuilder#addStateStore()
beforehand, the store will be automatically added for you. You can also implementConnectedStoreProvider#stores()
on theValue*
or*WithKey
supplier variants, orTransformerSupplier
or any of its variants.
Then we can leverage the PopularPageEmailAlert
processor in the DSL via KStream#process
.
KStream<String, GenericRecord> pageViews = ...;
// Send an email notification when the view count of a page reaches one thousand.
pageViews.groupByKey()
.count()
.filter((PageId pageId, Long viewCount) -> viewCount == 1000)
// PopularPageEmailAlert is your custom processor that implements the
// `Processor` interface, see further down below.
.process(() -> new PopularPageEmailAlert("alerts@yourcompany.com"));
Naming Operators in a Streams DSL application
Kafka Streams allows you to name processors created via the Streams DSLControlling KTable emit rate
A KTable is logically a continuously updated table. These updates make their way to downstream operators whenever new data is available, ensuring that the whole computation is as fresh as possible. Logically speaking, most programs describe a series of transformations, and the update rate is not a factor in the program behavior. In these cases, the rate of update is more of a performance concern. Operators are able to optimize both the network traffic (to the Kafka brokers) and the disk traffic (to the local state stores) by adjusting commit interval and batch size configurations.
However, some applications need to take other actions, such as calling out to external systems,
and therefore need to exercise some control over the rate of invocations, for example of KStream#foreach
.
Rather than achieving this as a side-effect of the KTable record cache,
you can directly impose a rate limit via the KTable#suppress
operator.
For example:
KGroupedTable<String, String> groupedTable = ...;
groupedTable
.count()
.suppress(untilTimeLimit(ofMinutes(5), maxBytes(1_000_000L).emitEarlyWhenFull()))
.toStream()
.foreach((key, count) -> updateCountsDatabase(key, count));
This configuration ensures that updateCountsDatabase
gets events for each key
no more than once every 5 minutes.
Note that the latest state for each key has to be buffered in memory for that 5-minute period.
You have the option to control the maximum amount of memory to use for this buffer (in this case, 1MB).
There is also an option to impose a limit in terms of number of records (or to leave both limits unspecified).
Additionally, it is possible to choose what happens if the buffer fills up. This example takes a relaxed approach and just emits the oldest records before their 5-minute time limit to bring the buffer back down to size. Alternatively, you can choose to stop processing and shut the application down. This may seem extreme, but it gives you a guarantee that the 5-minute time limit will be absolutely enforced. After the application shuts down, you could allocate more memory for the buffer and resume processing. Emitting early is preferable for most applications.
For more detailed information, see the JavaDoc on the Suppressed
config object
and KIP-328.
Using timestamp-based semantics for table processors
By default, tables in Kafka Streams use offset-based semantics. When multiple records arrive for the same key, the one with the largest record offset is considered the latest record for the key, and is the record that appears in aggregation and join results computed on the table. This is true even in the event of out-of-order data. The record with the largest offset is considered to be the latest record for the key, even if this record does not have the largest timestamp.
An alternative to offset-based semantics is timestamp-based semantics. With timestamp-based semantics, the record with the largest timestamp is considered the latest record, even if there is another record with a larger offset (and smaller timestamp). If there is no out-of-order data (per key), then offset-based semantics and timestamp-based semantics are equivalent; the difference only appears when there is out-of-order data.
Starting with Kafka Streams 3.5, Kafka Streams supports timestamp-based semantics through the use of versioned state stores. When a table is materialized with a versioned state store, it is a versioned table and will result in different processor semantics in the presence of out-of-order data.
- When performing a stream-table join, stream-side records will join with the latest-by-timestamp table record which has a timestamp less than or equal to the stream record's timestamp. This is in contrast to joining a stream to an unversioned table, in which case the latest-by-offset table record will be joined, even if the stream-side record is out-of-order and has a lower timestamp.
- Aggregations computed on the table will include the latest-by-timestamp record for each key, instead of the latest-by-offset record. Out-of-order
updates (per key) will not trigger a new aggregation result. This is true for
count
andreduce
operations as well, in addition toaggregate
operations. - Table joins will use the latest-by-timestamp record for each key, instead of the latest-by-offset record. Out-of-order updates (per key) will not trigger a new join result. This is true for both primary-key table-table joins and also foreign-key table-table joins. If a versioned table is joined with an unversioned table, the result will be the join of the latest-by-timestamp record from the versioned table with the latest-by-offset record from the unversioned table.
- Table filter operations will no longer suppress consecutive tombstones, so users may observe more
null
records downstream of the filter than compared to when filtering an unversioned table. This is done in order to preserve a complete version history downstream, in the event of out-of-order data. suppress
operations are not allowed on versioned tables, as this would collapse the version history and lead to undefined behavior.
Once a table is materialized with a versioned store, downstream tables are also considered versioned until any of the following occurs:
- A downstream table is explicitly materialized, either with an unversioned store supplier or with no store supplier (all stores are unversioned by default, including the default store supplier)
- Any stateful transformation occurs, including aggregations and joins
- A table is converted to a stream and back.
The results of certain processors should not be materialized with versioned stores, as these processors do not produce a complete older version history, and therefore materialization as a versioned table would lead to unpredictable results:
- Aggregate processors, for both table and stream aggregations. This includes
aggregate
,count
andreduce
operations. - Table-table join processors, including both primary-key and foreign-key joins.
For more on versioned stores and how to start using them in your application, see here.
Writing streams back to Kafka
Any streams and tables may be (continuously) written back to a Kafka topic. As we will describe in more detail below, the output data might be re-partitioned on its way to Kafka, depending on the situation.
Writing to Kafka | Description |
---|---|
To
|
Terminal operation. Write the records to Kafka topic(s). (KStream details) When to provide serdes explicitly:
A variant of Another variant of
Causes data re-partitioning if any of the following conditions is true:
|
Note
When you want to write to systems other than Kafka: Besides writing the data back to Kafka, you can also apply a custom processor as a stream sink at the end of the processing to, for example, write to external databases. First, doing so is not a recommended pattern – we strongly suggest to use the Kafka Connect API instead. However, if you do use such a sink processor, please be aware that it is now your responsibility to guarantee message delivery semantics when talking to such external systems (e.g., to retry on delivery failure or to prevent message duplication).
Testing a Streams application
Kafka Streams comes with atest-utils
module to help you test your application here.
Kafka Streams DSL for Scala
The Kafka Streams DSL Java APIs are based on the Builder design pattern, which allows users to incrementally build the target functionality using lower level compositional fluent APIs. These APIs can be called from Scala, but there are several issues:
- Additional type annotations - The Java APIs use Java generics in a way that are not fully compatible with the type inferencer of the Scala compiler. Hence the user has to add type annotations to the Scala code, which seems rather non-idiomatic in Scala.
- Verbosity - In some cases the Java APIs appear too verbose compared to idiomatic Scala.
- Type Unsafety - The Java APIs offer some options where the compile time type safety is sometimes subverted and can result in runtime errors. This stems from the fact that the Serdes defined as part of config are not type checked during compile time. Hence any missing Serdes can result in runtime errors.
The Kafka Streams DSL for Scala library is a wrapper over the existing Java APIs for Kafka Streams DSL that addresses the concerns raised above. It does not attempt to provide idiomatic Scala APIs that one would implement in a Scala library developed from scratch. The intention is to make the Java APIs more usable in Scala through better type inferencing, enhanced expressiveness, and lesser boilerplates.
The library wraps Java Stream DSL APIs in Scala thereby providing:
- Better type inference in Scala.
- Less boilerplate in application code.
- The usual builder-style composition that developers get with the original Java API.
- Implicit serializers and de-serializers leading to better abstraction and less verbosity.
- Better type safety during compile time.
All functionality provided by Kafka Streams DSL for Scala are under the root package name of org.apache.kafka.streams.scala
.
Many of the public facing types from the Java API are wrapped. The following Scala abstractions are available to the user:
org.apache.kafka.streams.scala.StreamsBuilder
org.apache.kafka.streams.scala.kstream.KStream
org.apache.kafka.streams.scala.kstream.KTable
org.apache.kafka.streams.scala.kstream.KGroupedStream
org.apache.kafka.streams.scala.kstream.KGroupedTable
org.apache.kafka.streams.scala.kstream.SessionWindowedKStream
org.apache.kafka.streams.scala.kstream.TimeWindowedKStream
The library also has several utility abstractions and modules that the user needs to use for proper semantics.
org.apache.kafka.streams.scala.ImplicitConversions
: Module that brings into scope the implicit conversions between the Scala and Java classes.org.apache.kafka.streams.scala.serialization.Serdes
: Module that contains all primitive Serdes that can be imported as implicits and a helper to create custom Serdes.
The library is cross-built with Scala 2.12 and 2.13. To reference the library compiled against Scala {{scalaVersion}} include the following in your maven pom.xml
add the following:
<dependency>
<groupId>org.apache.kafka</groupId>
<artifactId>kafka-streams-scala_{{scalaVersion}}</artifactId>
<version>{{fullDotVersion}}</version>
</dependency>
To use the library compiled against Scala 2.12 replace the artifactId
with kafka-streams-scala_2.12
.
When using SBT then you can reference the correct library using the following:
libraryDependencies += "org.apache.kafka" %% "kafka-streams-scala" % "{{fullDotVersion}}"
Sample Usage
The library works by wrapping the original Java abstractions of Kafka Streams within a Scala wrapper object and then using implicit conversions between them. All the Scala abstractions are named identically as the corresponding Java abstraction, but they reside in a different package of the library e.g. the Scala class org.apache.kafka.streams.scala.StreamsBuilder
is a wrapper around org.apache.kafka.streams.StreamsBuilder
, org.apache.kafka.streams.scala.kstream.KStream
is a wrapper around org.apache.kafka.streams.kstream.KStream
, and so on.
Here's an example of the classic WordCount program that uses the Scala StreamsBuilder
that builds an instance of KStream
which is a wrapper around Java KStream
. Then we reify to a table and get a KTable
, which, again is a wrapper around Java KTable
.
The net result is that the following code is structured just like using the Java API, but with Scala and with far fewer type annotations compared to using the Java API directly from Scala. The difference in type annotation usage is more obvious when given an example. Below is an example WordCount implementation that will be used to demonstrate the differences between the Scala and Java API.
import java.time.Duration
import java.util.Properties
import org.apache.kafka.streams.kstream.Materialized
import org.apache.kafka.streams.scala.ImplicitConversions._
import org.apache.kafka.streams.scala._
import org.apache.kafka.streams.scala.kstream._
import org.apache.kafka.streams.{KafkaStreams, StreamsConfig}
object WordCountApplication extends App {
import Serdes._
val props: Properties = {
val p = new Properties()
p.put(StreamsConfig.APPLICATION_ID_CONFIG, "wordcount-application")
p.put(StreamsConfig.BOOTSTRAP_SERVERS_CONFIG, "kafka-broker1:9092")
p
}
val builder: StreamsBuilder = new StreamsBuilder
val textLines: KStream[String, String] = builder.stream[String, String]("TextLinesTopic")
val wordCounts: KTable[String, Long] = textLines
.flatMapValues(textLine => textLine.toLowerCase.split("\\W+"))
.groupBy((_, word) => word)
.count(Materialized.as("counts-store"))
wordCounts.toStream.to("WordsWithCountsTopic")
val streams: KafkaStreams = new KafkaStreams(builder.build(), props)
streams.start()
sys.ShutdownHookThread {
streams.close(Duration.ofSeconds(10))
}
}
In the above code snippet, we don't have to provide any Serdes, Grouped
, Produced
, Consumed
or Joined
explicitly. They will also not be dependent on any Serdes specified in the config. In fact all Serdes specified in the config will be ignored by the Scala APIs. All Serdes and Grouped
, Produced
, Consumed
or Joined
will be handled through implicit Serdes as discussed later in the Implicit Serdes section. The complete independence from configuration based Serdes is what makes this library completely typesafe. Any missing instances of Serdes, Grouped
, Produced
, Consumed
or Joined
will be flagged as a compile time error.
Implicit Serdes
One of the common complaints of Scala users with the Java API has been the repetitive usage of the Serdes in API invocations. Many of the APIs need to take the Serdes through abstractions like Grouped
, Produced
, Repartitioned
, Consumed
or Joined
. And the user has to supply them every time through the with function of these classes.
The library uses the power of Scala implicit parameters to alleviate this concern. As a user you can provide implicit Serdes or implicit values of Grouped
, Produced
, Repartitioned
, Consumed
or Joined
once and make your code less verbose. In fact you can just have the implicit Serdes in scope and the library will make the instances of Grouped
, Produced
, Consumed
or Joined
available in scope.
The library also bundles all implicit Serdes of the commonly used primitive types in a Scala module - so just import the module vals and have all Serdes in scope. A similar strategy of modular implicits can be adopted for any user-defined Serdes as well (User-defined Serdes are discussed in the next section).
Here's an example:
// DefaultSerdes brings into scope implicit Serdes (mostly for primitives)
// that will set up all Grouped, Produced, Consumed and Joined instances.
// So all APIs below that accept Grouped, Produced, Consumed or Joined will
// get these instances automatically
import Serdes._
val builder = new StreamsBuilder()
val userClicksStream: KStream[String, Long] = builder.stream(userClicksTopic)
val userRegionsTable: KTable[String, String] = builder.table(userRegionsTopic)
// The following code fragment does not have a single instance of Grouped,
// Produced, Consumed or Joined supplied explicitly.
// All of them are taken care of by the implicit Serdes imported by DefaultSerdes
val clicksPerRegion: KTable[String, Long] =
userClicksStream
.leftJoin(userRegionsTable)((clicks, region) => (if (region == null) "UNKNOWN" else region, clicks))
.map((_, regionWithClicks) => regionWithClicks)
.groupByKey
.reduce(_ + _)
clicksPerRegion.toStream.to(outputTopic)
Quite a few things are going on in the above code snippet that may warrant a few lines of elaboration:
- The code snippet does not depend on any config defined Serdes. In fact any Serdes defined as part of the config will be ignored.
- All Serdes are picked up from the implicits in scope. And
import Serdes._
brings all necessary Serdes in scope. - This is an example of compile time type safety that we don't have in the Java APIs.
- The code looks less verbose and more focused towards the actual transformation that it does on the data stream.
User-Defined Serdes
When the default primitive Serdes are not enough and we need to define custom Serdes, the usage is exactly the same as above. Just define the implicit Serdes and start building the stream transformation. Here's an example with AvroSerde
:
// domain object as a case class
case class UserClicks(clicks: Long)
// An implicit Serde implementation for the values we want to
// serialize as avro
implicit val userClicksSerde: Serde[UserClicks] = new AvroSerde
// Primitive Serdes
import Serdes._
// And then business as usual ..
val userClicksStream: KStream[String, UserClicks] = builder.stream(userClicksTopic)
val userRegionsTable: KTable[String, String] = builder.table(userRegionsTopic)
// Compute the total per region by summing the individual click counts per region.
val clicksPerRegion: KTable[String, Long] =
userClicksStream
// Join the stream against the table.
.leftJoin(userRegionsTable)((clicks, region) => (if (region == null) "UNKNOWN" else region, clicks.clicks))
// Change the stream from <user> -> <region, clicks> to <region> -> <clicks>
.map((_, regionWithClicks) => regionWithClicks)
// Compute the total per region by summing the individual click counts per region.
.groupByKey
.reduce(_ + _)
// Write the (continuously updating) results to the output topic.
clicksPerRegion.toStream.to(outputTopic)
A complete example of user-defined Serdes can be found in a test class within the library.