Interface RecordContext


public interface RecordContext
The context associated with the current record being processed by an Processor
  • Method Summary

    Modifier and Type
    Method
    Description
    Return the headers of the current input record; could be an empty header if it is not available.
    long
    Return the offset of the current input record; could be -1 if it is not available.
    int
    Return the partition id of the current input record; could be -1 if it is not available.
    long
    Return the current timestamp.
    Return the topic name of the current input record; could be null if it is not available.
  • Method Details

    • topic

      String topic()
      Return the topic name of the current input record; could be null if it is not available.

      For example, if this method is invoked within a punctuation callback, or while processing a record that was forwarded by a punctuation callback, the record won't have an associated topic. Another example is KTable.transformValues(ValueTransformerWithKeySupplier, String...) (and siblings), that do not always guarantee to provide a valid topic name, as they might be executed "out-of-band" due to some internal optimizations applied by the Kafka Streams DSL.

      Returns:
      the topic name
    • partition

      int partition()
      Return the partition id of the current input record; could be -1 if it is not available.

      For example, if this method is invoked within a punctuation callback, or while processing a record that was forwarded by a punctuation callback, the record won't have an associated partition id. Another example is KTable.transformValues(ValueTransformerWithKeySupplier, String...) (and siblings), that do not always guarantee to provide a valid partition id, as they might be executed "out-of-band" due to some internal optimizations applied by the Kafka Streams DSL.

      Returns:
      the partition id
    • offset

      long offset()
      Return the offset of the current input record; could be -1 if it is not available.

      For example, if this method is invoked within a punctuation callback, or while processing a record that was forwarded by a punctuation callback, the record won't have an associated offset. Another example is KTable.transformValues(ValueTransformerWithKeySupplier, String...) (and siblings), that do not always guarantee to provide a valid offset, as they might be executed "out-of-band" due to some internal optimizations applied by the Kafka Streams DSL.

      Returns:
      the offset
    • timestamp

      long timestamp()
      Return the current timestamp.

      If it is triggered while processing a record streamed from the source processor, timestamp is defined as the timestamp of the current input record; the timestamp is extracted from ConsumerRecord by TimestampExtractor. Note, that an upstream Processor might have set a new timestamp by calling forward(..., To.all().withTimestamp(...)). In particular, some Kafka Streams DSL operators set result record timestamps explicitly, to guarantee deterministic results.

      If it is triggered while processing a record generated not from the source processor (for example, if this method is invoked from the punctuate call), timestamp is defined as the current task's stream time, which is defined as the largest timestamp of any record processed by the task.

      Returns:
      the timestamp
    • headers

      Headers headers()
      Return the headers of the current input record; could be an empty header if it is not available.

      For example, if this method is invoked within a punctuation callback, or while processing a record that was forwarded by a punctuation callback, the record might not have any associated headers. Another example is KTable.transformValues(ValueTransformerWithKeySupplier, String...) (and siblings), that do not always guarantee to provide a valid headers, as they might be executed "out-of-band" due to some internal optimizations applied by the Kafka Streams DSL.

      Returns:
      the headers