Class StickyAssignor
- All Implemented Interfaces:
- ConsumerPartitionAssignor
public class StickyAssignor
extends org.apache.kafka.clients.consumer.internals.AbstractStickyAssignor
The sticky assignor serves two purposes. First, it guarantees an assignment that is as balanced as possible, meaning either:
- the numbers of topic partitions assigned to consumers differ by at most one; or
- each consumer that has 2+ fewer topic partitions than some other consumer cannot get any of those topic partitions transferred to it.
Starting fresh it would work by distributing the partitions over consumers as evenly as possible. Even though this may sound similar to how round robin assignor works, the second example below shows that it is not. During a reassignment it would perform the reassignment in such a way that in the new assignment
- topic partitions are still distributed as evenly as possible, and
- topic partitions stay with their previously assigned consumers as much as possible.
Example 1. Suppose there are three consumers C0, C1, C2,
 four topics t0, t1, t2, t3, and each topic has 2 partitions,
 resulting in partitions t0p0, t0p1, t1p0, t1p1, t2p0,
 t2p1, t3p0, t3p1. Each consumer is subscribed to all three topics.
 The assignment with both sticky and round robin assignors will be:
 
- C0: [t0p0, t1p1, t3p0]
- C1: [t0p1, t2p0, t3p1]
- C2: [t1p0, t2p1]
C1 is removed and a reassignment is about to happen. The round robin assignor would produce:
 - C0: [t0p0, t1p0, t2p0, t3p0]
- C2: [t0p1, t1p1, t2p1, t3p1]
- C0 [t0p0, t1p1, t3p0, t2p0]
- C2 [t1p0, t2p1, t0p1, t3p1]
Example 2. There are three consumers C0, C1, C2,
 and three topics t0, t1, t2, with 1, 2, and 3 partitions respectively.
 Therefore, the partitions are t0p0, t1p0, t1p1, t2p0,
 t2p1, t2p2. C0 is subscribed to t0; C1 is subscribed to
 t0, t1; and C2 is subscribed to t0, t1, t2.
 The round robin assignor would come up with the following assignment:
 
- C0 [t0p0]
- C1 [t1p0]
- C2 [t1p1, t2p0, t2p1, t2p2]
- C0 [t0p0]
- C1 [t1p0, t1p1]
- C2 [t2p0, t2p1, t2p2]
C0 is removed, these two assignors would produce the following assignments.
 Round Robin (preserves 3 partition assignments):
 - C1 [t0p0, t1p1]
- C2 [t1p0, t2p0, t2p1, t2p2]
- C1 [t1p0, t1p1, t0p0]
- C2 [t2p0, t2p1, t2p2]
Impact on ConsumerRebalanceListener
 The sticky assignment strategy can provide some optimization to those consumers that have some partition cleanup code
 in their onPartitionsRevoked() callback listeners. The cleanup code is placed in that callback listener
 because the consumer has no assumption or hope of preserving any of its assigned partitions after a rebalance when it
 is using range or round robin assignor. The listener code would look like this:
 
 
 class TheOldRebalanceListener implements ConsumerRebalanceListener {
   void onPartitionsRevoked(Collection<TopicPartition> partitions) {
     for (TopicPartition partition: partitions) {
       commitOffsets(partition);
       cleanupState(partition);
     }
   }
   void onPartitionsAssigned(Collection<TopicPartition> partitions) {
     for (TopicPartition partition: partitions) {
       initializeState(partition);
       initializeOffset(partition);
     }
   }
 }
 
 
 As mentioned above, one advantage of the sticky assignor is that, in general, it reduces the number of partitions that
 actually move from one consumer to another during a reassignment. Therefore, it allows consumers to do their cleanup
 more efficiently. Of course, they still can perform the partition cleanup in the onPartitionsRevoked()
 listener, but they can be more efficient and make a note of their partitions before and after the rebalance, and do the
 cleanup after the rebalance only on the partitions they have lost (which is normally not a lot). The code snippet below
 clarifies this point:
 
 
 class TheNewRebalanceListener implements ConsumerRebalanceListener {
   Collection<TopicPartition> lastAssignment = Collections.emptyList();
   void onPartitionsRevoked(Collection<TopicPartition> partitions) {
     for (TopicPartition partition: partitions)
       commitOffsets(partition);
   }
   void onPartitionsAssigned(Collection<TopicPartition> assignment) {
     for (TopicPartition partition: difference(lastAssignment, assignment))
       cleanupState(partition);
     for (TopicPartition partition: difference(assignment, lastAssignment))
       initializeState(partition);
     for (TopicPartition partition: assignment)
       initializeOffset(partition);
     this.lastAssignment = assignment;
   }
 }
 
 
 Any consumer that uses sticky assignment can leverage this listener like this:
 consumer.subscribe(topics, new TheNewRebalanceListener());
 Note that you can leverage the CooperativeStickyAssignor so that only partitions which are being
 reassigned to another consumer will be revoked. That is the preferred assignor for newer cluster. See
 ConsumerPartitionAssignor.RebalanceProtocol for a detailed explanation of cooperative rebalancing.- 
Nested Class SummaryNested classes/interfaces inherited from class org.apache.kafka.clients.consumer.internals.AbstractStickyAssignororg.apache.kafka.clients.consumer.internals.AbstractStickyAssignor.MemberDataNested classes/interfaces inherited from class org.apache.kafka.clients.consumer.internals.AbstractPartitionAssignororg.apache.kafka.clients.consumer.internals.AbstractPartitionAssignor.MemberInfoNested classes/interfaces inherited from interface org.apache.kafka.clients.consumer.ConsumerPartitionAssignorConsumerPartitionAssignor.Assignment, ConsumerPartitionAssignor.GroupAssignment, ConsumerPartitionAssignor.GroupSubscription, ConsumerPartitionAssignor.RebalanceProtocol, ConsumerPartitionAssignor.Subscription
- 
Field SummaryFields Modifier and Type Field Description static StringSTICKY_ASSIGNOR_NAMEFields inherited from class org.apache.kafka.clients.consumer.internals.AbstractStickyAssignorDEFAULT_GENERATION, maxGeneration, partitionsTransferringOwnership
- 
Constructor SummaryConstructors Constructor Description StickyAssignor()
- 
Method SummaryModifier and Type Method Description protected org.apache.kafka.clients.consumer.internals.AbstractStickyAssignor.MemberDatamemberData(ConsumerPartitionAssignor.Subscription subscription)Stringname()Unique name for this assignor (e.g.voidonAssignment(ConsumerPartitionAssignor.Assignment assignment, ConsumerGroupMetadata metadata)Callback which is invoked when a group member receives its assignment from the leader.ByteBuffersubscriptionUserData(Set<String> topics)Return serialized data that will be included in theConsumerPartitionAssignor.Subscriptionsent to the leader and can be leveraged inConsumerPartitionAssignor.assign(Cluster, GroupSubscription)((e.g.Methods inherited from class org.apache.kafka.clients.consumer.internals.AbstractStickyAssignorassign, isStickyMethods inherited from class org.apache.kafka.clients.consumer.internals.AbstractPartitionAssignorassign, partitions, putMethods inherited from class java.lang.Objectclone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, waitMethods inherited from interface org.apache.kafka.clients.consumer.ConsumerPartitionAssignorsupportedProtocols, version
- 
Field Details- 
STICKY_ASSIGNOR_NAME- See Also:
- Constant Field Values
 
 
- 
- 
Constructor Details- 
StickyAssignorpublic StickyAssignor()
 
- 
- 
Method Details- 
nameDescription copied from interface:ConsumerPartitionAssignorUnique name for this assignor (e.g. "range" or "roundrobin" or "sticky"). Note, this is not required to be the same as the class name specified inConsumerConfig.PARTITION_ASSIGNMENT_STRATEGY_CONFIG- Returns:
- non-null unique name
 
- 
onAssignmentpublic void onAssignment(ConsumerPartitionAssignor.Assignment assignment, ConsumerGroupMetadata metadata)Description copied from interface:ConsumerPartitionAssignorCallback which is invoked when a group member receives its assignment from the leader.- Parameters:
- assignment- The local member's assignment as provided by the leader in- ConsumerPartitionAssignor.assign(Cluster, GroupSubscription)
- metadata- Additional metadata on the consumer (optional)
 
- 
subscriptionUserDataDescription copied from interface:ConsumerPartitionAssignorReturn serialized data that will be included in theConsumerPartitionAssignor.Subscriptionsent to the leader and can be leveraged inConsumerPartitionAssignor.assign(Cluster, GroupSubscription)((e.g. local host/rack information)- Parameters:
- topics- Topics subscribed to through- KafkaConsumer.subscribe(java.util.Collection)and variants
- Returns:
- nullable subscription user data
 
- 
memberDataprotected org.apache.kafka.clients.consumer.internals.AbstractStickyAssignor.MemberData memberData(ConsumerPartitionAssignor.Subscription subscription)- Specified by:
- memberDatain class- org.apache.kafka.clients.consumer.internals.AbstractStickyAssignor
 
 
-