...
Current state: Under Discussion
Discussion threadthreads: https://lists.apache.org/thread/zb5l1fsqw9vj25zkmtnrk6xm7q3dkm1v, https://lists.apache.org/thread/6o3sjvcb8dx1ozqfpltb7p0w76b4nd46
...
This KIP describes a protocol for extending KIP-595 so that the operators can programmatically update the voter voters set in a way that is safe consistent and available. There are two important use cases that this KIP supports. One use case is that the operator wants to change the number of controllers by adding or removing a controller. The other use case is that the operation wants to replace a controller because of a disk or hardware failure.
...
These are the definition of some important terms that are used through the document.
KRaft: The consensus protocol inspired by Raft for Kafka. This protocol is defined by KIP-595: A Raft Protocol for the Metadata Quorum, KIP-630: Kafka Raft Snapshot. The implementation is in the Java raft
module.
Controller: The controller or the metadata state machine is the application built using the KRaft consensus layer. The implementation is in the Java metadata
module.
Voters: A voter is any replica that can transition to the candidate state and to the leader state. Voters for the KRaft cluster metadata partition are also called controllers. Voters are required to have an ID and UUID.
Voter Voters set: A voter voters set is the group of voters for a partition. Each replica keeps their own set of voters that are part of the topic partition. For a replica, the voters are the replica ID and UUID is in its own voter voters set. A candidate needs to get votes from the majority of its own voter voters set before it can become the leader of an epoch. When a voter becomes a leader it will use its voter voters set to determine when an offset has been committed.
Observers: An observer is any replica that is not in the voter voters set. This is because they have an ID and UUID which is not in the voter voters set or they don't have an ID or UUID.
...
There will be two mechanisms for bootstrapping the KRaft cluster metadata partition. For context, the set of voters in the KRaft cluster metadata partition is currently bootstrapped and configured using the controller.quorum.voters
server property. This property is also used to configured the configure the brokers with the set of endpoints that know the location of the leader if it exists.
...
This command will parse the release-version to the matching metadata.version
and kraft.version
features. It will send a UpdateFeatures
request to a node with both features set to the matching version. KRaft will write the KRaftVersionRecord
control record, if all of the controllers and brokers support the new version. KRaft will use the information in the controller registration, broker registration and add voter records to determine if the new version is compatible.
Add
...
controller
To increase the number of controller the user needs to format a controller node, start the controller node and add that node to the KRaft cluster metadata partition. Formatting a controller node can be done with the following CLI command:
...
Code Block |
---|
kafka-metadata-quorum --bootstrap-server <endpoints> add-controller --config controller.properties |
Remove
...
controller
To decrease the number of controllers the user needs to execute the following CLI command:
Code Block |
---|
kafka-metadata-quorum --bootstrap-server <endpoints> remove-controller --controller-id <replica-id> --controller-uuid <replica-uuid> |
Common
...
scenarios
To better illustrate this feature this section describes two few common scenarios that the Kafka administrator may perform.
Disk Failure Recovery
If one of the replicas encounter a disk failure the operator can replace this disk with a new disk and start the replica.
Let's assume that the cluster has the following voters for the KRaft cluster metadata partition (1, UUID1), (2, UUID2) and (3, UUID3). The first element of the tuples is the replica id. The second element of the tuples is the replica uuid. The replica uuid, or directory id, is generated using the storage tool when the node is formatted.
Automatic joining controllers
Imagine that the user is trying to create KRaft controller quorum with 3 voters (1, 2, 3).
At this point the disk for replica 3 is replaced and formatted. This means that when replica 3 starts it will have a new replica uuid (UUID3') and an empty set of voters. Replica 3 will discover the partition leader either using For all of the controller properties files assume that the controller.quorum.auto.join.enable
is set to true
and the controller.quorum.bootstrap.servers
or by the leader sending a BeginQuorumEpoch request to replica 3. Once the new replica, (3, UUID3'), discovers the leader it send the Fetch and FetchSnapshot requests to the leader. At this point the leader's set of voters will be (1, UUID1), (2, UUID2) and (3, UUID3) and the set of observers will include (3, UUID3').
This state can be discovered by the operator by using the DescribeQuorum RPC, the Admin client or the kafka-metadata-quorum
CLI. The operator can now decide to add replica (3, UUID3') to the set of voters using the AddVoter RPC, the Admin client or the kafka-metadata-quorum
CLI.
When the AddVoter RPC succeeds the voters will be (1, UUID1), (2, UUID2), (3, UUID3) and (3, UUID3'). The operator can remove the failed disk from the voter set by using the RemoveVoter RPC, the Admin client or the kafka-metadata-quorum
CLI.
contains the endpoint of controller 1.
On controller 1 the user runs:
Code Block |
---|
kafka-storage format --cluster-id <cluster-id> --release-version 3.8 --standalone --config controller.properties |
and starts the controller. When controller 1 starts it sees that is already in the voters set so it eventually becomes leader.
On controller 2 and 3 the user runs:
Code Block |
---|
kafka-storage format --cluster-id <cluster-id> --release-version 3.8 --config controller.properties |
and starts the controllers. Notice that neither --standalone
or --controller-quorum-voters
is used for controller 2 and 3 so the controllers will start as observers. These controllers will discover the leader using controller.quorum.bootstrap.servers
and will use the AddVoter
RPC to join the voters set.
Disk failure recovery
If one of the replicas encounter a disk failure the operator can replace this disk with a new disk and start the replica.
Let's assume that the cluster has the following voters for the KRaft cluster metadata partition When the RemoveVoter RPC succeeds the voters will be (1, UUID1), (2, UUID2), and (3, UUID3'). At this point the Kafka cluster has successfully recover from a disk failure in a controller node in a consistent way.
Node Failure Recovery
Let's assume that the voter set is (1, UUID1), (2, UUID2) and (3, UUID3)The first element of the tuples is the replica id. The second element of the tuples is the replica uuid. The replica uuid, or directory id, is generated using the storage tool when the node is formatted.
At this point the disk for replica 3 has failed and the Kafka operator would like to replace it. The operator would start a new controller with replica id 4. The node 4's metadata log dir would get formatted, generate and persist the directory id UUID4. Replica 4 will discover the leader by sending Fetch and FetchSnapshot request to the servers enumerated in is replaced and formatted. This means that when replica 3 starts it will have a new replica uuid (UUID3') and an empty set of voters. Replica 3 will discover the partition leader either using controller.quorum.bootstrap.servers
. After a successful Fetch RPC, the leader's set of voters will be (1, UUID1), ( or by the leader sending a BeginQuorumEpoch
request to replica 3. Once the new replica, (3, UUID3'), discovers the leader it send the Fetch
and FetchSnapshot
requests to the leader. At this point the leader's set of voters will be (1, UUID1), (2, UUID2) , and (3, UUID3) and the set of observers will include (43, UUID4UUID3').
This state can be discovered by the operator by using the DescribeQuorum
RPC, the Admin client or the kafka-metadata-quorum
CLI. The operator can now decide to add replica (43, UUID4UUID3') to the set of voters using the Admin client or the kafka-metadata-quorum
CLI.
When the AddVoter
RPC . When this operation succeeds the voters set of voters will be (1, UUID1), (2, UUID2), (3, UUID3) and (43, UUID4UUID3'). The operator can now decided to remove replica (3, UUID3) remove the failed disk from the voters set of voters by using the RemoveVoter RPC.
Reference Explanation
The general goal of this KIP is to allow the user to dynamically change the set of voters (also known as controllers) for the KRaft cluster metadata partition. This is achieved by storing the set of voters and their known endpoints in the log instead of the controller.quorum.voters
properties. Because the set of voters is stored in the log it allows the leader to replicate this information to all of the fetching replicas (voters and observers). Since old log segments (records) can be deleted once a snapshot has been created, the KRaft snapshots will also contain the set of voters up to the included offset.
The fetching (following) voters will discover the set of voters and their endpoints by fetching the latest log from the leader. How do new voters discovery the leader's endpoint? The leader will push this information to new voters (or voters that were offline for a long time) using the BeginQuorumEpoch
request. The active leader sends BeginQuorumEpoch
to all of the voters in the voter set when it becomes leader for an epoch.
The leader doesn't send BeginQuorumEpoch
to observers since this are dynamic and are not included in the KRaft partition log. Observer will instead discover the leader using the controller.quorum.bootstrap.servers
. It is important that this property includes at least one of the available voters, else brokers (observers) will not be able to discover the leader of the KRaft cluster metadata partition.
To make changes to the voter set safe it is required that the majority of the competing voter sets commit the voter changes. In this design the competing voter sets are the current voter set and new voter set. Since this design only allows one voter change at a time the majority of the new configuration always overlaps (intercepts) the majority of the old configuration. This is done by the leader committing the current epoch when it becomes leader and committing single voter changes with the new voter set before accepting another voter change.
The rest of these section and subsection goes into the detail changes required to implement this new functionality.
Directory id or replica UUID
In addition to a replica ID each replica assigned to a KRaft topic partition will have a replica UUID. This UUID will be generated once and persisted in the meta.properties
file for the metadata.log.dir
. Replica UUID and directory id was first introduced in KIP-858: Handle JBOD broker disk failure in KRaft.
There are two cases when a directory id will be generated and persisted:
kafka-storage
format command will generate a directory id for all of the log directories including the metadata log dir.meta.properties
exists but it doesn't include adirectory.id
property. This case will generate and persist a directory id to support upgrade from Kafka versions that don't support this feature to versions that support this feature.
Supported features
This feature can only be enabled if all of the voters (controllers) and observers (brokers) support this feature. This is required mainly because this feature needs to write a new control record (VotersRecord
) to the KRaft cluster metadata partition. All replicas need to be able to read and decode these new control record.
There will be a new SupportedFeature added to the ApiVersions response of the Kafka nodes. The name of this new supported feature will be kraft.version
. The default value be 0 and represents that only KIP-595: A Raft Protocol for the Metadata Quorum, KIP-630: Kafka Raft Snapshot and KIP-996: Pre-Vote are supported. Version 1 means that this KIP is supported.
When the clients sends a UpdateFeatures RPC to the active controller, if the FeatureUpdates.Feature
property is kraft.version
, the associated information will be passed to KRaft client. The KRaft client will implement two different algorithms to check if the upgrade is supported by voters and observers. For voters the KRaft client will comparing the upgraded version against all of the persisted voter set for the KRaft cluster metadata partition. The KRaft client cannot do this for observers (brokers) since their supported versions are not persisted in the log. The controller will instead push the broker registration information to the KRaft client.
Voter Changes
Adding Voters
Voters are added to the cluster metadata partition by sending an AddVoter
RPC to the leader. For safety Kafka will only allow one voter change operation at a time. If there are any pending voter change operations the leader will wait for them to finish.
If there are no pending voter change operations the leader send an ApiVersions request to the new voter's endpoint to discover it's kraft.version
support features. If the new leader supports the current kraft.version
, it will write a VotersRecord
to the log, with the current voters set plus the voter getting added, and immediately update its in-memory quorum state to include this voter as part of the quorum. Any replica that replicates and reads this VotersRecord
will update their in-memory voter set to include this new voter. Voters will not wait for these records to get committed before updating their voter set.
Once the VotersRecord
operation has been committed by the majority of the new voter set, the leader can respond to the AddVoter
RPC and process new voter change operations.
Removing Voters
Voter are removed from the cluster metadata partition by sending a RemoveVoter
RPC to the leader. This works similar to adding a voter. If there are no pending voter change operations the leader will append the VotersRecord
to the log, the the current voters set minus the voter getting removed, and immediately update its voter set to the new configuration.
Once the VotersRecord
operation has been committed by the majority of the new voter set, the leader can respond to the RPC. If the removed voters is the leader, the leader will resign from the quorum when the VotersRecord
has been committed. To allow this operation to be committed and for the leader to resign the followers will continue to fetch from the leader even if the leader is not part of the new voter set. In KRaft, leader election is triggered when the voter hasn't received a successful response in the fetch timeout.
Bootstrapping
The section describe how the quorum will get bootstrap to support this KIP. There are two configurations that Kafka will support and each will be explain separately. The Kafka cluster can be configured to use the existing controller.quorum.voters
and the new property called controller.quorum.bootstrap.servers
.
controller.quorum.voters
This is a static quorum configuration where all of the voters' ID, host and port are specified. An example value for this configuration is 1@localhost:9092,2@localhost:9093,3@localhost:9094
.
If the kraft.version
is 0, this properties will be used to configure the set of voters for the KRaft cluster metadata partition.
When the kraft.version
is upgraded to a version greater 0 and the version is supported by the voters, the leader will write a control record batch that will include the kraft.version
record and all of the VotersRecord control record with all of the voters specified. This will allow KRaft to ignore the controller.quorum.voters properties and instead rely solely on the log state when it is upgraded to kraft.version 1.
controller.quorum.bootstrap.servers
This configuration describe the set of hosts and ports that can be queried to discover the cluster metadata partition leader. Observers and to-be-added voters will send Fetch requests to this list of servers until the leader is discovered.
When using this configuration for a new cluster, the quorum should be started with only one voter. This voter is bootstrapped by running the storage tool format command. This tool will create the cluster metadata partition and append the VotersRecord
to it. Additional voters can get added by using the AddVoter
RPC as described in this KIP.
Leader Election
It is possible for the leader to add a new voter to the voters set, write the VotersRecord
to the log and only replicate it to some of the voters in the new configuration. If the leader fails before this record has been replicated to the new voter it is possible that a new leader cannot be elected. This is because voters reject vote request from replicas that are not in the voter set. This check will be removed and replicas will reply to votes request when the candidate is not in the voter set or the voting replica is not in the voter set. The candidate must still have a longer log offset and epoch before the voter will grant a vote to it.
First Leader
When a KRaft voter becomes leader it will write a KRaftVersionRecord
and VotersRecord
to the log if the log or the latest snapshot doesn't contain any VotersRecord. This is done to make sure that the voter set in the bootstrap snapshot gets replicated to all of the voters and to not rely on all of the voters being configured with the same bootstrapped voter set.
Automatic endpoint and directory id discovery
To improve the usability of this feature it would beneficial for the leader of the KRaft cluster metadata leader to automatically rediscover the voters' endpoints. This makes it possible for the operator to update the endpoint of a voter without having to use the kafka-metadata-quorum
tool. When a voter becomes a follower and discovers a new leader, it will always send an UpdateVoter RPC to the leader. This request instructs the leader to update the endpoints of the matching replica id and replica uuid. When at voter becomes a leader it will also write an VotersRecord controler record with the updated endpoints and kraft.version
feature.
The directory id, or replica uuid, will behave differently. The quorum shouldn't automatically update the directory id, since different values means that the disk was replaced. For directory id, the leader will only override it if it was not previously set. This behavior is useful for when a cluster gets upgraded to a kraft.version
greater than 0.
High Watermark
As describe in KIP-595, the high-watermark will be calculated using the fetch offset of the majority of the voters. When a replica is removed or added it is possible for the high-watermark to decrease. The leader will not allow the high-watermark to decrease and will guarantee that is is monotonically increasing for both the state machines and the remote replicas.
With this KIP, it is possible for the leader to not be part of the voter set when the replica removed is the leader. In this case the leader will continue to handle Fetch and FetchSnapshot request as normal but it will not count itself when computing the high watermark.
Snapshots
The snapshot generation code needs to be extended to include these new KRaft specific control record for VotersRecord. Before this KIP the snapshot didn't include any KRaft generated control records. When the state machine (controller or broker) calls RaftClient.createSnapshot, the KRaft client will write the SnapshotHeaderRecord, KRaftVersionRecord and VotersRecord controller records at the beginning of the snapshot.
Note that the current KRaft implementation already writes the SnapshotHeaderRecord. The new KRaft implementation will write KRraftVersionRecord
and VotersRecord
, if the kraft.version
is greater than 0.
To efficiently implement this feature the KRaft implement we keep track of all voters sets between the latest snapshot and the LEO. The replicas will update these order list of voters set whenever the latest snapshot id increases, a VotersRecord
control record is read from the log and the log is truncated.
Internal Listener
The KRaft implementation and protocol describe in KIP-595 and KIP-630 never read from the log or snapshot. This KIP requires the KRaft implementation now read uncommitted data from log and snapshot to discover the voter set. This also means that the KRaft implementation needs to handle this uncommitted state getting truncated and reverted.
Public Interfaces
Configuration
There only two configurations for this feature.
controller.quorum.voters
This is an existing configuration. This configuration describes the state of the quorum and will only be used if the kraft.version feature is 0.
controller.quorum.bootstrap.servers
This is a list of nodes that brokers and new controllers can use to discover the quorum leader. Brokers and new controllers (observers) will send Fetch requests to all of the nodes in this configuration until they discover the quorum leader and the Fetch request succeeds. The quorum voters and their configuration will be learned by fetching and reading the records from the log and snapshot. This includes committed and uncommitted records.
If this configuration is specified, observers will not use the controller.quorum.voters
endpoints to discover the leader.
Log and Snapshot Control Records
A few new control records will be added to the log and snapshot of a KRaft partition.
KRaftVersionRecord
Control record for recording the latest KRaft protocol version. The KRaft protocol version is used to determine which version of LeaderChangeMessage to write. It is also used to determine if the VotersRecord control record can be written to the topic partition.
The ControlRecordType
is TBD and will be updated when the code is committed to Kafka.
Code Block |
---|
{
"type": "data",
"name": "KRaftVersionRecord",
"validVersions": "0",
"flexibleVersions": "0+",
"fields": [
{ "name": "Version", "type": "int16", "versions": "0+",
"about": "The version of the kraft version record" },
{ "name": "KRaftVersion", "type": "int16", "versions": "0+",
"about": "The kraft protocol version" }
]
} |
Handling
When KRaft replicas read this record from the log they will update their finalized kraft.version
once the HWM shows this record as committed.
LeaderChangeMessage
Add an optional VoterUuid to Voter. This change is not needed for correctness but it is nice to have for tracing and debugging. The leader will write version 0 if the kraft.version
is 0. The leader will write version 1 if the kraft.version
is 1.
Admin client or the kafka-metadata-quorum
CLI.
When the RemoveVoter
RPC succeeds the voters will be (1, UUID1), (2, UUID2), and (3, UUID3'). At this point the Kafka cluster has successfully recover from a disk failure in a controller node in a consistent way.
Node failure recovery
Let's assume that the voters set is (1, UUID1), (2, UUID2) and (3, UUID3).
At this point replica 3 has failed and the Kafka operator would like to replace it. The operator would start a new controller with replica id 4. The node 4's metadata log dir would get formatted, generate and persist the directory id UUID4. Replica 4 will discover the leader by sending Fetch
and FetchSnapshot
request to the servers enumerated in controller.quorum.bootstrap.servers
. After a successful Fetch
RPC, the leader's set of voters will be (1, UUID1), (2, UUID2), (3, UUID3) and set of observers will include (4, UUID4).
The operator can now decide to add replica (4, UUID4) to the set of voters. When this operation succeeds the set of voters will be (1, UUID1), (2, UUID2), (3, UUID3) and (4, UUID4).
The operator can now decided to remove replica (3, UUID3) from the set of voters.
Reference explanation
The general goal of this KIP is to allow the user to dynamically change the set of voters (also known as controllers) for the KRaft cluster metadata partition. This is achieved by storing the set of voters and their known endpoints in the log instead of the controller.quorum.voters
properties. Because the set of voters is stored in the log it allows the leader to replicate this information to all of the fetching replicas (voters and observers). Since old log segments (records) can be deleted once a snapshot has been created, the KRaft snapshots will also contain the set of voters up to the included offset.
The fetching (following) voters will discover the set of voters and their endpoints by fetching the latest log from the leader. How do new voters discovery the leader's endpoint? The leader will push this information to new voters (or voters that were offline for a long time) using the BeginQuorumEpoch
request. The active leader sends BeginQuorumEpoch
to all of the voters in the voters set when it becomes leader for an epoch.
The leader doesn't send BeginQuorumEpoch
to observers since this are dynamic and are not included in the KRaft partition log. Observer will instead discover the leader using the controller.quorum.bootstrap.servers
. It is important that this property includes at least one of the available voters, else brokers (observers) will not be able to discover the leader of the KRaft cluster metadata partition.
To make changes to the voters set consistent it is required that the majority of the competing voters sets commit the voter changes. In this design the competing voters sets are the current voters set and new voters set. Since this design only allows one voter change at a time the majority of the new configuration always overlaps (intercepts) the majority of the old configuration. This is done by the leader committing the current epoch when it becomes leader and committing single voter changes with the new voters set before accepting another voter change.
The rest of these section and subsection goes into the detail changes required to implement this new functionality.
Directory id or replica UUID
In addition to a replica ID each replica assigned to a KRaft topic partition will have a replica UUID. This UUID will be generated once and persisted in the meta.properties
file for the metadata.log.dir
. Replica UUID and directory id was first introduced in KIP-858: Handle JBOD broker disk failure in KRaft.
There are two cases when a directory id will be generated and persisted:
kafka-storage
format command will generate a directory id for all of the log directories including the metadata log dir.meta.properties
exists but it doesn't include adirectory.id
property. This case will generate and persist a directory id to support upgrade from Kafka versions that don't support this feature to versions that support this feature.
Supported features
This feature can only be enabled if all of the voters (controllers) and observers (brokers) support this feature. This is required mainly because this feature needs to write a new control record (VotersRecord
) to the KRaft cluster metadata partition. All replicas need to be able to read and decode these new control records.
There will be a new SupportedFeature added to the ApiVersions response of the Kafka nodes. The name of this new supported feature will be kraft.version
. The default value be 0 and represents that only KIP-595: A Raft Protocol for the Metadata Quorum, KIP-630: Kafka Raft Snapshot and KIP-996: Pre-Vote are supported. Version 1 means that this KIP is supported.
When the clients sends a UpdateFeatures
RPC to the active controller, if the FeatureUpdates.Feature
property is kraft.version
, the associated information will be passed to KRaft client. The KRaft client will implement two different algorithms to check if the upgrade is supported by voters and observers. For voters, the KRaft leader will compare the upgraded version against all of the voters' supported kraft.version
. The KRaft leader will learn about the supported versions through the UpdateVoter RPC.
The KRaft leader doesn't know the kraft.version
for the observers (brokers) because observers don't send the UpdateVoter request to the leader. The controller state machine will instead push the brokers' kraft.version
information to the KRaft client (RaftClient
in the implementation). The KRaft will use the controller provided kraft.version
to check if an upgrade is supported.
Any UpdateFeature
RPC that attempts to downgrade the kraft.version from 1 to 0 will be rejected.
Voter changes
Adding voters
Voters are added to the cluster metadata partition by sending an AddVoter
RPC to the leader. For safety Kafka will only allow one voter change operation at a time. If there are any pending voter change operations the leader will wait for them to finish.
If there are no pending voter change operations the leader send an ApiVersions request to the new voter's endpoint to discover it's kraft.version
support features. If the new leader supports the current kraft.version
, it will write a VotersRecord
to the log, with the current voters set plus the voter getting added, and immediately update its in-memory quorum state to include this voter as part of the quorum. Any replica that replicates and reads this VotersRecord
will update their in-memory voters set to include this new voter. Voters will not wait for these records to get committed before updating their voters set.
Once the VotersRecord
operation has been committed by the majority of the new voters set, the leader can respond to the AddVoter
RPC and process new voter change operations.
Removing voters
Voter are removed from the cluster metadata partition by sending a RemoveVoter
RPC to the leader. This works similar to adding a voter. If there are no pending voter change operations the leader will append the VotersRecord
to the log, the the current voters set minus the voter getting removed, and immediately update its voters set to the new configuration.
Once the VotersRecord
operation has been committed by the majority of the new voters set, the leader can respond to the RPC. If the removed voters is the leader, the leader will resign from the quorum when the VotersRecord
has been committed. To allow this operation to be committed and for the leader to resign the followers will continue to fetch from the leader even if the leader is not part of the new voters set. In KRaft, leader election is triggered when the voter hasn't received a successful response in the fetch timeout.
Endpoints information
Replicas need to be able to connect and send RPCs to each others to implement the KRaft protocol. There are two important sets of endpoints that are needed to allow the replicas to connect to each other.
Voters set
The voters set is the collection of replicas and their endpoints that are part that are voters and can become leaders. The voters set is stored in the snapshot and log using the VotersRecord
control record. If the partition doesn't contain a VotersRecord
control record then the value stored in the controller.quorum.voters
property will be used.
This set of endpoints and replicas will be use by the Vote
, BeginQuorumEpoch
and EndQuorumEpoch
RPCs. In other words, replicas will use the voters set to establish leadership and to propagate leadership information to all of the voters.
Bootstrap servers
When the leader is known Fetch
requests will be sent to the leader's endpoint. The leader's endpoint can be discovered from many of the KRaft RPCs but typically it will be discover from the BeginQuorumEpoch
request for voters (controllers) or the Fetch
response for observers (brokers).
In the case that the leader is not known the replica (observer) will send the Fetch
RPC to the endpoints in the bootstrap servers configuration. The set of endpoints in the bootstrap servers configuration will come from the controller.quorum.bootstrap.servers
configuration property. If that property is not specified the endpoints in controller.quorum.voters
will be used.
Leader election
It is possible for the leader to add a new voter to the voters set, write the VotersRecord
to the log and only replicate it to some of the voters in the new configuration. If the leader fails before this record has been replicated to the new voter it is possible that a new leader cannot be elected. This is because voters reject vote request from replicas that are not in the voters set. This check will be removed and replicas will reply to Vote
request when the candidate is not in the voters set or the voting replica is not in the voters set. The candidate must still have a longer log (offset and epoch) before the voter will grant a vote to the candidate.e
Once a leader is elected, leader will propagate this information to all of the voters using the BeginQuorumEpoch
RPC. The leader will continue to send the BeginQuorumEpoch
requests to a voter if the voter doesn't send a Fetch
or FetchSnapshot
request within the "check quorum" timeout.
First leader
When a KRaft voter becomes leader it will write a KRaftVersionRecord
and VotersRecord
to the log if the log or the latest snapshot doesn't contain any VotersRecord. This is done to make sure that the voters set in the bootstrap snapshot gets replicated to all of the voters and to not rely on all of the voters being configured with the same bootstrapped voters set.
Automatic endpoint and directory id discovery
To improve the usability of this feature it would beneficial for the leader of the KRaft cluster metadata partition to automatically rediscover the voters' endpoints. This makes it possible for the operator to update the endpoint of a voter without having to use the kafka-metadata-quorum
tool. When a voter becomes a follower and discovers a new leader, it will always send an UpdateVoter RPC to the leader. This request instructs the leader to update the endpoints of the matching replica id and replica uuid. When a voter becomes a leader it will also write a VotersRecord control record with the updated endpoints and kraft.version
feature.
The directory id, or replica uuid, will behave differently. The quorum shouldn't automatically update the directory id, since different values means that the disk was replaced. For directory id, the leader will only override it if it was not previously set. This behavior is useful for when a cluster gets upgraded to a kraft.version
greater than 0.
High watermark
As described in KIP-595, the high-watermark will be calculated using the fetch offset of the majority of the voters. When a replica is removed or added it is possible for the high-watermark to decrease. The leader will not allow the high-watermark to decrease and will guarantee that is is monotonically increasing for both the state machines and the remote replicas.
With this KIP, it is possible for the leader to not be part of the voters set when the replica removed is the leader. In this case the leader will continue to handle Fetch and FetchSnapshot request as normal but it will not count itself when computing the high watermark.
Snapshots
The snapshot generation code needs to be extended to include these new KRaft specific control record for VotersRecord. Before this KIP the snapshot didn't include any KRaft generated control records. When the state machine (controller or broker) calls RaftClient.createSnapshot, the KRaft client will write the SnapshotHeaderRecord, KRaftVersionRecord and VotersRecord controller records at the beginning of the snapshot.
Note that the current KRaft implementation already writes the SnapshotHeaderRecord. The new KRaft implementation will write KRraftVersionRecord
and VotersRecord
, if the kraft.version
is greater than 0.
To efficiently implement this feature the KRaft implementation will keep track of all voters sets between the latest snapshot and the LEO. The replicas will append the offset and voters set as it reads VotersRecord control records from the snapshot or log. Whenever the replica generates a snapshot for a given offset it can remove any additional entry that has a smaller offset.
Internal listener
This KIP requires that the KRaft implementation now read and decode uncommitted data from log and snapshot to discover the voters set. This also means that the KRaft implementation needs to handle this uncommitted voter sets getting truncated and removed from the log. As the internal listener reads new voters sets and endpoints it needs to update the network client with this new information.
Controller auto joinning
To make it easier for users to operate a KRaft controller cluster, they can have KRaft controllers automatically join the cluster's voters set by setting the controller.quorum.auto.join.enable
to true
. In this case, the controller will remove any voter in the voters set that matches its replica id but doesn't match its directory id (replica uuid) by sending the RemoveVoter
RPC. Once the controller has remove all duplicate replica uuid it will add itself to the voters set by sending a AddVoter
RPC to the leader, if its replica id and replica uuid is not in the voters set. The TimoutMs
for the AddVoter
RPC will be set to 30 seconds.
For an example, imagine that the user is trying to create KRaft controller quorum with 3 voters (1, 2, 3).
On controller 1 the user runs:
Code Block |
---|
kafka-storage format --cluster-id ABC --release-version "3.8" --standalone --config ... |
and starts the controller. When controller 1 starts it sees that is already in the voters set so it doesn't perform any AddVoter and RemoveVoter RPC. Controller 1 eventually becomes leader.
On controller 2 and 3 the user runs:
Code Block |
---|
kafka-storage format --cluster-id ABC --release-version "3.8" --config ... |
and starts the controllers. Notice that neither --standalone
or --controller-quorum-voters
is used for controller 2 and 3 so the controllers start as observers. These controllers will discover the leader using controller.quorum.bootstrap.servers
and will use the RemoveVoter
and AddVoter
RPC as described in the beginning of this section.
Public Interfaces
Configuration
There only two configurations for this feature.
controller.quorum.voters
This is an existing configuration. This configuration describes the state of the quorum and will only be used if the kraft.version feature is 0.
controller.quorum.bootstrap.servers
This is a list of nodes that brokers and new controllers can use to discover the quorum leader. Brokers and new controllers (observers) will send Fetch requests to all of the nodes in this configuration until they discover the quorum leader and the Fetch request succeeds. The quorum voters and their configuration will be learned by fetching and reading the records from the log and snapshot. This includes committed and uncommitted records.
If this configuration is specified, observers will not use the controller.quorum.voters
endpoints to discover the leader.
controller.quorum.auto.join.enable
Controls whether a KRaft controller should automatically join the cluster metadata partition for its cluster id. If the configuration is set to true the controller must be stopped before removing the controller with kafka-metadata-quorum remove-controller
. The default value is false.
Log and snapshot control records
A few new control records will be added to the log and snapshot of a KRaft partition.
KRaftVersionRecord
Control record for recording the latest KRaft protocol version. The KRaft protocol version is used to determine which version of LeaderChangeMessage to write. It is also used to determine if the VotersRecord control record can be written to the topic partition.
The ControlRecordType
is TBD and will be updated when the code is committed to Kafka.
Code Block | ||
---|---|---|
{
"type": "data",
"name": "KRaftVersionRecord",
| ||
Code Block | ||
| ||
git diff upstream/trunk clients/src/main/resources/common/message/LeaderChangeMessage.json diff --git a/clients/src/main/resources/common/message/LeaderChangeMessage.json b/clients/src/main/resources/common/message/LeaderChangeMessage.json index fdd7733388..2b019a2a80 100644 --- a/clients/src/main/resources/common/message/LeaderChangeMessage.json +++ b/clients/src/main/resources/common/message/LeaderChangeMessage.json @@ -16,7 +16,7 @@ { "type": "data", "name": "LeaderChangeMessage", - "validVersions": "0", + "validVersions": "0-1", "flexibleVersions": "0+", "fields": [ { {"name": "Version", "type": "int16", "versions": "0+", @@ -30,7 +30,8 @@ ], "commonStructs": [ { "name": "Voter", "versions": "0+", "fields": [ - {"name": "VoterId", "type": "int32", "versions": "0+"} + "about": "The version of the kraft version record" }, { "name": "VoterIdKRaftVersion", "type": "int32int16", "versions": "0+" }, + { "nameabout": "VoterUuid", "type": "uuid", "versions": "1+"The kraft protocol version" } ]} ] } |
VotersRecord
A control record for specifying the latest and entire voters set. This record will be written to the generated snapshot and will specify the voters set at the snapshot id's end offset.
This record will also be written to the log by the leader when handling the AddVoter
, RemoveVoter
and UpdateVoter
RPCs. When handling these RPCs the leader will make sure that the difference in voters is at most one addition or one removal. This is needed to satisfy the invariant that voter changes are committed by the old voter set and new voter set.
This record will also be written by the leader to the log if it has never been written to the log in the past. This semantic is nice to have to consistently replicate the bootstrapping snapshot, at 00000000000000000000-0000000000.checkpoint
, at the leader to all of the voters.
The ControlRecordType
is TBD and will be updated when the code is committed to Kafka.
} |
Handling
When KRaft replicas read this record from the log they will update their finalized kraft.version
once the HWM shows this record as committed.
LeaderChangeMessage
Add an optional VoterUuid to Voter. This change is not needed for correctness but it is nice to have for tracing and debugging. The leader will write version 0 if the kraft.version
is 0. The leader will write version 1 if the kraft.version
is 1.
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/LeaderChangeMessage.json
diff --git a/clients/src/main/resources/common/message/LeaderChangeMessage.json b/clients/src/main/resources/common/message/LeaderChangeMessage.json
index fdd7733388..2b019a2a80 100644
--- a/clients/src/main/resources/common/message/LeaderChangeMessage.json
+++ b/clients/src/main/resources/common/message/LeaderChangeMessage.json
@@ -16,7 +16,7 @@
{
"type": "data",
| ||
Code Block | ||
{ "type": "data", "name": "VotersRecord", "validVersions": "0", "flexibleVersions": "0+", "fields": [ { "name": "VersionLeaderChangeMessage", - "typevalidVersions": "int160", + "versionsvalidVersions": "0+-1", "aboutflexibleVersions": "The0+", version of the voters record" }, "fields": [ { "name": "VotersVersion", "type": "[]Voterint16", "versionversions": "0+", @@ -30,7 +30,8 @@ ], "fieldscommonStructs": [ { "name": "VoterId", "type": "int32Voter", "versions": "0+", "entityTypefields": "brokerId",[ - {"name": "VoterId", "abouttype": "The replica id of the voter in the topic partition" }, { "name": "VoterUuid", "type": "uuid", "int32", "versions": "0+", "about": "The directory id of the voter in the topic partition" }, + { "name": "EndPointsVoterId", "type": "[]Endpointint32", "versions": "0+", "about": "The endpoint that can be used to communicate with the voter", "fields": [ }, + { "name": "NameVoterUuid", "type": "stringuuid", "versions": "01+", "mapKey": true, } ]} ] } |
VotersRecord
A control record for specifying the latest and entire voters set. This record will be written to the generated snapshot and will specify the voters set at the snapshot id's end offset.
This record will also be written to the log by the leader when handling the AddVoter
, RemoveVoter
and UpdateVoter
RPCs. When handling these RPCs the leader will make sure that the difference in voters is at most one addition or one removal. This is needed to satisfy the invariant that voter changes are committed by the old voters set and new voters set.
This record will also be written by the leader to the log if it has never been written to the log in the past. This semantic is nice to have to consistently replicate the bootstrapping snapshot, at 00000000000000000000-0000000000.checkpoint
, at the leader to all of the voters.
The ControlRecordType
is TBD and will be updated when the code is committed to Kafka.
Code Block |
---|
{ "type": "data", "about": "The name of the endpoint" }, { "name": "Host", "type": "string", "versions": "0+", "about": "The hostname" }, { "name": "PortVotersRecord", "typevalidVersions": "uint160", "versionsflexibleVersions": "0+", "aboutfields": "The port" }, [ { "name": "SecurityProtocolVersion", "type": "int16", "versions": "0+", "about": "The securityversion protocol"of } the voters record" ]}, { "name": "KRaftVersionFeatureVoters", "type": "KRaftVersionFeature[]Voter", "versions": "0+", "fields": [ { "aboutname": "The range of versions of the protocol that the replica supports", "fields": [ VoterId", "type": "int32", "versions": "0+", "entityType": "brokerId", "about": "The replica id of the voter in the topic partition" }, { "name": "MinSupportedVersionVoterUuid", "type": "int16uuid", "versions": "0+", "about": "The minimum supported KRaft protocol versiondirectory id of the voter in the topic partition" }, { "name": "MaxSupportedVersionEndPoints", "type": "int16[]Endpoint", "versions": "0+", "about": "The maximum supported KRaft protocol version" } endpoint that can be used to communicate with the voter", "fields": [ ]} { ]} ] } |
Handling
KRaft replicas will read all of the control records in the snapshot and the log irrespective of the commit state and HWM. When a replica encounters a VotersRecord it will replace the current voter set.
If the local replica is the leader and it is getting removed, the replica will stay leader until the VotersRecord
gets committed or the epoch advances (which forces it to lose leadership).
If the local replica is getting added to the voter set, it will allow the transition to prospective candidate when the fetch timer expires. The fetch timer is reset whenever it receives a successful Fetch
or FetchSnapshot
response.
Quorum State
Each KRaft topic partition has a quorum state (QuorumStateData
) that gets persisted in the quorum-state
file in the directory for the topic partition. The following changes will be made to this state:
- ClusterId will get removed in version 1. This field is not used because the cluster id is persisted in
<metadata.log.dir>/meta.properties
. - AppliedOffset will get removed in version 1. This field is not used in version 0. The field will get removed because it implied an inclusive offset. This is not correct because the default value is 0.
- VoterUuid will get added in version 1. The voting replica will persist both the voter ID and UUID of the candidate for which it voted.
- AppliedRecord will get added in version 1. This is the newest record that the KRaft client has read and applied to update the current set of voters. EndOffset is exclusive and represents the next offset that the KRaft client should read if available.
- CurrentVoters will get extended in version 1 to include the voter's directory id or UUID.
Version 0 of this data will get written if the kraft.version
is 0. Version 1 of this data will get written if the kraft.version
is 1.
If a candidate sends a vote request with a replica UUID (version 2 of the RPC) but the voter's kraft.version is at 0, the voter will not persist the voted UUID and use version 0 of quorum data. The voter will continue to track the voted UUID in memory. This mean that after a restart the voter can vote for a different UUID but this should be rare and can only happing while the protocol is getting upgraded.
"name": "Name", "type": "string", "versions": "0+", "mapKey": true,
"about": "The name of the endpoint" },
{ "name": "Host", "type": "string", "versions": "0+",
"about": "The hostname" },
{ "name": "Port", "type": "uint16", "versions": "0+",
"about": "The port" }
]},
{ "name": "KRaftVersionFeature", "type": "KRaftVersionFeature", "versions": "0+",
"about": "The range of versions of the protocol that the replica supports", "fields": [
{ "name": "MinSupportedVersion", "type": "int16", "versions": "0+",
"about": "The minimum supported KRaft protocol version" },
{ "name": "MaxSupportedVersion", "type": "int16", "versions": "0+",
"about": "The maximum supported KRaft protocol version" }
]}
]}
]
} |
Handling
KRaft replicas will read all of the control records in the snapshot and the log irrespective of the commit state and HWM. When a replica encounters a VotersRecord
it will replace the current voters set.
If the local replica is the leader and it is getting removed, the replica will stay leader until the VotersRecord
gets committed or the epoch advances (which forces it to lose leadership).
If the local replica is getting added to the voters set, it will allow the transition to prospective candidate when the fetch timer expires. The fetch timer is reset whenever it receives a successful Fetch
or FetchSnapshot
response.
Quorum state
Each KRaft topic partition has a quorum state (QuorumStateData
) that gets persisted in the quorum-state
file in the directory for the topic partition. The following changes will be made to this state:
- ClusterId will get removed in version 1. This field is not used because the cluster id is persisted in
<metadata.log.dir>/meta.properties
. - AppliedOffset will get removed in version 1. This field is not used in version 0.
- VotedUuid will get added in version 1. The voting replica will persist both the voter ID and UUID of the candidate for which it voted.
- CurrentVoters will get removed in version 1. This field was only used to validate that the controller.quorum.voters value didn't change. In kraft.version 1 the set of voters will be stored in the snapshot and log instead.
Version 0 of this data will get written if the kraft.version
is 0. Version 1 of this data will get written if the kraft.version
is 1.
If a candidate sends a vote request with a replica UUID (version 2 of the RPC) but the voter's kraft.version is at 0, the voter will not persist the voted UUID and use version 0 of quorum data. The voter will continue to track the voted UUID in memory. This mean that after a restart the voter can vote for a different UUID but this should be rare and can only happing while the protocol is getting upgraded.
Code Block | ||
---|---|---|
| ||
% git diff upstream/trunk raft/src/main/resources/common/message/QuorumStateData.json
diff --git a/raft/src/main/resources/common/message/QuorumStateData.json b/raft/src/main/resources/common/message/QuorumStateData.json
index d71a32c75d..1ae14ec843 100644
--- a/raft/src/main/resources/common/message/QuorumStateData.json
+++ b/raft/src/main/resources/common/message/QuorumStateData.json
@@ -16,19 +16,17 @@
{
"type": "data",
"name": "QuorumStateData",
- "validVersions": "0",
+ "validVersions": "0-1",
"flexibleVersions": "0+",
"fields": [ | ||
Code Block | ||
| ||
git diff upstream/trunk raft/src/main/resources/common/message/QuorumStateData.json diff --git a/raft/src/main/resources/common/message/QuorumStateData.json b/raft/src/main/resources/common/message/QuorumStateData.json index d71a32c75d..70ff2d6f42 100644 --- a/raft/src/main/resources/common/message/QuorumStateData.json +++ b/raft/src/main/resources/common/message/QuorumStateData.json @@ -16,19 +16,43 @@ { "type": "data", "name": "QuorumStateData", - "validVersions": "0", + "validVersions": "0-1", "flexibleVersions": "0+", "fields": [ - {"name": "ClusterId", "type": "string", "versions": "0+"}, - {"name": "LeaderId", "type": "int32", "versions": "0+", "default": "-1"}, - {"name": "LeaderEpoch", "type": "int32", "versions": "0+", "default": "-1"}, - {"name": "VotedId", "type": "int32", "versions": "0+", "default": "-1"}, - {"name": "AppliedOffsetClusterId", "type": "int64string", "versions": "0+"}, - {"name": "CurrentVotersLeaderId", "type": "[]Voterint32", "versions": "0+", "nullableVersionsdefault": "0+-1"}, +- { "name": "ClusterIdLeaderEpoch", "type": "stringint32", "versions": "0+", "default": "-1"}, +- { "name": "LeaderIdVotedId", "type": "int32", "versions": "0+", "default": "-1" }, +- { "name": "LeaderEpochAppliedOffset", "type": "int32int64", "versions": "0+", "default": "-1" }, +- { "name": "VotedIdCurrentVoters", "type": "int32[]Voter", "versions": "0+", "defaultnullableVersions": "-1" }, +0+"} - ], - "commonStructs": [ - { "name": "VotedUuidVoter", "typeversions": "uuid0+", "versionsfields": "1+" }, +[ - { "name": "AppliedOffsetVoterId", "type": "int64int32", "versions": "0+" }, + { "name": "AppliedRecordClusterId", "type": "AppliedRecordstring", "versions": "1+0", "fields": [ }, + { "name": "EndOffsetLeaderId", "type": "int64int32", "versions": "1+0+", "default": "-1" }, + { "name": "EpochLeaderEpoch", "type": "int32", "versions": "10+" } + ], "default": "-1" }, + { "name": "CurrentVotersVotedId", "type": "[]Voterint32", "versions": "0+", "nullableVersionsdefault": "0+-1" } ], "commonStructs": [ { "name": "Voter", "versions": "0+", "fields": [ - {"name": "VoterId", "type": "int32", "versions": "0+"} + { "name": "VoterIdVotedUuid", "type": "int32uuid", "versions": "01+" }, + { "name": "VoterUuidAppliedOffset", "type": "uuidint64", "versions": "1+0" }, + { "name": "EndPointsCurrentVoters", "type": "[]EndpointVoter", "versions": "1+0", + "aboutnullableVersions": "The endpoint that can be used to communicate with the voter0", "fields": [ + { "name": "NameVoterId", "type": "stringint32", "versions": "1+0", "mapKey": true, + } ]} "about": "The name of the endpoint" }, + { "name": "Host", "type": "string", "versions": "1+", + "about": "The hostname" }, + { "name": "Port", "type": "uint16", "versions": "1+", + "about": "The port" }, + { "name": "SecurityProtocol", "type": "int16", "versions": "1+", + "about": "The security protocol" } + ]}, + { "name": "KRaftVersionFeature", "type": "KRaftVersionFeature", "versions": "1+", + "about": "The range of versions of the protocol that the replica supports", "fields": [ + { "name": "MinSupportedVersion", "type": "int16", "versions": "1+", + "about": "The minimum supported KRaft protocol version" }, + { "name": "MaxSupportedVersion", "type": "int16", "versions": "1+", + "about": "The maximum supported KRaft protocol version" } + ]} ]} ] } |
RPCs
UpdateFeatures
The schema and version for this API will not change but the handling of the RPC will be updated to handle the kraft.version
feature.
Handling
The controller node will forward any changes to the kraft.version
to the RaftClient.
When upgrading the version, the RaftClient will:
- Verify that the current set of voters support the new version. The leader will learn the range of supported version from the UpdateVoter RPC. See that section for more details.
- Verify that all of the observers (brokers) support the new version. The leader will learn range of supported version from the QuorumController. The quorum controller will update the RaftClient whenever the broker registration changes.
- If supported, the RaftClient will atomically write a control record batch to the log with all of the KRaftVersion and the
VotersRecord
control records. The VotersRecord controller records only need to get written when updating from version 0 to version 1. - Reply to the RPC when the control record batch is committed.
AddVoter
This RPC can be sent by an administrative client to add a voter to the set of voters. This RPC can be sent to a broker or controller, when sent to a broker, the broker will forward the request to the active controller.
Handling
When the leader receives an AddVoter request it will do the following:
- Wait for the fetch offset of the replica (ID, UUID) to catch up to the log end offset of the leader.
- Wait until there are no uncommitted VotersRecord. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Wait for the LeaderChangeMessage control record from the current epoch to get committed. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Send an ApiVersions RPC to the first listener to discover the supported
kraft.version
of the new voter. - Check that the new voter supports the current
kraft.version
. - Append the updated VotersRecord to the log.
- The KRaft internal listener will read this record from the log and add the voter to the voters set.
- Wait for the VotersRecord to commit using the majority of new voters set.
- Send the AddVoter response to the client.
In 1., the leader needs to wait for the replica to catch up because when the VotersRecord is appended to the log, the set of voter changes. If the new voter is too far behind then it can take some time for it to reach the HWM. During this time the leader cannot commit data and the quorum will be unavailable from the perspective of the state machine. We mitigate this by waiting for the new replica to catch up before adding it to the set of voters.
Here is an example, where waiting for the new voter to catch up to the LEO is helpful. Let's assume that there are 3 voters with only two of them at the LEO or HWM.
Voter 1: LEO = 200
Voter 2: LEO = 200
Voter 3: LEO = 100
In this example the HWM is 200. To advance the HWM past 200 only voter 1 and 2 need to Fetch the new records. For voter 3 to be counted towards advancing the HWM it needs to Fetch to or past offset 200 (100 records in this example).
At this point if the user decides to add a new voter (4) to the voters set and the leader doesn't wait for voter (4) to catch up to the LEO, the voter set may become:
Voter 1: LEO = 200
Voter 2: LEO = 200
Voter 3: LEO = 100
Voter 4: LEO = 0
In this state the majority must include either voter 3, voter 4 or both voters 3 and 4. This means that leader cannot advance the HWM until at least either voter 3 or voter 4 have caught up to the HWM. In other, the KRaft topic partition won't be able to commit new records until at least one of those two voters (3 or 4) have caught.
In 3., the leader will wait for its current epoch to commit by waiting for the LeaderChangeMessage to commit. This is required to guarantee that two competing voters set, the one from a previous leader and the one from the current leader, only differ by at most one voter. Waiting for the current epoch to commit means that there cannot be some other competing voter set from another leader that can later override this leader's new voter set. See bug in single-server membership changes for more details on this.
In 4., the new replica will be part of the quorum so the leader will start sending BeginQuorumEpoch requests to this replica. It is possible that the new replica has not yet replicated and applied this VotersRecord so it doesn't know that it is a voter for this topic partition. The new replica will accept the BeginQuorumEpoch RPC even if it is doesn't believe it is a member of the voter set.
The replica will return the following errors:
- NOT_LEADER_FOR_PARTITION - when the request is sent to a replica that is not the leader.
- VOTER_ALREADY_ADDED - when the request contains a replica ID and UUID that is already in the committed voter set.
- UNSUPPORTED_VERSION - when the
kraft.version
is not greater than 1. - INVALID_REQUEST - when the new voter doesn't support the current
kraft.version
. - REQUEST_TIMED_OUT - when the new voter didn't catch-up to the LEO in the time specified in the request.
Request
]
} |
RPCs
UpdateFeatures
The schema and version for this API will not change but the handling of the RPC will be updated to handle the kraft.version
feature.
Handling
The controller node will forward any changes to the kraft.version
to the RaftClient.
When upgrading the version, the RaftClient will:
- Verify that the current set of voters support the new version. The leader will learn the range of supported version from the UpdateVoter RPC. See that section for more details.
- Verify that all of the observers (brokers) support the new version. The leader will learn range of supported version from the QuorumController. The quorum controller will update the RaftClient whenever the broker registration changes.
- If supported, the RaftClient will atomically write a control record batch to the log with all of the KRaftVersion and the
VotersRecord
control records. The VotersRecord controller records only need to get written when updating from version 0 to version 1. - Reply to the RPC when the control record batch is committed.
Downgrading to the kraft.version from 1 to 0 will be rejected with INVALID_UPDATE_VERSION.
AddVoter
This RPC can be sent by an administrative client to add a voter to the set of voters. This RPC can be sent to a broker or controller, when sent to a broker, the broker will forward the request to the active controller.
Handling
When the leader receives an AddVoter request it will do the following:
- Wait for the fetch offset of the replica (ID, UUID) to catch up to the log end offset of the leader.
- Wait until there are no uncommitted VotersRecord. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Wait for the LeaderChangeMessage control record from the current epoch to get committed. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Send an ApiVersions RPC to the first listener to discover the supported
kraft.version
of the new voter. - Check that the new voter supports the current
kraft.version
. - Append the updated VotersRecord to the log.
- The KRaft internal listener will read this record from the log and add the voter to the voters set.
- Wait for the VotersRecord to commit using the majority of new voters set. Return a REQUEST_TIMED_OUT error if it doesn't succeed in time.
- Send the AddVoter response to the client.
In 1., the leader needs to wait for the replica to catch up because when the VotersRecord is appended to the log, the set of voter changes. If the new voter is too far behind then it can take some time for it to reach the HWM. During this time the leader cannot commit data and the quorum will be unavailable from the perspective of the state machine. We mitigate this by waiting for the new replica to catch up before adding it to the set of voters.
Here is an example, where waiting for the new voter to catch up to the LEO is helpful. Let's assume that there are 3 voters with only two of them at the LEO or HWM.
Voter 1: LEO = 200
Voter 2: LEO = 200
Voter 3: LEO = 100
In this example the HWM is 200. To advance the HWM past 200 only voter 1 and 2 need to Fetch the new records. For voter 3 to be counted towards advancing the HWM it needs to Fetch to or past offset 200 (100 records in this example).
At this point if the user decides to add a new voter (4) to the voters set and the leader doesn't wait for voter (4) to catch up to the LEO, the voters set may become:
Voter 1: LEO = 200
Voter 2: LEO = 200
Voter 3: LEO = 100
Voter 4: LEO = 0
In this state the majority must include either voter 3, voter 4 or both voters 3 and 4. This means that leader cannot advance the HWM until at least either voter 3 or voter 4 have caught up to the HWM. In other words, the KRaft topic partition won't be able to commit new records until at least one of those two voters (3 or 4) are caught up to the HWM/LEO.
In 3., the leader will wait for its current epoch to commit by waiting for the LeaderChangeMessage to commit. This is required to guarantee that the two competing voters sets, the one from a previous leader and the one from the current leader, only differ by at most one voter. Waiting for the current epoch to commit means that there cannot be some other competing voters set from another leader that can later override this leader's new voters set. See bug in single-server membership changes for more details on this.
In 7., the new replica will be part of the quorum so the leader will start sending BeginQuorumEpoch requests to this replica. It is possible that the new replica has not yet replicated and applied this VotersRecord so it doesn't know that it is a voter for this topic partition. The new replica will accept the BeginQuorumEpoch RPC even if it is doesn't believe it is a member of the voters set.
The replica will return the following errors:
- NOT_LEADER_FOR_PARTITION - when the request is sent to a replica that is not the leader.
- DUPLICATE_VOTER - when the request contains a replica id is already in the committed voters set. Note that this means that duplicate replica ids are not allowed. This is useful to make automatic voter addition safer.
- UNSUPPORTED_VERSION - when the
kraft.version
is not greater than 1. - INVALID_REQUEST - when the new voter doesn't support the current
kraft.version
. - REQUEST_TIMED_OUT - when the new voter didn't catch-up to the LEO in the time specified in the request.
Request
Code Block | ||
---|---|---|
| ||
{
"apiKey": 76,
"type": "request",
"listeners": ["controller", "broker"],
"name": "AddVoterRequest",
"validVersions": "0",
"flexibleVersions": "0+",
"fields": [
{ "name": "ClusterId", "type": "string", "versions": "0+" },
{ "name": "TimeoutMs", "type": "int32", "versions": "0+" },
{ "name": "TopicName", "type": "string", "versions": "0+", "entityType": "topicName",
"about": "The name of the topic" },
{ "name": "TopicId", "type": "uuid", "versions": "0+",
"about": "The unique topic ID" },
{ "name": "Partition", "type": "int32", "versions": "0+",
"about": "The partition index" },
{ "name": "VoterId", "type": "int32", "versions": "0+",
"about": "The replica id of the voter getting added to the topic partition" }, | ||
Code Block | ||
| ||
{ "apiKey": 75, "type": "request", "listeners": ["controller", "broker"], "name": "AddVoterRequest", "validVersions": "0", "flexibleVersions": "0+", "fields": [ { "name": "ClusterIdVoterUuid", "type": "stringuuid", "versions": "0+" }, { "nameabout": "TimeoutMs", "type": "int32", "versions": "0+"The directory id of the voter getting added to the topic partition" }, { "name": "TopicNameListeners", "type": "string[]Listener", "versions": "0+", "entityType": "topicName", "about": "The name ofendpoints that can be used to communicate with the topic" }, voter", "fields": [ { "name": "TopicIdName", "type": "uuidstring", "versions": "0+", "mapKey": true, "about": "The uniquename of topicthe IDendpoint" }, { "name": "PartitionHost", "type": "int32string", "versions": "0+", "about": "The partition indexhostname" }, { "name": "VoterIdPort", "type": "int32uint16", "versions": "0+", "about": "The replicaport" id} of the voter getting]} ] } |
Response
Code Block | ||
---|---|---|
| ||
{ "apiKey": 76added to the topic partition" }, "type": "response", { "name": "VoterUuidAddVoterResponse", "typevalidVersions": "uuid0", "versionsflexibleVersions": "0+", "aboutfields": "The directory id of the voter getting added to the topic partition" },[ { "name": "ListenersErrorCode", "type": "[]Listenerint16", "versions": "0+", "about": "The endpointserror thatcode, canor be0 usedif tothere communicatewas withno theerror" voter", "fields": [ }, { "name": "NameErrorMessage", "type": "string", "versions": "0+", "mapKey"nullableVersions": "0+", "ignorable": true, "about": "The name of the endpoint error message, or null if there was no error." }, { "name": "HostCurrentLeader", "type": "stringLeaderIdAndEpoch", "versions": "0+", "about"taggedVersions": "0+", "tag": "The hostname" },0, "fields": [ { "name": "PortLeaderId", "type": "uint16int32", "versions": "0+", "default": "-1", "entityType" : "brokerId", "about": "The port replica id of the current leader or -1 if the leader is unknown" }, { "name": "SecurityProtocolLeaderEpoch", "type": "int16int32", "versions": "0+", "default": "-1", "about": "The latest securityknown leader protocolepoch" } ]}, ] } |
Response
Code Block | ||
---|---|---|
| ||
{ "apiKey": 75, "type { "name": "responseNodeEndpoint", "nametype": "AddVoterResponseNodeEndpoint", "validVersionsversions": "0+", "flexibleVersionstaggedVersions": "0+", "fieldstag": [1, { "nameabout": "ErrorCode", "type": "int16", "versions": "0+", "about": "The error code, or 0 if there was no error" }, Endpoint for current leader of the topic partition", "fields": [ { "name": "CurrentLeaderHost", "type": "LeaderIdAndEpochstring", "versions": "0+", "taggedVersionsabout": "0+", "tag": 0, "fields": [The node's hostname" }, { "name": "LeaderIdPort", "type": "int32", "versions": "0+", "default": "-1", "entityType" : "brokerId", "about": "The replica id of the current leader or -1 if the leader is unknownnode's port" }, { "name": "LeaderEpoch", "type": "int32", "versions": "0+", "default": "-1", "about": "The latest known leader epoch" } ]}, { "name": "NodeEndpoint", "type": "NodeEndpoint", "versions": "0+", "taggedVersions": "0+", "tag": 1, "about": "Endpoint for current leader of the topic partition", "fields": [ { "name": "Host", "type": "string", "versions": "0+", "about": "The node's hostname" }, { "name": "Port", "type": "int32", "versions": "0+", "about": "The node's port" } ]} ] } |
RemoveVoter
This RPC can be sent by an administrative client to remove a voter from the set of voters. This RPC can be sent to a broker or controller. The broker will forward the request to the controller.
Handling
When the leader receives a RemoveVoter request it will do the following:
- Wait until there are no uncommitted VotersRecord. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Wait for the LeaderChangeMessage control record from the current epoch to get committed. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Append the VotersRecord to the log with the updated voters set.
- The KRaft internal listener will read this record from the log and remove the voter from the voter set.
- Wait for the VotersRecord to commit using the majority of new configuration.
- Send the RemoveVoter response to the client.
- Resign by sending EndQuorumEpoch RPCs if the removed replica is the leader.
In 3. and 4. it is possible for the VotersRecord would remove the current leader from the voters set. In this case the leader needs to allow Fetch and FetchSnapshot requests from replicas. The leader should not count itself when determining the majority and determining if records have been committed.
The replica will return the following errors:
- NOT_LEADER_FOR_PARTITION - when the request is sent to a replica that is not the leader.
- VOTER_ALREADY_REMOVED - when the request contains a replica ID and UUID that is already not in the committed voter set.
- UNSUPPORTED_VERSION - when the
kraft.version
is not greater than 1. - REQUEST_TIMED_OUT
Request
]}
]
} |
RemoveVoter
This RPC can be sent by an administrative client to remove a voter from the set of voters. This RPC can be sent to a broker or controller. The broker will forward the request to the controller.
Handling
When the leader receives a RemoveVoter request it will do the following:
- Wait until there are no uncommitted VotersRecord. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Wait for the LeaderChangeMessage control record from the current epoch to get committed. Note that the implementation may just return a REQUEST_TIMED_OUT error if there are pending operations.
- Append the VotersRecord to the log with the updated voters set.
- The KRaft internal listener will read this record from the log and remove the voter from the voters set.
- Wait for the VotersRecord to commit using the majority of new configuration. Return a REQUEST_TIMED_OUT error if it doesn't succeed in time.
- Send the RemoveVoter response to the client.
- Resign by sending EndQuorumEpoch RPCs if the removed replica is the leader.
In 3. and 4. it is possible for the VotersRecord would remove the current leader from the voters set. In this case the leader needs to allow Fetch and FetchSnapshot requests from replicas. The leader should not count itself when determining the majority and determining if records have been committed.
The replica will return the following errors:
- NOT_LEADER_FOR_PARTITION - when the request is sent to a replica that is not the leader.
- VOTER_NOT_FOUND - when the request contains a replica ID and UUID that is already not in the committed voters set.
- UNSUPPORTED_VERSION - when the
kraft.version
is not greater than 1. - REQUEST_TIMED_OUT
Request
Code Block | ||
---|---|---|
| ||
{
"apiKey": 77,
"type": "request",
" | ||
Code Block | ||
| ||
{
"apiKey": 76,
"type": "request",
"listeners": ["controller", "broker"],
"name": "RemoveVoterRequest",
"validVersions": "0",
"flexibleVersions": "0+",
"fields": [
{ "name": "ClusterId", "type": "string", "versions": "0+" },
{ "name": "TopicName", "type": "string", "versions": "0+", "entityType": "topicName",
"about": "The name of the topic" },
{ "name": "TopicId", "type": "uuid", "versions": "0+",
"about": "The unique topic ID" },
{ "name": "Partition", "type": "int32", "versions": "0+",
"about": "The partition index" },
{ "name": "VoterId", "type": "int32", "versions": "0+",
"about": "The replica id of the voter getting removed from the topic partition" },
{ "name": "VoterUuid", "type": "uuid", "versions": "0+",
"about": "The directory id of the voter getting removed from the topic partition" }
]
} |
...
Code Block | ||
---|---|---|
| ||
{ "apiKey": 7677, "type": "response", "name": "RemoveVoterResponse", "validVersions": "0", "flexibleVersions": "0+", "fields": [ { "name": "ErrorCode", "type": "int16", "versions": "0+", "about": "The error code, or 0 if there was no error" }, { "name": "CurrentLeaderErrorMessage", "type": "LeaderIdAndEpochstring", "versions": "0+", "taggedVersionsnullableVersions": "0+", "tagignorable": 0,true, "fieldsabout": [ "The error message, or null if there was no error." }, { "name": "LeaderIdCurrentLeader", "type": "int32LeaderIdAndEpoch", "versions": "0+", "defaulttaggedVersions": "-10+", "entityTypetag": :0, "brokerIdfields",: [ { "aboutname": "LeaderId"The replica , "type": "int32", "versions": "0+", "default": "-1", "entityType" : "brokerId", "about": "The replica id of the current leader or -1 if the leader is unknown" }, { "name": "LeaderEpoch", "type": "int32", "versions": "0+", "default": "-1", "about": "The latest known leader epoch" } ]}, { "name": "NodeEndpoint", "type": "NodeEndpoint", "versions": "0+", "taggedVersions": "0+", "tag": 1, "about": "Endpoint for current leader of the topic partition", "fields": [ { "name": "Host", "type": "string", "versions": "0+", "about": "The node's hostname" }, { "name": "Port", "type": "int32", "versions": "0+", "about": "The node's port" } ]} ] } |
...
The voter will always send the UpdateVoter RPC whenever it starts and whenever the leader changes. The voter will continue to send the UpdateVoter RPC until the call as has been acknowledge by the current leader.
...
- NOT_LEADER_FOR_PARTITION - when the request is sent to a replica that is not the leader.
- VOTER_NOT_FOUND - when the request contains a replica ID and UUID that is not in the committed voters set.
- UNSUPPORTEDINVALID_VERSION UPDATE - when the minimum and maximum supported
kraft.version
is not greater than 1 in the request doesn't include the finalized kraft.version. - REQUEST_TIMED_OUT
Request
Code Block |
---|
{ "apiKey": 7778, "type": "request", "listeners": ["controller"], "name": "UpdateVoterRequest", "validVersions": "0", "flexibleVersions": "0+", "fields": [ { "name": "ClusterId", "type": "string", "versions": "0+" }, { "name": "TopicName", "type": "string", "versions": "0+", "entityType": "topicName", "about": "The name of the topic" }, { "name": "TopicId", "type": "uuid", "versions": "0+", "about": "The unique topic ID" }, { "name": "Partition", "type": "int32", "versions": "0+", "about": "The partition index" }, { "name": "VoterId", "type": "int32", "versions": "0+", "about": "The replica id of the voter getting updated in the topic partition" }, { "name": "VoterUuid", "type": "uuid", "versions": "0+", "about": "The directory id of the voter getting updated in the topic partition" }, { "name": "Listeners", "type": "[]Listener", "versions": "0+", "about": "The endpoint that can be used to communicate with the leader", "fields": [ { "name": "Name", "type": "string", "versions": "0+", "mapKey": true, "about": "The name of the endpoint" }, { "name": "Host", "type": "string", "versions": "0+", "about": "The hostname" }, { "name": "Port", "type": "uint16", "versions": "0+", "about": "The port" }, ]}, { "name": "SecurityProtocolKRaftVersionFeature", "type": "int16KRaftVersionFeature", "versions": "0+", "about": "The security protocol" } ]}, range of versions of the protocol that the replica supports", "fields": [ { "name": "MinSupportedVersion", "type": "int16", "versions": "0+", { "name": "KRaftVersionFeature", "type": "KRaftVersionFeature", "versions": "0+", "about": "The range of versions of the protocol that the replica supports", "fields": [ { "name": "MinSupportedVersion", "type": "int16", "versions": "0+", "about": "The minimum supported KRaft protocol version" }, { "name": "MaxSupportedVersion", "type": "int16", "versions": "0+", "about": "The maximum supported KRaft protocol version" } ]} ] } |
Response
about": "The minimum supported KRaft protocol version" },
{ "name": "MaxSupportedVersion", "type": "int16", "versions": "0+",
"about": "The maximum supported KRaft protocol version" }
]}
]
} |
Response
Code Block |
---|
{
"apiKey": 78,
"type": "response",
"name": "UpdateVoterResponse",
"validVersions": "0",
"flexibleVersions": "0+",
"fields": [
{ "name": "ErrorCode", "type": "int16", "versions": "0+",
"about": "The error code, or 0 if there was no error" },
{ "name": "CurrentLeader", "type": "LeaderIdAndEpoch", "versions": "0+", "taggedVersions": "0+", "tag": 0, "fields": [
{ "name": "LeaderId", "type": "int32", "versions": "0+", "default": "-1", "entityType" : "brokerId",
"about": "The replica id of the current leader or -1 if the leader is unknown" },
{ "name": "LeaderEpoch", "type": "int32", "versions": "0+", "default": "-1",
"about": "The latest known leader epoch" }
]},
{ "name": "NodeEndpoint", "type": "NodeEndpoint", "versions": "0+", "taggedVersions": "0+", "tag": 1,
"about": "Endpoint for current leader of the topic partition", "fields": [
{ "name": "Host", "type": "string", "versions": "0+", "about": "The node's hostname" },
{ "name": "Port", "type": "int32", "versions": "0+", "about": "The node's port" }
]}
]
} |
Vote
Handling
Since the set of voters can change and not all replicas know the latest voters set, handling of Vote request needs to be relaxed from what was defined and implemented for KIP-595. KRaft replicas will accept Vote requests from all replicas. Candidate replicas don't need to be in the voters' voters set to receive a vote. This is needed to be able to elect a leader from the new voters set even though the new voters set hasn't been replicated to all of its voters.
When the leader removes a voter from the voters set it is possible for the removed voter's Fetch timeout to expire before the replica has replicated the latest VotersRecord. If this happens removed replica will become a candidate, increase its epoch and eventually force the leader to change. To avoid this scenario this KIP is going to rely on KIP-996: Pre-Vote to fenced the removed replica from increasing its epoch:
When servers receive VoteRequests with the
PreVote
field set totrue
, they will respond withVoteGranted
set to
true
if they are not a Follower and the epoch and offsets in the Pre-Vote request satisfy the same requirements as a standard votefalse
if otherwise
The voter will persist both the candidate ID and UUID in the quorum state so that it only votes for at most one candidate for a given epoch.
The replica will return the following new errors:
- INVALID_REQUEST - when the voter ID and UUID doesn't match the local ID and UUID.
- UNSUPPORTED_VERSION - when a non-empty candidate UUID is specified but the voter doesn't support kraft.version 1.
The replica will not return the following errors anymore:
- INCONSISTENT_VOTER_SET - because the voting replica will not validate that either the candidate or the voter are voters.
Request
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/VoteRequest.json
diff --git a/clients/src/main/resources/common/message/VoteRequest.json b/clients/src/main/resources/common/message/VoteRequest.json
index 35583a790b..ff808cbafe 100644
--- a/clients/src/main/resources/common/message/VoteRequest.json
+++ b/clients/src/main/resources/common/message/VoteRequest.json
@@ -18,11 +18,13 @@
"type": "request",
"listeners": ["controller"],
"name": "VoteRequest",
- "validVersions": "0",
+ | ||
Code Block | ||
{ "apiKey": 77, "type": "response", "name": "UpdateVoterResponse", "validVersions": "0-1", "flexibleVersions": "0+", "fields": [ { "name": "ErrorCodeClusterId", "type": "int16string", "versions": "0+", "aboutnullableVersions": "The error code0+", or 0 if there was no error" "default": "null"}, + { "name": "CurrentLeaderVoterId", "type": "LeaderIdAndEpochint32", "versions": "01+", "taggedVersionsignorable": true, "default": "0+-1", "tagentityType": 0"brokerId", + "fieldsabout": [ "The replica id of the voter receiving the request" }, { "name": "LeaderIdTopics", "type": "int32[]TopicData", "versions": "0+", "defaultfields": "-1", "entityType" : "brokerId",[ { "aboutname": "The replica id of the current leader or -1 if the leader is unknown" }, TopicName", "type": "string", "versions": "0+", "entityType": "topicName", @@ -34,14 +36,16 @@ { "name": "LeaderEpochCandidateEpoch", "type": "int32", "versions": "0+", "default": "-1", "about": "The latest known leader epoch" }bumped epoch of the candidate sending the request"}, ]}, { "name": "NodeEndpointCandidateId", "type": "NodeEndpointint32", "versions": "0+", "taggedVersionsentityType": "0+brokerId", - "tagabout": 1, "The ID of the voter sending the request"}, + "about": "EndpointThe forreplica currentid leader of the voter sending topicthe partitionrequest"}, + "fields": [ { "name": "HostCandidateUuid", "type": "stringuuid", "versions": "01+", "about": "The node's hostname + "about": "The directory id of the voter sending the request" }, + { "name": "PortVoterUuid", "type": "int32uuid", "versions": "0+", "about": "The node's port" } ]} ] } |
Vote
Handling
Since the set of voters can change and not all replicas know the latest voter set, handling of Vote request needs to be relaxed from what was defined and implemented for KIP-595.
KRaft replicas will accept Vote requests from all replicas. Candidate replicas don't need to be in the voters' voter set to receive a vote. This is needed to be able to elect a leader from the new voter set even though the new voter set hasn't been replicated to all of its voters.
The voter will persist both the candidate ID and UUID in the quorum state so that it only votes for at most one candidate for a given epoch.
The replica will return the following new errors:
- INVALID_REQUEST - when the voter ID and UUID doesn't match the local ID and UUID.
- UNSUPPORTED_VERSION - when a non-empty candidate UUID is specified but the voter doesn't support kraft.version 1.
The replica will not return the following errors anymore:
- INCONSISTENT_VOTER_SET - because the voting replica will not validate that either the candidate or the voter are voters.
"1+",
+ "about": "The directory id of the voter receiving the request to vote, empty uuid if unknown" },
{ "name": "LastOffsetEpoch", "type": "int32", "versions": "0+",
"about": "The epoch of the last record written to the metadata log"},
{ "name": "LastOffset", "type": "int64", "versions": "0+",
"about": "The offset of the last record written to the metadata log"}
]}
]}
]
}
|
Response
...
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/VoteRequestVoteResponse.json diff --git a/clients/src/main/resources/common/message/VoteRequestVoteResponse.json b/clients/src/main/resources/common/message/VoteRequestVoteResponse.json index 35583a790bb92d0070c1..ff808cbafe21d0aa5312 100644 --- a/clients/src/main/resources/common/message/VoteRequestVoteResponse.json +++ b/clients/src/main/resources/common/message/VoteRequestVoteResponse.json @@ -1817,117 +1817,137 @@ "typeapiKey": "request"52, "listenerstype": ["controllerresponse"], "name": "VoteRequestVoteResponse", - "validVersions": "0", + "validVersions": "0-1", "flexibleVersions": "0+", "fields": [ { "name": "ClusterIdErrorCode", "type": "stringint16", "versions": "0+", @@ -37,9 +37,14 @@ "nullableVersions"about": "The latest known leader epoch"}, { "name": "VoteGranted", "type": "bool", "versions": "0+", "defaultabout": "True if the vote was granted and false "null"otherwise"} ]} ]}, + { "name": "VoterIdNodeEndpoints", "type": "int32[]NodeEndpoint", "versions": "1+", "ignorable": true, "default"taggedVersions": "-1+", "entityTypetag": "brokerId"0, + "about": "TheEndpoints replicafor idall ofcurrent-leaders theenumerated voterin receiving the request" }, PartitionData", "fields": [ + { "name": "TopicsNodeId", "type": "int32", "versions": "[]TopicData1+", + "versions "mapKey": true, "entityType": "0+brokerId", "fieldsabout": [ "The ID of the associated node"}, + { "name": "TopicNameHost", "type": "string", "versions": "01+", "entityTypeabout": "topicName", @@ -34,14 +36,16 @@ The node's hostname" }, + { "name": "CandidateEpochPort", "type": "int32", "versions": "01+", "about": "The bumped epoch of the candidate sending the request"}, "about": "The node's port" } + ]} { "name": "CandidateId", "type": "int32", "versions": "0+", "entityType": "brokerId", - ] } |
BeginQuorumEpoch
Handling
When handling the BeginQuorumEpoch request the replica will accept the request if the LeaderEpoch is equal or greater than their epoch. The receiving replica will not check if the new leader or itself is in the voters set. This change is required because the receiving replica may not have fetched the latest voters set.
The BeginQuorumEpoch RPC will be used by the leader to propagate the leader endpoint to all of the voters. This is needed so that all voters have the necessary information to send Fetch and FetchSnaphsot requests.
The check quorum algorithm implemented by the leader needs to be extended so that BeginQuorumEpoch request are sent to any voter that is not actively fetching (Fetch and FetchSnapshot) from the leader.
The replica will return the following new errors:
- INVALID_REQUEST - when the voter ID and UUID doesn't match the local ID and UUID.
The replica will not return the following errors anymore:
- INCONSISTENT_VOTER_SET - because the voting replica will not validate that either the candidate or the voter are voters.
Request
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/BeginQuorumEpochRequest.json diff --git a/clients/src/main/resources/common/message/BeginQuorumEpochRequest.json b/clients/src/main/resources/common/message/BeginQuorumEpochRequest.json index d9d6d92c88..edd128fb8c 100644 --- a/clients/src/main/resources/common/message/BeginQuorumEpochRequest.json +++ b/clients/src/main/resources/common/message/BeginQuorumEpochRequest.json @@ -18,24 +18,35 @@ "type": "request", "listeners": ["controller"], "name": "BeginQuorumEpochRequest", - "validVersions": "0", - "flexibleVersions": "none", + "validVersions": "0-1", + "flexibleVersions": "1+", "fields": [ "about": "The ID of the voter sending the request"}, + "about": "The replica id of the voter sending the request"}, + { "name": "CandidateUuid", "type": "uuid", "versions": "1+", + "about": "The directory id of the voter sending the request" }, + { "name": "VoterUuid", "type": "uuid", "versions": "1+", + "about": "The directory id of the voter receiving the request to vote, empty uuid if unknown" }, { "name": "LastOffsetEpochClusterId", "type": "int32string", "versions": "0+", "nullableVersions": "0+", "aboutdefault": "The epoch of the last record written to the metadata lognull"}, + { "name": "LastOffsetVoterId", "type": "int64int32", "versions": "01+", "about": "The offset of the last record written to the metadata log"} "entityType": "brokerId", + ]} ]} ] } |
Response
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/VoteResponse.json diff --git a/clients/src/main/resources/common/message/VoteResponse.json b/clients/src/main/resources/common/message/VoteResponse.json index b92d0070c1..21d0aa5312 100644 --- a/clients/src/main/resources/common/message/VoteResponse.json +++ b/clients/src/main/resources/common/message/VoteResponse.json @@ -17,7 +17,7 @@ "apiKey": 52, "type": "response", "name": "VoteResponse", - "validVersions": "0", + "validVersions": "0-1", "flexibleVersions"about": "The voter ID of the receiving replica" }, { "name": "Topics", "type": "[]TopicData", "versions": "0+", "fields": [ { "name": "TopicName", "type": "string", "versions": "0+", "entityType": "topicName", "about": "The topic name" }, { "name": "Partitions", "type": "[]PartitionData", "versions": "0+", "fields": [ { "name": "ErrorCodePartitionIndex", "type": "int16int32", "versions": "0+", @@ -37,9 +37,14 @@ "about": "The latest known leader epoch"partition index" }, + { "name": "VoteGrantedVoterUuid", "type": "booluuid", "versions": "01+", + "about": "True if the vote was granted and false"about": otherwise"} The directory id of the receiving replica" ]}, ]}, + { "name": "NodeEndpointsLeaderId", "type": "[]NodeEndpointint32", "versions": "10+", "taggedVersionsentityType": "1+brokerId", "tag": 0, + "about": "EndpointsThe forID allof current-leadersthe enumeratednewly inelected PartitionDataleader"}, "fields": [ + { "name": "NodeIdLeaderEpoch", "type": "int32", "versions": "10+", + "mapKey": true, "entityType": "brokerId", "about": "The IDepoch of the newly associatedelected nodeleader"}, ]} + ]}, + { "name": "HostNodeEndpoint", "type": "stringNodeEndpoint", "versions": "1+", "taggedVersions": "1+", "tag": 0, + "about": "The node's hostname" },Endpoint for the leader", "fields": [ + { "name": "PortNodeId", "type": "int32", "versions": "1+", + "about": "The node's port" } + ]} ] } |
BeginQuorumEpoch
Handling
When handling the BeginQuorumEpoch request the replica will accept the request if the LeaderEpoch is equal or greater than their epic. The receiving replica will not check if the new leader or itself is in the voter set. This change is required because the receiving may not have fetched the latest voter set.
The BeginQuorumEpoch RPC will be used by the leader to propagate the leader endpoint to all of the voters. This is needed so that all voters have the necessary information to send Fetch and FetchSnaphsot requests.
The check quorum algorithm implemented by the leader needs to be extended so that BeginQuorumEpoch request are sent to any voter that is not actively fetching (Fetch and FetchSnapshot) from the leader.
The replica will return the following new errors:
- INVALID_REQUEST - when the voter ID and UUID doesn't match the local ID and UUID.
The replica will not return the following errors anymore:
- INCONSISTENT_VOTER_SET - because the voting replica will not validate that either the candidate or the voter are voters.
...
"mapKey": true, "entityType": "brokerId", "about": "The ID of the associated node" },
+ { "name": "Host", "type": "string", "versions": "1+", "about": "The node's hostname" },
+ { "name": "Port", "type": "int32", "versions": "1+", "about": "The node's port" }
]}
]
} |
Response
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/BeginQuorumEpochRequestBeginQuorumEpochResponse.json diff --git a/clients/src/main/resources/common/message/BeginQuorumEpochRequestBeginQuorumEpochResponse.json b/clients/src/main/resources/common/message/BeginQuorumEpochRequestBeginQuorumEpochResponse.json index d9d6d92c884b7d7f5a95..edd128fb8c12639bba2f 100644 --- a/clients/src/main/resources/common/message/BeginQuorumEpochRequestBeginQuorumEpochResponse.json +++ b/clients/src/main/resources/common/message/BeginQuorumEpochRequest.json @@ -18,24 +18,35 @@ "type": "request", "listeners": ["controller"], "name": "BeginQuorumEpochRequest", - "validVersions": "0", - "flexibleVersions": "none", + "validVersions": "0-1", + "flexibleVersions": "1+",BeginQuorumEpochResponse.json @@ -17,25 +17,32 @@ "fieldsapiKey": [53, { "nametype": "ClusterIdresponse", "type": "string", "versionsname": "0+BeginQuorumEpochResponse", - "nullableVersionsvalidVersions": "0+", - "defaultflexibleVersions": "nullnone"}, + { "namevalidVersions": "VoterId", "type": "int32", "versions": "1+", "entityType": "brokerId", + "about": "The voter ID of the receiving replica" }, { "name": "Topics", "type": "[]TopicData0-1", + "flexibleVersions": "1+", "versions": "0+", "fields": [ { "name": "TopicNameErrorCode", "type": "stringint16", "versions": "0+", "entityType": "topicName", "about": "The topic name" top level error code"}, { "name": "PartitionsTopics", "type": "[]PartitionDataTopicData", "versions": "0+", "fields": [ { "name": "PartitionIndexTopicName", "type": "int32string", "versions": "0+", "entityType": "topicName", "about": "The partitiontopic indexname" }, + { "name": "VoterUuidPartitions", "type": "uuid", "versions": "1+"[]PartitionData", + "about": "The directory id of the receiving replica" }, { "name": "LeaderId", "type": "int32", "versions": "0+", "entityType": "brokerId", "about": "The ID of the newly elected leader"},fields": [ { "name": "LeaderEpochPartitionIndex", "type": "int32", "versions": "0+", "about": "The epoch of the newly elected leader"}partition index" }, ]} + ]}, + { "name": "NodeEndpointErrorCode", "type": "NodeEndpointint16", "versions": "10+"}, "taggedVersions": "1+", "tag": 0, + { "aboutname": "Endpoint for the leaderLeaderId", "fieldstype": [ + { "name": "NodeId""int32", "typeversions": "int320+", "versionsentityType": "1+brokerId", + "mapKey": true, "entityType": "brokerId", "about": "The ID of the associatedcurrent node" }, + { "name": "Host", "type": "string", "versions": "1+", "about": "The node's hostname" }, +leader or -1 if the leader is unknown"}, { "name": "PortLeaderEpoch", "type": "int32", "versions": "10+", "about": "The node's port" latest known leader epoch"} ]} + ] } |
Response
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/BeginQuorumEpochResponse.json diff --git a/clients/src/main/resources/common/message/BeginQuorumEpochResponse.json b/clients/src/main/resources/common/message/BeginQuorumEpochResponse.json index 4b7d7f5a95..12639bba2f 100644 --- a/clients/src/main/resources/common/message/BeginQuorumEpochResponse.json +++ b/clients/src/main/resources/common/message/BeginQuorumEpochResponse.json @@ -17,25 +17,32 @@ "apiKey": 53, "type": "response", }, + { "name": "NodeEndpoints", "type": "[]NodeEndpoint", "versions": "1+", "taggedVersions": "1+", "tag": 0, + "about": "Endpoints for all leaders enumerated in PartitionData", "fields": [ + { "name": "BeginQuorumEpochResponseNodeId", - "type": "validVersionsint32", "versions": "01+", - "flexibleVersions+ "mapKey": true, "entityType": "brokerId", "about": "none"The ID of the associated node" }, + { "validVersionsname": "0-1Host", + "type": "string", "flexibleVersionsversions": "1+", "about": "fields": [ "The node's hostname" }, + { "name": "ErrorCodePort", "type": "int16int32", "versions": "01+", "about": "The node's port" } "about": "The top level error code"}, { "]} ] } |
EndQuorumEpoch
In addition to the current cases where the leader resigns and sends an EndQuorumEpoch RPC, the leader will also resign and send this RPC once a VotersRecord has committed and the leader is not part of the latest voters set.
Handling
This request will be handle similar to how it is described in KIP-595. The receiving replica will take the directory id (candidate uuid) into account when computing the fetch timeout exponential backoff.
Request
PreferredSuccessor which is an array is replica ids, will be replaced by PreferredCandidates which is an array of the tuple candidate id and candidate uuid.
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/EndQuorumEpochRequest.json diff --git a/clients/src/main/resources/common/message/EndQuorumEpochRequest.json b/clients/src/main/resources/common/message/EndQuorumEpochRequest.json index a6e4076412..d9122fa930 100644 --- a/clients/src/main/resources/common/message/EndQuorumEpochRequest.json +++ b/clients/src/main/resources/common/message/EndQuorumEpochRequest.json @@ -18,8 +18,8 @@ "type": "request", "listeners": ["controller"], "name": "EndQuorumEpochRequest", - "validVersions": "0", - "flexibleVersions": "none", + "validVersions": "0-1", + "flexibleVersions": "1+", "fields": [ name": "Topics", "type": "[]TopicData", "versions": "0+", "fields": [ { "name": "TopicName", "type": "string", "versions": "0+", "entityType": "topicName", "about": "The topic name" }, { "name": "Partitions", "type": "[]PartitionData", "versions": "0+", "fields": [ { "name": "PartitionIndexClusterId", "type": "int32string", "versions": "0+", "nullableVersions": "0+", "aboutdefault": "The partition index" },null"}, @@ -35,8 +35,13 @@ { "nameabout": "ErrorCode", "type": "int16", "versions": "0+The current leader ID that is resigning"}, { "name": "LeaderIdLeaderEpoch", "type": "int32", "versions": "0+", "entityType": "brokerId", "about": "The ID of the current leader or -1 if the leader is unknowncurrent epoch"}, - { "name": "LeaderEpochPreferredSuccessors", "type": "[]int32", "versions": "0+", - "about": "TheA latestsorted knownlist leader epoch"} ]of preferred successors to start the election"} + ]}, + { "name": "NodeEndpointsPreferredSuccessors", "type": "[]NodeEndpointint32", "versions": "1+0", + "taggedVersions": "1+", "tag": 0, + "about": "Endpoints for all leaders enumerated in PartitionData", "fields": [A sorted list of preferred successors to start the election" }, + { "name": "NodeIdPreferredCandidates", "type": "int32[]ReplicaInfo", "versions": "1+", + "mapKey": true, "entityTypeabout": "brokerIdA sorted list of preferred successors to start the election", "aboutfields": "The ID of the associated node" }, +[ + { "name": "HostCandidateId", "type": "stringint32", "versions": "1+", "aboutentityType": "The node's hostnamebrokerId" }, + { "name": "PortCandidateUuid", "type": "int32uuid", "versions": "1+", "about": "The node's port" } + ]} ]} ] } |
EndQuorumEpoch
In addition to the current cases where the leader resigns and sends an EndQuorumEpoch RPC, the leader will also resign and send this RPC once a VotersRecord has committed and the leader is not part of the latest voters set.
Handling
This request will be handle similar to how it is described in KIP-595. The receiving replica will take the directory id (candidate uuid) into account when computing the fetch timeout exponential backoff.
Request
...
}
] |
Response
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/EndQuorumEpochRequestEndQuorumEpochResponse.json diff --git a/clients/src/main/resources/common/message/EndQuorumEpochRequestEndQuorumEpochResponse.json b/clients/src/main/resources/common/message/EndQuorumEpochRequestEndQuorumEpochResponse.json index a6e4076412cd23247045..d9122fa9300d5d61b7e7 100644 --- a/clients/src/main/resources/common/message/EndQuorumEpochRequestEndQuorumEpochResponse.json +++ b/clients/src/main/resources/common/message/EndQuorumEpochRequestEndQuorumEpochResponse.json @@ -1817,8 +1817,8 @@ "typeapiKey": "request"54, "listenerstype": ["controllerresponse"], "name": "EndQuorumEpochRequestEndQuorumEpochResponse", - "validVersions": "0", - "flexibleVersions": "none", + "validVersions": "0-1", + "flexibleVersions": "1+", "fields": [ { "name": "ClusterIdErrorCode", "type": "stringint16", "versions": "0+", "nullableVersions": "0+", "default": "null"}, @@ -35,8 +35,13 @@ "about": "The currenttop leaderlevel ID that is resigningerror code."}, { "name": "LeaderEpoch", "type": "int32", "versions": "0+", "about": "The current epoch"}, -@@ -36,6 +36,13 @@ { "name": "PreferredSuccessorsLeaderEpoch", "type": "[]int32", "versions": "0+", - "about": "A sorted list of preferred successors to start the election"The latest known leader epoch"} ]} + ]}, + { "name": "PreferredSuccessorsNodeEndpoints", "type": "[]int32NodeEndpoint", "versions": "01+", + "taggedVersions": "1+", "tag": 0, + "about": "AEndpoints sortedfor listall ofleaders preferredenumerated successorsin to start the election" }, + PartitionData", "fields": [ + { "name": "PreferredCandidatesNodeId", "type": "[]ReplicaInfoint32", "versions": "1+", + "mapKey": true, "entityType": "brokerId", "about": "AThe sorted listID of preferred successors to start the election", "fields": [ + associated node" }, + { "name": "CandidateIdHost", "type": "int32string", "versions": "1+", "entityTypeabout": "brokerIdThe node's hostname" }, + + { "name": "CandidateUuidPort", "type": "uuidint32", "versions": "1+" } + ], "about": "The node's port" } ]} ] ]} ] |
...
} |
Fetch
Handling
The leader will track the fetched offset for the replica tuple (ID and UUID). Replicas are uniquely identified by their ID and UUID so their state will be tracked using their ID and UUID.
When removing the leader from the voters set, it will remain the leader for that epoch until the VotersRecord gets committed. This means that the leader needs to allow replicas (voters and observers) to fetch from the leader even if it is not part of the voters set. This also means that if the leader is not part of the voters set it should not include itself when computing the committed offset (also known as the high-watermark) and when checking that the quorum is alive.
Request
Add the replica UUID (directory id) to the Fetch request. This is needed so that the leader can correctly track persisted offsets all of the voters.
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/EndQuorumEpochResponseFetchRequest.json diff --git a/clients/src/main/resources/common/message/EndQuorumEpochResponseFetchRequest.json b/clients/src/main/resources/common/message/EndQuorumEpochResponseFetchRequest.json index cd23247045235357d004..0d5d61b7e7ff86469831 100644 --- a/clients/src/main/resources/common/message/EndQuorumEpochResponseFetchRequest.json +++ b/clients/src/main/resources/common/message/EndQuorumEpochResponse.json @@ -17,8 +17,8 @@/common/message/FetchRequest.json @@ -55,7 +55,9 @@ // deprecate the old ReplicaId field and set its default value to -1. (KIP-903) // // Version 16 is the same as version 15 (KIP-951). - "validVersions": "0-16", + // + // Version 17 adds directory id support from KIP-853 + "validVersions": "0-17", "apiKeydeprecatedVersions": 54"0-3", "typeflexibleVersions": "response12+", "fields": [ @@ -100,7 +102,9 @@ { "name": "EndQuorumEpochResponseLogStartOffset", - "type": "validVersionsint64", "versions": "05+", - "default": "flexibleVersions-1":, "noneignorable": true, + "validVersionsabout": "0-1", + "flexibleVersions": "1+"The earliest available offset of the follower replica. The field is only used when the request is sent by the follower."}, "fields": [ { "name": "ErrorCodePartitionMaxBytes", "type": "int16int32", "versions": "0+", - "about": "The top level error code."}, @@ -36,6 +36,13 @@ maximum bytes to fetch from this partition. See KIP-74 for cases where this limit may not be honored." } + "about": "The maximum bytes to fetch from this partition. See KIP-74 for cases where this limit may not be honored." }, + { "name": "LeaderEpochReplicaUuid", "type": "uuid", "versions": "17+", "taggedVersions": "int3217+", "versionstag": "0+", + "about": "The directory id latestof knownthe leaderfollower epochfetching" } ]} + ]}, + { "name": "NodeEndpointsForgottenTopicsData", "type": "[]NodeEndpointForgottenTopic", "versions": "1+", "taggedVersions": "1+", "tag": 0, + "about": "Endpoints for all leaders enumerated in PartitionData", "fields": [ + { "name": "NodeId", "type": "int32", "versions": "1+", + "mapKey": true, "entityType": "brokerId", "about": "The ID of the associated node" }, + { "name": "Host", "type": "string", "versions": "1+", "about7+", "ignorable": false, |
Response
Code Block |
---|
git diff upstream/trunk clients/src/main/resources/common/message/FetchResponse.json diff --git a/clients/src/main/resources/common/message/FetchResponse.json b/clients/src/main/resources/common/message/FetchResponse.json index e5f49ba6fd..b432a79719 100644 --- a/clients/src/main/resources/common/message/FetchResponse.json +++ b/clients/src/main/resources/common/message/FetchResponse.json @@ -47,7 +47,7 @@ // Version 15 is the same as version 14 (KIP-903). // // Version 16 adds the 'NodeEndpoints' field (KIP-951). - "validVersions": "0-16", + "validVersions": "0-17", "flexibleVersions": "12+", "fields": [ ": "The node's hostname" }, + { "name": "PortThrottleTimeMs", "type": "int32", "versions": "1+", "aboutignorable": "The node's port" } ]} ] } |
Fetch
Handling
The leader will track the fetched offset for the replica tuple (ID and UUID). Replicas are uniquely identified by their ID and UUID so their state will be tracking using their ID and UUID.
true, |
FetchSnapshot
Handling
No changes are needed to the handling of this RPC. The only thing to keep in mind is that replicas will accept FetchSnapshot request as long as they are the latest leader even if they are not in the current voters setWhen removing the leader from the voters set, it will remain the leader for that epoch until the VotersRecord gets committed. This means that the leader needs to allow replicas (voters and observers) to fetch from the leader even if it is not part of the voter set. This also means that if the leader is not part of the voter set it should not include itself when computing the committed offset (also known as the high-watermark) and when checking that the quorum is alive.
Request
Add the replica UUID (directory id) to the Fetch request. This is needed so that the leader can correctly track persisted offsets all of the votersnot needed for correctness because leaders only track fetched offsets using the Fetch RPC but it is good add for consistency and debugability.
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/FetchRequestFetchSnapshotRequest.json diff --git a/clients/src/main/resources/common/message/FetchRequestFetchSnapshotRequest.json b/clients/src/main/resources/common/message/FetchRequestFetchSnapshotRequest.json index 235357d004358ef2e322..ff864698319a577d6289 100644 --- a/clients/src/main/resources/common/message/FetchRequestFetchSnapshotRequest.json +++ b/clients/src/main/resources/common/message/FetchRequestFetchSnapshotRequest.json @@ -5518,711 +55,9 @@ // deprecate the old ReplicaId field and set its default value to -1. (KIP-903) // // Version 16 is the same as version 15 (KIP-951). - "validVersions": "0-16", + // + // Version 17 adds directory id support from KIP-853 +18,11 @@ "type": "request", "listeners": ["controller"], "name": "FetchSnapshotRequest", - "validVersions": "0-17", + "deprecatedVersionsvalidVersions": "0-31", "flexibleVersions": "120+", "fields": [ @@ -100,7 +102,9 @@ { "name": "LogStartOffsetClusterId", "type": "int64string", "versions": "50+", "defaultnullableVersions": "-10+", "ignorabledefault": true"null", "taggedVersions": "0+", "tag": 0, "about": "The earliestcluster availableID offset of the follower replica. The field is only used when the request is sent by the follower."}, if known" }, { "name": "PartitionMaxBytesReplicaId", "type": "int32", "versions": "0+", - "aboutdefault": "The maximum bytes to fetch from this partition. See KIP-74 for cases where this limit may not be honored." } +-1", "entityType": "brokerId", "about": "The maximumbroker bytesID toof fetch from this partition. See KIP-74 for cases where this limit may not be honored.the follower" }, + { "name": "ReplicaUuidMaxBytes", "type": "uuidint32", "versions": "170+", "taggedVersionsdefault": "17+0x7fffffff", "tag": 0, + @@ -44,7 +44,9 @@ { "name": "Epoch", "type": "int32", "aboutversions": "The directory id of the follower fetching0+" } ]}, ]}, { "name": "ForgottenTopicsDataPosition", "type": "[]ForgottenTopicint64", "versions": "7+", "ignorable": false, |
Response
Code Block |
---|
git diff upstream/trunk clients/src/main/resources/common/message/FetchResponse.json
diff --git a/clients/src/main/resources/common/message/FetchResponse.json b/clients/src/main/resources/common/message/FetchResponse.json
index e5f49ba6fd..b432a79719 100644
--- a/clients/src/main/resources/common/message/FetchResponse.json
+++ b/clients/src/main/resources/common/message/FetchResponse.json
@@ -47,7 +47,7 @@
// Version 15 is the same as version 14 (KIP-903).
//
// Version 16 adds the 'NodeEndpoints' field (KIP-951).
- "validVersions": "0-16",
+ "validVersions": "0-17",
"flexibleVersions": "12+",
"fields": [
{ "name": "ThrottleTimeMs", "type": "int32", "versions": "1+", "ignorable": true, |
FetchSnapshot
Handling
No changes are needed to the handling of this RPC. The only thing to keep in mind is that replicas will accept FetchSnapshot request as long as they are the latest leader even if they are not in the current voter set.
Request
"0+",
- "about": "The byte position within the snapshot to start fetching from" }
+ "about": "The byte position within the snapshot to start fetching from" },
+ { "name": "ReplicaUuid", "type": "uuid", "versions": "1+", "taggedVersions": "1+", "tag": 0,
+ "about": "The directory id of the follower fetching" }
]}
]}
] |
Response
Version 1 renames LeaderIdAndEpoch to CurrentLeader and adds Endpoint to CurrentLeaderAdd the replica UUID (directory id) to the Fetch request. This is not needed for correctness because leaders only track fetched offsets using the Fetch RPC but it is good add for consistency and debugability.
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/FetchSnapshotRequestFetchSnapshotResponse.json diff --git a/clients/src/main/resources/common/message/FetchSnapshotRequestFetchSnapshotResponse.json b/clients/src/main/resources/common/message/FetchSnapshotRequestFetchSnapshotResponse.json index 358ef2e322887a5e4401..9a577d62892d9d269930 100644 --- a/clients/src/main/resources/common/message/FetchSnapshotRequestFetchSnapshotResponse.json +++ b/clients/src/main/resources/common/message/FetchSnapshotRequestFetchSnapshotResponse.json @@ -1817,1123 +1817,1123 @@ "typeapiKey": "request"59, "listenerstype": ["controllerresponse"], "name": "FetchSnapshotRequestFetchSnapshotResponse", - "validVersions": "0", + "validVersions": "0-1", "flexibleVersions": "0+", "fields": [ { "name": "ClusterIdThrottleTimeMs", "type": "stringint32", "versions": "0+", "nullableVersionsignorable": "0+", "default": "null", "taggedVersions": "0+", "tag": 0,true, "about": "The clusterduration IDin ifmilliseconds known" }, { "name": "ReplicaId", "type": "int32", "versions": "0+", "default": "-1", "entityType": "brokerId", "about": "The broker ID of the follower" for which the request was throttled due to a quota violation, or zero if the request did not violate any quota" }, { "name": "MaxBytesErrorCode", "type": "int32int16", "versions": "0+", "defaultignorable": "0x7fffffff"false, @@ -44,7 +44,9 @@ "about": "The top level response error code" }, { "name": "EpochTopics", "type": "int32[]TopicSnapshot", "versions": "0+", } "about": "The topics to ]}fetch", "fields": [ { "name": "PositionName", "type": "int64string", "versions": "0+", - "about"entityType": "The byte position within the snapshot to start fetching from" } +topicName", "about": "The bytename positionof within the snapshottopic to start fetching fromfetch" }, + { "name": "ReplicaUuidPartitions", "type": "uuid[]PartitionSnapshot", "versions": "10+", + "about": "The directorypartitions idto of the follower fetching" } ]} ]} ] |
Response
Version 1 renames LeaderIdAndEpoch to CurrentLeader and adds Endpoint to CurrentLeader.
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/FetchSnapshotResponse.json diff --git a/clients/src/main/resources/common/message/FetchSnapshotResponse.json b/clients/src/main/resources/common/message/FetchSnapshotResponse.json index 887a5e4401..2d9d269930 100644 --- a/clients/src/main/resources/common/message/FetchSnapshotResponse.json +++ b/clients/src/main/resources/common/message/FetchSnapshotResponse.json @@ -17,23 +17,23 @@ "apiKey": 59, "type": "response", fetch", "fields": [ { "name": "Index", "type": "int32", "versions": "0+", "about": "The partition index" }, { "name": "FetchSnapshotResponseErrorCode", - "validVersionstype": "0int16", + "validVersionsversions": "0-1+", "flexibleVersionsabout": "0+"The error code, or 0 if there was no fetch error" }, "fields": [ { "name": "ThrottleTimeMsSnapshotId", "type": "int32SnapshotId", "versions": "0+", "ignorable": true, "about": "The durationsnapshot inendOffset millisecondsand for which the request was throttled due to a quota violation, or zero if the request did not violate any quota" }, epoch fetched", "fields": [ @@ -43,17 +43,24 @@ { "name": "ErrorCodeCurrentLeader", "type": "int16LeaderIdAndEpoch", "versions": "0+", "ignorabletaggedVersions": false, "about": "The top level response error code" }, "0+", "tag": 0, "fields": [ { "name": "TopicsLeaderId", "type": "[]TopicSnapshotint32", "versions": "0+", "entityType": "brokerId", "about": "The topics to fetch", "fields": [ ID of the current leader or -1 if the leader is unknown" }, { "name": "NameLeaderEpoch", "type": "stringint32", "versions": "0+", "entityType": "topicName", "about": "The name of the topic to fetch" latest known leader epoch" } ]}, { "name": "PartitionsSize", "type": "[]PartitionSnapshotint64", "versions": "0+", "about": "The total partitionssize toof fetch",the snapshot"fields": [}, { "name": "IndexPosition", "type": "int32int64", "versions": "0+", "about": "The partition index": "The starting byte position within the snapshot included in the Bytes field" }, { "name": "ErrorCodeUnalignedRecords", "type": "int16records", "versions": "0+", "about": "The error code, or 0 if there was no fetch error" },Snapshot data in records format which may not be aligned on an offset boundary" } ]} + ]}, + { "name": "SnapshotIdNodeEndpoints", "type": "SnapshotId[]NodeEndpoint", "versions": "01+", "taggedVersions": "1+", "tag": 0, + "about": "TheEndpoints for snapshotall endOffsetcurrent-leaders andenumerated epochin fetchedPartitionSnapshot", "fields": [ @@ -43,17 +43,24 @@ { "name": "CurrentLeaderNodeId", "type": "LeaderIdAndEpochint32", "versions": "1+", + "versionsmapKey": "0+"true, "taggedVersionsentityType": "0+brokerId", "tagabout": 0, "fields": [ "The ID of the associated node" }, + { "name": "LeaderIdHost", "type": "int32string", "versions": "01+", "entityType": "brokerId", "about": "The ID of the current leader or -1 if the leader is unknownnode's hostname" }, + { "name": "LeaderEpochPort", "type": "int32", "versions": "0+", "about": "The latest known leader epoch" } ]}, { "name": "Size", "type": "int64", "versions": "0+", "about": "The total size of the snapshot" }, { "name": "Position", "type": "int64", "versions": "0+", "about": "The starting byte position within the snapshot included in the Bytes field" }, { "name": "UnalignedRecords", "type": "records", "versions": "0+", "about": "Snapshot data in records format which may not be aligned on an offset boundary" } ]} + ]}, + 1+", "about": "The node's port" } ]} ] } |
DescribeQuorum
Handling
The DescribeQuorum RPC will get forwarded to the KRaft cluster metadata leader. The response for this RPC will be extended to include the UUID of all of the voters and observers, and the endpoint information for all of the voters.
Since the node and listener information is index by the node id, if there are multiple voters with the same replica id only the latest entry will be returned and used.
Request
Code Block |
---|
git diff upstream/trunk clients/src/main/resources/common/message/DescribeQuorumRequest.json diff --git a/clients/src/main/resources/common/message/DescribeQuorumRequest.json b/clients/src/main/resources/common/message/DescribeQuorumRequest.json index cee8fe6982..93ab303aaf 100644 --- a/clients/src/main/resources/common/message/DescribeQuorumRequest.json +++ b/clients/src/main/resources/common/message/DescribeQuorumRequest.json @@ -19,7 +19,7 @@ "listeners": ["broker", "controller"], "name": "DescribeQuorumRequest", // Version 1 adds additional fields in the response. The request is unchanged (KIP-836). - "validVersions": "0-1", + "validVersions": "0-2", "flexibleVersions": "0+", "fields": [ { "name": "NodeEndpointsTopics", "type": "[]NodeEndpointTopicData", |
Response
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/DescribeQuorumResponse.json diff --git a/clients/src/main/resources/common/message/DescribeQuorumResponse.json b/clients/src/main/resources/common/message/DescribeQuorumResponse.json index 0ea6271238..b54cd6bd50 100644 --- a/clients/src/main/resources/common/message/DescribeQuorumResponse.json +++ b/clients/src/main/resources/common/message/DescribeQuorumResponse.json @@ -18,11 +18,13 @@ "type": "response", "name": "DescribeQuorumResponse", // Version 1 adds LastFetchTimeStamp and LastCaughtUpTimestamp in ReplicaState (KIP-836). - "validVersions": "0-1", + "validVersions": "0-2", "flexibleVersions": "0+", "fields": [ "versions": "1+", "taggedVersions": "1+", "tag": 0, + "about": "Endpoints for all current-leaders enumerated in PartitionSnapshot", "fields": [ + { "name": "NodeId", "type": "int32", "versions": "1+", + "mapKey": true, "entityType": "brokerId", "about": "The ID of the associated node" }, + { "name": "Host", "type": "string", "versions": "1+", "about": "The node's hostname" }, + { "name": "PortErrorCode", "type": "int32int16", "versions": "10+", "about": "The node's port" } ]} ] } |
DescribeQuorum
Handling
The DescribeQuorum RPC will get forwarded to the KRaft cluster metadata leader. The response for this RPC will be extended to include the UUID of all of the voters and observers, and the endpoint information for all of the voters.
Since the node and listener information is index by the node id, if there are multiple voters with the same replica id only the latest entry will be returned and used.
Request
Code Block |
---|
git diff upstream/trunk clients/src/main/resources/common/message/DescribeQuorumRequest.json diff --git a/clients/src/main/resources/common/message/DescribeQuorumRequest.json b/clients/src/main/resources/common/message/DescribeQuorumRequest.json index cee8fe6982..93ab303aaf 100644 --- a/clients/src/main/resources/common/message/DescribeQuorumRequest.json +++ b/clients/src/main/resources/common/message/DescribeQuorumRequest.json @@ -19,7 +19,7 @@ "listeners": ["broker", "controller"], "name": "DescribeQuorumRequest", // Version 1 adds additional fields in the response. The request is unchanged (KIP-836). - "validVersions": "0-1", + "validVersions": "0-2", "flexibleVersions": "0+", "fields": [ "about": "The top level error code."}, + { "name": "ErrorMessage", "type": "string", "versions": "2+", "nullableVersions": "2+", "ignorable": true, + "about": "The error message, or null if there was no error." }, { "name": "Topics", "type": "[]TopicData", "versions": "0+", "fields": [ { "name": "TopicName", "type": "string", "versions": "0+", "entityType": "topicName", @@ -32,6 +34,8 @@ { "name": "PartitionIndex", "type": "int32", "versions": "0+", "about": "The partition index." }, { "name": "TopicsErrorCode", "type": "[]TopicDataint16", |
Response
Code Block | ||
---|---|---|
| ||
git diff upstream/trunk clients/src/main/resources/common/message/DescribeQuorumResponse.json diff --git a/clients/src/main/resources/common/message/DescribeQuorumResponse.json b/clients/src/main/resources/common/message/DescribeQuorumResponse.json index 0ea6271238..e2481dff04 100644 --- a/clients/src/main/resources/common/message/DescribeQuorumResponse.json +++ b/clients/src/main/resources/common/message/DescribeQuorumResponse.json @@ -18,7 +18,7 @@ "versions": "0+"}, + { "name": "ErrorMessage", "type": "responsestring", "versions": "2+", "namenullableVersions": "DescribeQuorumResponse"2+", "ignorable": true, + // Version 1 adds LastFetchTimeStamp and LastCaughtUpTimestamp in ReplicaState (KIP-836). - "validVersions": "0-1", + "validVersions "about": "The error message, or null if there was no error." }, { "name": "LeaderId", "type": "int32", "versions": "0-2+", "flexibleVersionsentityType": "0+brokerId", "fieldsabout": [ "The ID of the current leader or -1 if the leader is unknown."}, { "name": "ErrorCodeLeaderEpoch", "type": "int16int32", "versions": "0+", @@ -40,10 +4044,25 @@ { "name": "CurrentVoters", "type": "[]ReplicaState", "versions": "0+" }, { "name": "Observers", "type": "[]ReplicaState", "versions": "0+" } ]} - ]}], + ]}, + { "name": "Nodes", "type": "[]Node", "versions": "2+", "fields": [ + { "name": "NodeId", "type": "int32", "versions": "2+", + "mapKey": true, "entityType": "brokerId", "about": "The ID of the associated node" }, + { "name": "Listeners", "type": "[]Listener", + "about": "The listeners of this controller", "versions": "0+", "fields": [ + { "name": "Name", "type": "string", "versions": "0+", "mapKey": true, + "about": "The name of the endpoint" }, + { "name": "Host", "type": "string", "versions": "0+", + "about": "The hostname" }, + { "name": "Port", "type": "uint16", "versions": "0+", + "about": "The port" } + ]} + ]} + ], "commonStructs": [ { "name": "ReplicaState", "versions": "0+", "fields": [ { "name": "ReplicaId", "type": "int32", "versions": "0+", "entityType": "brokerId" }, + { "name": "ReplicaUuid", "type": "uuid", "versions": "2+" }, { "name": "LogEndOffset", "type": "int64", "versions": "0+", "about": "The last known log end offset of the follower or -1 if it is unknown"}, { "name": "LastFetchTimestamp", "type": "int64", "versions": "1+", "ignorable": true, "default": -1, |
Admin
...
client
The Java Admin client will be extended to support the new field in the DescribeQuorum response and the new AddVoter and RemoveVoter RPCs. When the broker bootstrap servers is used the admin client will send the request to the least loaded broker. When the controller bootstrap servers is used the admin client will send the request to the KRaft leader (active controller).
Monitoring
NAME | TAGS | TYPE | NOTE |
---|---|---|---|
number-of-voters | type=raft-metrics | gauge | number of voters for the cluster metadata topic partition. |
number-of-observers | type=raft-metrics | guage | number of observer that could be promoted to voters. |
pending-add-voter | type=raft-metrics | guage | 1 if there is a pending add voter operation, 0 otherwise. |
pending-remove-voter | type=raft-metrics | guage | 1 if there is a pending remove voter operation, 0 otherwise. |
TBD | TBD | guage | 1 if a controller node is not a voter for the KRaft cluster metadata partition, 0 otherwise. |
duplicate-voter-ids | type=raft-metrics | gauge | Counts the number of duplicate replica id in the set of voters. |
number-of-offline-voters | type=raft-metrics | gauge | Number of voters with a last Fetch timestamp greater than the Fetch timeout. |
ignored-static-voters | TBD | gauge | 1 if controller.quorum.voter is set and the kraft.version is greater than 0, 0 otherwise. |
Command
...
line interface
kafka-metadata-shell
A future KIP will describe how the kafka-metadata-shell tool will be extended to be able to read and display KRaft control records from the quorum, snapshot and log.
...
This tool as described in KIP-595 and KIP-836 will be improved to support these additional commands and options:
describe --status
This command be extended to print the new information added to the DescribeQuorum RPC. The includes the directory id for all of the replicas (voters and observers). The known endpoints for all of the voters. Any uncommitted voter changes.
...
This command is use to add controllers to the KRaft cluster metadata partition. This command must be executed using the server configuration of the new controller. The command will read the server properties file to read the replica id, the endpoints, and the meta.properties for the directory id. The tool will set the TimoutMs
for the AddVoterRequest
to 30 seconds.
remove-controller --controller-id <controller-id> --controller-uuid <controller-uuid>
This command is used to remove voters from the KRaft cluster metadata partition. The flags --controller-id and --controller-uuid must be specified.
Compatibility,
...
deprecation, and
...
migration plan
RPC versions will be negotiated using the ApiVersions
RPC. KRaft will use the kraft.version
to determine which version of KRaftVersionRecord
and VotersRecord
to write to the log and which version of QuorumStateData
to write to the quorum-state
file.
Downgrading the kraft.version
from 1 to 0 is not possible. This is mainly due to the fact that kraft.version
1 writes data to the log and snapshot that gets replicated to all of the replicasThe features in this KIP will be supported if the ApiVersions of all of the voters and observers is greater than the versions described here. If the leader has a replica UUID for all of the voters then this KIP is supported by all of the voters.
When to remove controller.quorum.voters
It is safe for the operator to remove the configuration for controller.qourumquorum.voters
when the kraft.version
has been upgrade to version 1. All of the Kafka nodes will expose the ignored-static-voter
metrics. If all of the Kafka nodes expose a 1 for this metrics, it is safe to remove controller.quorum.voters
from the node configuration and specify the controller.quorum.bootstrap.servers instead.
Test
...
plan
This KIP will be tested using unittest, integration tests, system test, simulation tests and TLA+ specification.
Rejected
...
alternatives
KIP-642: Dynamic quorum reassignment
KIP-642: Dynamic quorum reassignment. KIP-642 describe how to perform dynamic reassignment will multiple voters added and removed. KIP-642 doesn't support disk failures and it would be more difficult to implement compared to this KIP-853. In a future KIP we can describe how we can add administrative operations that support the addition and removal of multiple voters.
Leverage metadata controller mechanisms
There are some common problems between the Kafka Controller and KRaft like bootstrapping cluster information, versioning and propagating node information. The Kafka controller has solved both of these problems in the metadata layer. The metadata layer is an application built using the KRaft layer. KRaft and KIP-853 should not dependent on the metadata layer for its solution to these problems. One advantage is to allow other use cases for KRaft that are not the cluster metadata partition but the main advantage is achieving a clear separation for both reliability, testability and debuggability.
In the future we can extend Kafka so that the Kafka controller relies on KRaft's solution to these problems.
References
- Ongaro, Diego, and John Ousterhout. "In search of an understandable consensus algorithm." 2014 {USENIX} Annual Technical Conference ({USENIX}{ATC} 14). 2014.
Ongaro, Diego. Consensus: Bridging theory and practice. Diss. Stanford University, 2014.
- Bug in single-server membership changes
- KIP-595: A Raft Protocol for the Metadata Quorum
- KIP-630: Kafka Raft Snapshot
- KIP-631: The Quorum-based Kafka Controller