Interface TimestampExtractor

All Known Implementing Classes:
FailOnInvalidTimestamp, LogAndSkipOnInvalidTimestamp, UsePartitionTimeOnInvalidTimestamp, WallclockTimestampExtractor

public interface TimestampExtractor
An interface that allows the Kafka Streams framework to extract a timestamp from an instance of ConsumerRecord. The extracted timestamp is defined as milliseconds.
  • Method Summary

    Modifier and Type Method Description
    long extract​(ConsumerRecord<Object,​Object> record, long partitionTime)
    Extracts a timestamp from a record.
  • Method Details

    • extract

      long extract​(ConsumerRecord<Object,​Object> record, long partitionTime)
      Extracts a timestamp from a record. The timestamp must be positive to be considered a valid timestamp. Returning a negative timestamp will cause the record not to be processed but rather silently skipped. In case the record contains a negative timestamp and this is considered a fatal error for the application, throwing a RuntimeException instead of returning the timestamp is a valid option too. For this case, Streams will stop processing and shut down to allow you investigate in the root cause of the negative timestamp.

      The timestamp extractor implementation must be stateless.

      The extracted timestamp MUST represent the milliseconds since midnight, January 1, 1970 UTC.

      It is important to note that this timestamp may become the message timestamp for any messages sent to changelogs updated by KTables and joins. The message timestamp is used for log retention and log rolling, so using nonsensical values may result in excessive log rolling and therefore broker performance degradation.

      Parameters:
      record - a data record
      partitionTime - the highest extracted valid timestamp of the current record's partition˙ (could be -1 if unknown)
      Returns:
      the timestamp of the record