Class StreamsRebalanceListener

    • Method Detail

      • onPartitionsRevoked

        public void onPartitionsRevoked​(Collection<TopicPartition> partitions)
        Description copied from interface: ConsumerRebalanceListener
        A callback method the user can implement to provide handling of offset commits to a customized store. This method will be called during a rebalance operation when the consumer has to give up some partitions. It can also be called when consumer is being closed (KafkaConsumer.close(Duration)) or is unsubscribing (KafkaConsumer.unsubscribe()). It is recommended that offsets should be committed in this callback to either Kafka or a custom offset store to prevent duplicate data.

        In eager rebalancing, it will always be called at the start of a rebalance and after the consumer stops fetching data. In cooperative rebalancing, it will be called at the end of a rebalance on the set of partitions being revoked iff the set is non-empty. For examples on usage of this API, see Usage Examples section of KafkaConsumer.

        It is common for the revocation callback to use the consumer instance in order to commit offsets. It is possible for a WakeupException or InterruptException to be raised from one of these nested invocations. In this case, the exception will be propagated to the current invocation of KafkaConsumer.poll(java.time.Duration) in which this callback is being executed. This means it is not necessary to catch these exceptions and re-attempt to wakeup or interrupt the consumer thread.

        Specified by:
        onPartitionsRevoked in interface ConsumerRebalanceListener
        Parameters:
        partitions - The list of partitions that were assigned to the consumer and now need to be revoked (may not include all currently assigned partitions, i.e. there may still be some partitions left)
      • onPartitionsLost

        public void onPartitionsLost​(Collection<TopicPartition> partitions)
        Description copied from interface: ConsumerRebalanceListener
        A callback method you can implement to provide handling of cleaning up resources for partitions that have already been reassigned to other consumers. This method will not be called during normal execution as the owned partitions would first be revoked by calling the ConsumerRebalanceListener.onPartitionsRevoked(java.util.Collection<org.apache.kafka.common.TopicPartition>), before being reassigned to other consumers during a rebalance event. However, during exceptional scenarios when the consumer realized that it does not own this partition any longer, i.e. not revoked via a normal rebalance event, then this method would be invoked.

        For example, this function is called if a consumer's session timeout has expired, or if a fatal error has been received indicating the consumer is no longer part of the group.

        By default it will just trigger ConsumerRebalanceListener.onPartitionsRevoked(java.util.Collection<org.apache.kafka.common.TopicPartition>); for users who want to distinguish the handling logic of revoked partitions v.s. lost partitions, they can override the default implementation.

        It is possible for a WakeupException or InterruptException to be raised from one of these nested invocations. In this case, the exception will be propagated to the current invocation of KafkaConsumer.poll(java.time.Duration) in which this callback is being executed. This means it is not necessary to catch these exceptions and re-attempt to wakeup or interrupt the consumer thread.

        Specified by:
        onPartitionsLost in interface ConsumerRebalanceListener
        Parameters:
        partitions - The list of partitions that were assigned to the consumer and now have been reassigned to other consumers. With the current protocol this will always include all of the consumer's previously assigned partitions, but this may change in future protocols (ie there would still be some partitions left)