Browse Source

MINOR: Make it explicit in consumer docs that poll() is needed for callback to run (#4480)

Make it clear in the docs that the rebalance listener is only invoked during an active call to `poll()`. Plus a few additional doc cleanups.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Jason Gustafson <jason@confluent.io>
pull/4290/merge
dan norwood 7 years ago committed by Jason Gustafson
parent
commit
55aa7de62a
  1. 4
      clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceListener.java
  2. 36
      clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java

4
clients/src/main/java/org/apache/kafka/clients/consumer/ConsumerRebalanceListener.java

@ -44,7 +44,7 @@ import org.apache.kafka.common.TopicPartition; @@ -44,7 +44,7 @@ import org.apache.kafka.common.TopicPartition;
* partition is reassigned it may want to automatically trigger a flush of this cache, before the new owner takes over
* consumption.
* <p>
* This callback will execute in the user thread as part of the {@link Consumer#poll(long) poll(long)} call whenever partition assignment changes.
* This callback will only execute in the user thread as part of the {@link Consumer#poll(long) poll(long)} call whenever partition assignment changes.
* <p>
* It is guaranteed that all consumer processes will invoke {@link #onPartitionsRevoked(Collection) onPartitionsRevoked} prior to
* any process invoking {@link #onPartitionsAssigned(Collection) onPartitionsAssigned}. So if offsets or other state is saved in the
@ -103,7 +103,7 @@ public interface ConsumerRebalanceListener { @@ -103,7 +103,7 @@ public interface ConsumerRebalanceListener {
/**
* A callback method the user can implement to provide handling of customized offsets on completion of a successful
* partition re-assignment. This method will be called after an offset re-assignment completes and before the
* consumer starts fetching data.
* consumer starts fetching data, and only as the result of a {@link Consumer#poll(long) poll(long)} call.
* <p>
* It is guaranteed that all the processes in a consumer group will execute their
* {@link #onPartitionsRevoked(Collection)} callback before any instance executes its

36
clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java

@ -868,17 +868,20 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> { @@ -868,17 +868,20 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> {
*
* <p>
* As part of group management, the consumer will keep track of the list of consumers that belong to a particular
* group and will trigger a rebalance operation if one of the following events trigger -
* group and will trigger a rebalance operation if any one of the following events are triggered:
* <ul>
* <li>Number of partitions change for any of the subscribed list of topics
* <li>Topic is created or deleted
* <li>An existing member of the consumer group dies
* <li>A new member is added to an existing consumer group via the join API
* <li>Number of partitions change for any of the subscribed topics
* <li>A subscribed topic is created or deleted
* <li>An existing member of the consumer group is shutdown or fails
* <li>A new member is added to the consumer group
* </ul>
* <p>
* When any of these events are triggered, the provided listener will be invoked first to indicate that
* the consumer's assignment has been revoked, and then again when the new assignment has been received.
* Note that this listener will immediately override any listener set in a previous call to subscribe.
* Note that rebalances will only occur during an active call to {@link #poll(long)}, so callbacks will
* also only be invoked during that time.
*
* The provided listener will immediately override any listener set in a previous call to subscribe.
* It is guaranteed, however, that the partitions revoked/assigned through this interface are from topics
* subscribed in this call. See {@link ConsumerRebalanceListener} for more details.
*
@ -926,7 +929,7 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> { @@ -926,7 +929,7 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> {
*
* <p>
* This is a short-hand for {@link #subscribe(Collection, ConsumerRebalanceListener)}, which
* uses a noop listener. If you need the ability to seek to particular offsets, you should prefer
* uses a no-op listener. If you need the ability to seek to particular offsets, you should prefer
* {@link #subscribe(Collection, ConsumerRebalanceListener)}, since group rebalances will cause partition offsets
* to be reset. You should also provide your own listener if you are doing your own offset
* management since the listener gives you an opportunity to commit offsets before a rebalance finishes.
@ -944,17 +947,14 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> { @@ -944,17 +947,14 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> {
/**
* Subscribe to all topics matching specified pattern to get dynamically assigned partitions.
* The pattern matching will be done periodically against topic existing at the time of check.
* The pattern matching will be done periodically against all topics existing at the time of check.
* This can be controlled through the {@code metadata.max.age.ms} configuration: by lowering
* the max metadata age, the consumer will refresh metadata more often and check for matching topics.
* <p>
* As part of group management, the consumer will keep track of the list of consumers that
* belong to a particular group and will trigger a rebalance operation if one of the
* following events trigger -
* <ul>
* <li>Number of partitions change for any of the subscribed list of topics
* <li>Topic is created or deleted
* <li>An existing member of the consumer group dies
* <li>A new member is added to an existing consumer group via the join API
* </ul>
* See {@link #subscribe(Collection, ConsumerRebalanceListener)} for details on the
* use of the {@link ConsumerRebalanceListener}. Generally rebalances are triggered when there
* is a change to the topics matching the provided pattern and when consumer group membership changes.
* Group rebalances only take place during an active call to {@link #poll(long)}.
*
* @param pattern Pattern to subscribe to
* @param listener Non-null listener instance to get notifications on partition assignment/revocation for the
@ -988,7 +988,7 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> { @@ -988,7 +988,7 @@ public class KafkaConsumer<K, V> implements Consumer<K, V> {
* The pattern matching will be done periodically against topics existing at the time of check.
* <p>
* This is a short-hand for {@link #subscribe(Pattern, ConsumerRebalanceListener)}, which
* uses a noop listener. If you need the ability to seek to particular offsets, you should prefer
* uses a no-op listener. If you need the ability to seek to particular offsets, you should prefer
* {@link #subscribe(Pattern, ConsumerRebalanceListener)}, since group rebalances will cause partition offsets
* to be reset. You should also provide your own listener if you are doing your own offset
* management since the listener gives you an opportunity to commit offsets before a rebalance finishes.

Loading…
Cancel
Save