public interface ConsumerPartitionAssignor
KafkaConsumer
. Members of the consumer group subscribe
to the topics they are interested in and forward their subscriptions to a Kafka broker serving
as the group coordinator. The coordinator selects one member to perform the group assignment and
propagates the subscriptions of all members to it. Then assign(Cluster, GroupSubscription)
is called
to perform the assignment and the results are forwarded back to each respective members
In some cases, it is useful to forward additional metadata to the assignor in order to make
assignment decisions. For this, you can override subscriptionUserData(Set)
and provide custom
userData in the returned Subscription. For example, to have a rack-aware assignor, an implementation
can use this user data to forward the rackId belonging to each member.Modifier and Type | Interface and Description |
---|---|
static class |
ConsumerPartitionAssignor.Assignment |
static class |
ConsumerPartitionAssignor.GroupAssignment |
static class |
ConsumerPartitionAssignor.GroupSubscription |
static class |
ConsumerPartitionAssignor.RebalanceProtocol
The rebalance protocol defines partition assignment and revocation semantics.
|
static class |
ConsumerPartitionAssignor.Subscription |
Modifier and Type | Method and Description |
---|---|
ConsumerPartitionAssignor.GroupAssignment |
assign(Cluster metadata,
ConsumerPartitionAssignor.GroupSubscription groupSubscription)
Perform the group assignment given the member subscriptions and current cluster metadata.
|
static List<ConsumerPartitionAssignor> |
getAssignorInstances(List<String> assignorClasses,
Map<String,Object> configs)
Get a list of configured instances of
ConsumerPartitionAssignor
based on the class names/types specified by ConsumerConfig.PARTITION_ASSIGNMENT_STRATEGY_CONFIG |
String |
name()
Unique name for this assignor (e.g.
|
default void |
onAssignment(ConsumerPartitionAssignor.Assignment assignment,
ConsumerGroupMetadata metadata)
Callback which is invoked when a group member receives its assignment from the leader.
|
default ByteBuffer |
subscriptionUserData(Set<String> topics)
Return serialized data that will be included in the
ConsumerPartitionAssignor.Subscription sent to the leader
and can be leveraged in assign(Cluster, GroupSubscription) ((e.g. |
default List<ConsumerPartitionAssignor.RebalanceProtocol> |
supportedProtocols()
Indicate which rebalance protocol this assignor works with;
By default it should always work with
ConsumerPartitionAssignor.RebalanceProtocol.EAGER . |
default short |
version()
Return the version of the assignor which indicates how the user metadata encodings
and the assignment algorithm gets evolved.
|
default ByteBuffer subscriptionUserData(Set<String> topics)
ConsumerPartitionAssignor.Subscription
sent to the leader
and can be leveraged in assign(Cluster, GroupSubscription)
((e.g. local host/rack information)topics
- Topics subscribed to through KafkaConsumer.subscribe(java.util.Collection)
and variantsConsumerPartitionAssignor.GroupAssignment assign(Cluster metadata, ConsumerPartitionAssignor.GroupSubscription groupSubscription)
metadata
- Current topic/broker metadata known by consumergroupSubscription
- Subscriptions from all members including metadata provided through subscriptionUserData(Set)
default void onAssignment(ConsumerPartitionAssignor.Assignment assignment, ConsumerGroupMetadata metadata)
assignment
- The local member's assignment as provided by the leader in assign(Cluster, GroupSubscription)
metadata
- Additional metadata on the consumer (optional)default List<ConsumerPartitionAssignor.RebalanceProtocol> supportedProtocols()
ConsumerPartitionAssignor.RebalanceProtocol.EAGER
.default short version()
String name()
ConsumerConfig.PARTITION_ASSIGNMENT_STRATEGY_CONFIG
static List<ConsumerPartitionAssignor> getAssignorInstances(List<String> assignorClasses, Map<String,Object> configs)
ConsumerPartitionAssignor
based on the class names/types specified by ConsumerConfig.PARTITION_ASSIGNMENT_STRATEGY_CONFIG