@@ -2905,6 +2906,26 @@ active-process-ratio metrics which have a recording level of info
:
The total number of records that have been emitted downstream from suppression operation nodes. |
kafka.streams:type=stream-processor-node-metrics,thread-id=([-.\w]+),task-id=([-.\w]+),processor-node-id=([-.\w]+) |
+
+ emit-final-latency-max |
+ The max latency to emit final records when a record could be emitted. |
+ kafka.streams:type=stream-processor-node-metrics,thread-id=([-.\w]+),task-id=([-.\w]+),processor-node-id=([-.\w]+) |
+
+
+ emit-final-latency-avg |
+ The avg latency to emit final records when a record could be emitted. |
+ kafka.streams:type=stream-processor-node-metrics,thread-id=([-.\w]+),task-id=([-.\w]+),processor-node-id=([-.\w]+) |
+
+
+ emit-final-records-rate |
+ The rate of records emitted when records could be emitted. |
+ kafka.streams:type=stream-processor-node-metrics,thread-id=([-.\w]+),task-id=([-.\w]+),processor-node-id=([-.\w]+) |
+
+
+ emit-final-records-total |
+ The total number of records emitted. |
+ kafka.streams:type=stream-processor-node-metrics,thread-id=([-.\w]+),task-id=([-.\w]+),processor-node-id=([-.\w]+) |
+
record-e2e-latency-avg |
The average end-to-end latency of a record, measured by comparing the record timestamp with the system time when it has been fully processed by the node. |
diff --git a/docs/streams/upgrade-guide.html b/docs/streams/upgrade-guide.html
index 8bec3a5504b..d46ba4ba3d3 100644
--- a/docs/streams/upgrade-guide.html
+++ b/docs/streams/upgrade-guide.html
@@ -105,6 +105,62 @@
More details about the new config StreamsConfig#TOPOLOGY_OPTIMIZATION
can be found in KIP-295.
+
+
+ Kafka Streams does not send a "leave group" request when an instance is closed. This behavior implies
+ that a rebalance is delayed until max.poll.interval.ms
passed.
+ KIP-812
+ introduces KafkaStreams.close(CloseOptions)
overload, which allows forcing an instance to leave the
+ group immediately.
+
+ Note: Due to internal limitations, CloseOptions
only works for static consumer groups at this point
+ (cf. KAFKA-16514 for more details and a fix in
+ some future release).
+
+
+
+ KIP-820
+ adapts the PAPI type-safety improvement of KIP-478 into the DSL. The existing methods KStream.transform
,
+ KStream.flatTransform
, KStream.transformValues
, and KStream.flatTransformValues
+ as well as all overloads of void KStream.process
are deprecated in favor of the newly added methods
+
+ KStream<KOut,VOut> KStream.process(ProcessorSupplier, ...)
+ KStream<K,VOut> KStream.processValues(FixedKeyProcessorSupplier, ...)
+
+ Both new methods have multiple overloads and return a KStream
instead of void
as the
+ deprecated process()
methods did. In addition, FixedKeyProcessor
, FixedKeyRecord
,
+ FixedKeyProcessorContext
, and ContextualFixedKeyProcessor
are introduced to guard against
+ disallowed key modification inside processValues()
. Furthermore, ProcessingContext
is
+ added for a better interface hierarchy.
+
+
+
+ Emitting a windowed aggregation result only after a window is closed is currently supported via the
+ suppress()
operator. However, suppress()
uses an in-memory implementation and does not
+ support RocksDB. To close this gap,
+ KIP-825
+ introduces "emit strategies", which are built into the aggregation operator directly to use the already existing
+ RocksDB store. TimeWindowedKStream.emitStrategy(EmitStrategy)
and
+ SessionWindowedKStream.emitStrategy(EmitStrategy)
allow picking between "emit on window update" (default)
+ and "emit on window close" strategies. Additionally, a few new emit metrics are added, as well as a necessary
+ new method, SessionStore.findSessions(long, long)
.
+
+
+
+ KIP-834 allows pausing
+ and resuming a Kafka Streams instance. Pausing implies that processing input records and executing punctuations will
+ be skipped; Kafka Streams will continue to poll to maintain its group membership and may commit offsets.
+ In addition to the new methods KafkaStreams.pause()
and KafkaStreams.resume()
, it is also
+ supported to check if an instance is paused via the KafkaStreams.isPaused()
method.
+
+
+
+ To improve monitoring of Kafka Streams applications, KIP-846
+ adds four new metrics bytes-consumed-total
, records-consumed-total
,
+ bytes-produced-total
, and records-produced-total
within a new topic level scope.
+ The metrics are collected at INFO level for source and sink nodes, respectively.
+
+
RocksDB offers many metrics which are critical to monitor and tune its performance. Kafka Streams started to make RocksDB metrics accessible