You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@kafka.apache.org by da...@apache.org on 2022/12/03 01:00:01 UTC

[kafka] branch trunk updated: KAFKA-14304 Add RPC changes, records, and config from KIP-866 (#12928)

This is an automated email from the ASF dual-hosted git repository.

davidarthur pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/kafka.git


The following commit(s) were added to refs/heads/trunk by this push:
     new 7b7e40a536a KAFKA-14304 Add RPC changes, records, and config from KIP-866 (#12928)
7b7e40a536a is described below

commit 7b7e40a536a79cebf35cc278b9375c8352d342b9
Author: David Arthur <mu...@gmail.com>
AuthorDate: Fri Dec 2 19:59:52 2022 -0500

    KAFKA-14304 Add RPC changes, records, and config from KIP-866 (#12928)
    
    Reviewers: Colin Patrick McCabe <cm...@apache.org>
---
 .../common/message/BrokerRegistrationRequest.json  |   4 +-
 .../common/message/LeaderAndIsrRequest.json        |   6 +-
 .../common/message/LeaderAndIsrResponse.json       |   2 +-
 .../common/message/StopReplicaRequest.json         |   6 +-
 .../common/message/StopReplicaResponse.json        |   2 +-
 .../common/message/UpdateMetadataRequest.json      |   7 +-
 .../common/message/UpdateMetadataResponse.json     |   2 +-
 .../controller/ControllerChannelManager.scala      |   9 +-
 core/src/main/scala/kafka/server/KafkaConfig.scala |   6 +
 core/src/main/scala/kafka/zk/ZkData.scala          |  41 ++++++-
 .../java/kafka/test/ClusterTestExtensionsTest.java |   2 +-
 .../java/kafka/test/annotation/ClusterTest.java    |   2 +-
 .../unit/kafka/admin/FeatureCommandTest.scala      |   4 +-
 .../controller/ControllerChannelManagerTest.scala  |  16 ++-
 .../migration/ZkMigrationLeadershipState.java      | 125 +++++++++++++++++++++
 .../kafka/metadata/migration/ZkMigrationState.java |  74 ++++++++++++
 .../common/metadata/RegisterBrokerRecord.json      |   4 +-
 .../common/metadata/ZkMigrationRecord.json         |  19 ++--
 .../kafka/controller/QuorumControllerTest.java     |  14 +--
 .../kafka/server/common/MetadataVersion.java       |   7 +-
 20 files changed, 310 insertions(+), 42 deletions(-)

diff --git a/clients/src/main/resources/common/message/BrokerRegistrationRequest.json b/clients/src/main/resources/common/message/BrokerRegistrationRequest.json
index d96369caf24..c29e190dd01 100644
--- a/clients/src/main/resources/common/message/BrokerRegistrationRequest.json
+++ b/clients/src/main/resources/common/message/BrokerRegistrationRequest.json
@@ -50,6 +50,8 @@
     ]
     },
     { "name": "Rack", "type": "string", "versions": "0+", "nullableVersions": "0+",
-      "about": "The rack which this broker is in." }
+      "about": "The rack which this broker is in." },
+    { "name": "IsMigratingZkBroker", "type": "int8", "versions": "0+", "taggedVersions": "0+", "tag": 0, "ignorable": true,
+      "about": "Set by a ZK broker if the required configurations for ZK migration are present." }
   ]
 }
diff --git a/clients/src/main/resources/common/message/LeaderAndIsrRequest.json b/clients/src/main/resources/common/message/LeaderAndIsrRequest.json
index 97881be27ff..7042ec84d5b 100644
--- a/clients/src/main/resources/common/message/LeaderAndIsrRequest.json
+++ b/clients/src/main/resources/common/message/LeaderAndIsrRequest.json
@@ -29,11 +29,15 @@
   // Version 5 adds Topic ID and Type to the TopicStates, as described in KIP-516.
   //
   // Version 6 adds LeaderRecoveryState as described in KIP-704.
-  "validVersions": "0-6",
+  //
+  // Version 7 adds KRaft Controller ID field as part of KIP-866
+  "validVersions": "0-7",
   "flexibleVersions": "4+",
   "fields": [
     { "name": "ControllerId", "type": "int32", "versions": "0+", "entityType": "brokerId",
       "about": "The current controller ID." },
+    { "name": "KRaftControllerId", "type": "int32", "versions": "7+", "entityType": "brokerId", "default": "-1",
+      "about": "The KRaft controller id, used during migration. See KIP-866" },
     { "name": "ControllerEpoch", "type": "int32", "versions": "0+",
       "about": "The current controller epoch." },
     { "name": "BrokerEpoch", "type": "int64", "versions": "2+", "ignorable": true, "default": "-1",
diff --git a/clients/src/main/resources/common/message/LeaderAndIsrResponse.json b/clients/src/main/resources/common/message/LeaderAndIsrResponse.json
index 79ad819f5e6..f4f2a4308fd 100644
--- a/clients/src/main/resources/common/message/LeaderAndIsrResponse.json
+++ b/clients/src/main/resources/common/message/LeaderAndIsrResponse.json
@@ -27,7 +27,7 @@
   //
   // Version 5 removes TopicName and replaces it with TopicId and reorganizes
   // the partitions by topic, as described by KIP-516.
-  "validVersions": "0-6",
+  "validVersions": "0-7",
   "flexibleVersions": "4+",
   "fields": [
     { "name": "ErrorCode", "type": "int16", "versions": "0+",
diff --git a/clients/src/main/resources/common/message/StopReplicaRequest.json b/clients/src/main/resources/common/message/StopReplicaRequest.json
index b10154fe1fb..67ed752a555 100644
--- a/clients/src/main/resources/common/message/StopReplicaRequest.json
+++ b/clients/src/main/resources/common/message/StopReplicaRequest.json
@@ -24,11 +24,15 @@
   // Version 2 is the first flexible version.
   //
   // Version 3 adds the leader epoch per partition (KIP-570).
-  "validVersions": "0-3",
+  //
+  // Version 4 adds KRaft Controller ID field as part of KIP-866
+  "validVersions": "0-4",
   "flexibleVersions": "2+",
   "fields": [
     { "name": "ControllerId", "type": "int32", "versions": "0+", "entityType": "brokerId",
       "about": "The controller id." },
+    { "name": "KRaftControllerId", "type": "int32", "versions": "4+", "entityType": "brokerId", "default": "-1",
+      "about": "The KRaft controller id, used during migration. See KIP-866" },
     { "name": "ControllerEpoch", "type": "int32", "versions": "0+",
       "about": "The controller epoch." },
     { "name": "BrokerEpoch", "type": "int64", "versions": "1+", "default": "-1", "ignorable": true,
diff --git a/clients/src/main/resources/common/message/StopReplicaResponse.json b/clients/src/main/resources/common/message/StopReplicaResponse.json
index 64b355e0406..35551c47e1c 100644
--- a/clients/src/main/resources/common/message/StopReplicaResponse.json
+++ b/clients/src/main/resources/common/message/StopReplicaResponse.json
@@ -22,7 +22,7 @@
   // Version 2 is the first flexible version.
   //
   // Version 3 returns FENCED_LEADER_EPOCH if the epoch of the leader is stale (KIP-570).
-  "validVersions": "0-3",
+  "validVersions": "0-4",
   "flexibleVersions": "2+",
   "fields": [
     { "name": "ErrorCode", "type": "int16", "versions": "0+",
diff --git a/clients/src/main/resources/common/message/UpdateMetadataRequest.json b/clients/src/main/resources/common/message/UpdateMetadataRequest.json
index 5f397a92c04..287b0ed1a4b 100644
--- a/clients/src/main/resources/common/message/UpdateMetadataRequest.json
+++ b/clients/src/main/resources/common/message/UpdateMetadataRequest.json
@@ -27,12 +27,17 @@
   // Version 4 adds the offline replica list.
   //
   // Version 5 adds the broker epoch field and normalizes partitions by topic.
+  //
   // Version 7 adds topicId
-  "validVersions": "0-7",
+  //
+  // Version 8 adds KRaft Controller ID field as part of KIP-866
+  "validVersions": "0-8",
   "flexibleVersions": "6+",
   "fields": [
     { "name": "ControllerId", "type": "int32", "versions": "0+", "entityType": "brokerId",
       "about": "The controller id." },
+    { "name": "KRaftControllerId", "type": "int32", "versions": "8+", "entityType": "brokerId",
+      "about": "The KRaft controller id, used during migration." },
     { "name": "ControllerEpoch", "type": "int32", "versions": "0+",
       "about": "The controller epoch." },
     { "name": "BrokerEpoch", "type": "int64", "versions": "5+", "ignorable": true, "default": "-1",
diff --git a/clients/src/main/resources/common/message/UpdateMetadataResponse.json b/clients/src/main/resources/common/message/UpdateMetadataResponse.json
index 62203229c97..4e091e50101 100644
--- a/clients/src/main/resources/common/message/UpdateMetadataResponse.json
+++ b/clients/src/main/resources/common/message/UpdateMetadataResponse.json
@@ -18,7 +18,7 @@
   "type": "response",
   "name": "UpdateMetadataResponse",
   // Versions 1, 2, 3, 4, and 5 are the same as version 0
-  "validVersions": "0-7",
+  "validVersions": "0-8",
   "flexibleVersions": "6+",
   "fields": [
       { "name": "ErrorCode", "type": "int16", "versions": "0+",
diff --git a/core/src/main/scala/kafka/controller/ControllerChannelManager.scala b/core/src/main/scala/kafka/controller/ControllerChannelManager.scala
index f6b9cbd07dd..145eca453a1 100755
--- a/core/src/main/scala/kafka/controller/ControllerChannelManager.scala
+++ b/core/src/main/scala/kafka/controller/ControllerChannelManager.scala
@@ -461,7 +461,8 @@ abstract class AbstractControllerBrokerRequestBatch(config: KafkaConfig,
 
   private def sendLeaderAndIsrRequest(controllerEpoch: Int, stateChangeLog: StateChangeLogger): Unit = {
     val leaderAndIsrRequestVersion: Short =
-      if (config.interBrokerProtocolVersion.isAtLeast(IBP_3_2_IV0)) 6
+      if (config.interBrokerProtocolVersion.isAtLeast(IBP_3_4_IV0)) 7
+      else if (config.interBrokerProtocolVersion.isAtLeast(IBP_3_2_IV0)) 6
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_8_IV1)) 5
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_4_IV1)) 4
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_4_IV0)) 3
@@ -512,7 +513,8 @@ abstract class AbstractControllerBrokerRequestBatch(config: KafkaConfig,
 
     val partitionStates = updateMetadataRequestPartitionInfoMap.values.toBuffer
     val updateMetadataRequestVersion: Short =
-      if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_8_IV1)) 7
+      if (config.interBrokerProtocolVersion.isAtLeast(IBP_3_4_IV0)) 8
+      else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_8_IV1)) 7
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_4_IV1)) 6
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_2_IV0)) 5
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_1_0_IV0)) 4
@@ -568,7 +570,8 @@ abstract class AbstractControllerBrokerRequestBatch(config: KafkaConfig,
   private def sendStopReplicaRequests(controllerEpoch: Int, stateChangeLog: StateChangeLogger): Unit = {
     val traceEnabled = stateChangeLog.isTraceEnabled
     val stopReplicaRequestVersion: Short =
-      if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_6_IV0)) 3
+      if (config.interBrokerProtocolVersion.isAtLeast(IBP_3_4_IV0)) 4
+      else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_6_IV0)) 3
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_4_IV1)) 2
       else if (config.interBrokerProtocolVersion.isAtLeast(IBP_2_2_IV0)) 1
       else 0
diff --git a/core/src/main/scala/kafka/server/KafkaConfig.scala b/core/src/main/scala/kafka/server/KafkaConfig.scala
index e2c8db8dbb7..2ad9c462c58 100755
--- a/core/src/main/scala/kafka/server/KafkaConfig.scala
+++ b/core/src/main/scala/kafka/server/KafkaConfig.scala
@@ -413,6 +413,9 @@ object KafkaConfig {
   val QuorumVotersProp = RaftConfig.QUORUM_VOTERS_CONFIG
   val MetadataMaxIdleIntervalMsProp = "metadata.max.idle.interval.ms"
 
+  /** ZK to KRaft Migration configs */
+  val MigrationEnabledProp = "zookeeper.metadata.migration.enable"
+
   /************* Authorizer Configuration ***********/
   val AuthorizerClassNameProp = "authorizer.class.name"
   val EarlyStartListenersProp = "early.start.listeners"
@@ -1189,6 +1192,7 @@ object KafkaConfig {
       .define(MetadataMaxRetentionBytesProp, LONG, Defaults.MetadataMaxRetentionBytes, null, HIGH, MetadataMaxRetentionBytesDoc)
       .define(MetadataMaxRetentionMillisProp, LONG, Defaults.LogRetentionHours * 60 * 60 * 1000L, null, HIGH, MetadataMaxRetentionMillisDoc)
       .define(MetadataMaxIdleIntervalMsProp, INT, Defaults.MetadataMaxIdleIntervalMs, atLeast(0), LOW, MetadataMaxIdleIntervalMsDoc)
+      .define(MigrationEnabledProp, BOOLEAN, false, HIGH, "Enable ZK to KRaft migration")
 
       /************* Authorizer Configuration ***********/
       .define(AuthorizerClassNameProp, STRING, Defaults.AuthorizerClassName, new ConfigDef.NonNullValidator(), LOW, AuthorizerClassNameDoc)
@@ -1661,6 +1665,8 @@ class KafkaConfig private(doLog: Boolean, val props: java.util.Map[_, _], dynami
   def requiresZookeeper: Boolean = processRoles.isEmpty
   def usesSelfManagedQuorum: Boolean = processRoles.nonEmpty
 
+  val migrationEnabled: Boolean = getBoolean(KafkaConfig.MigrationEnabledProp)
+
   private def parseProcessRoles(): Set[ProcessRole] = {
     val roles = getList(KafkaConfig.ProcessRolesProp).asScala.map {
       case "broker" => BrokerRole
diff --git a/core/src/main/scala/kafka/zk/ZkData.scala b/core/src/main/scala/kafka/zk/ZkData.scala
index 7006a21f94b..6bd3b19d7dc 100644
--- a/core/src/main/scala/kafka/zk/ZkData.scala
+++ b/core/src/main/scala/kafka/zk/ZkData.scala
@@ -41,6 +41,7 @@ import org.apache.kafka.common.security.token.delegation.{DelegationToken, Token
 import org.apache.kafka.common.utils.{SecurityUtils, Time}
 import org.apache.kafka.common.{KafkaException, TopicPartition, Uuid}
 import org.apache.kafka.metadata.LeaderRecoveryState
+import org.apache.kafka.metadata.migration.ZkMigrationLeadershipState
 import org.apache.kafka.server.common.{MetadataVersion, ProducerIdsBlock}
 import org.apache.kafka.server.common.MetadataVersion.{IBP_0_10_0_IV1, IBP_2_7_IV0}
 import org.apache.zookeeper.ZooDefs
@@ -56,8 +57,11 @@ import scala.util.{Failure, Success, Try}
 
 object ControllerZNode {
   def path = "/controller"
-  def encode(brokerId: Int, timestamp: Long): Array[Byte] = {
-    Json.encodeAsBytes(Map("version" -> 1, "brokerid" -> brokerId, "timestamp" -> timestamp.toString).asJava)
+  def encode(brokerId: Int, timestamp: Long, kraftControllerEpoch: Int = -1): Array[Byte] = {
+    Json.encodeAsBytes(Map(
+      "version" -> 2, "brokerid" -> brokerId,
+      "timestamp" -> timestamp.toString,
+      "kraftControllerEpoch" -> kraftControllerEpoch).asJava)
   }
   def decode(bytes: Array[Byte]): Option[Int] = Json.parseBytes(bytes).map { js =>
     js.asJsonObject("brokerid").to[Int]
@@ -1019,6 +1023,36 @@ object FeatureZNode {
   }
 }
 
+object MigrationZNode {
+  val path = "/migration"
+
+  def encode(migration: ZkMigrationLeadershipState): Array[Byte] = {
+    val jsonMap = Map(
+      "version" -> 0,
+      "kraft_controller_id" -> migration.kraftControllerId(),
+      "kraft_controller_epoch" -> migration.kraftControllerEpoch(),
+      "kraft_metadata_offset" -> migration.kraftMetadataOffset(),
+      "kraft_metadata_epoch" -> migration.kraftMetadataEpoch()
+    )
+    Json.encodeAsBytes(jsonMap.asJava)
+  }
+
+  def decode(bytes: Array[Byte], zkVersion: Int, modifyTimeMs: Long): ZkMigrationLeadershipState = {
+    val jsonDataAsString = bytes.map(_.toChar).mkString
+    Json.parseBytes(bytes).map(_.asJsonObject).flatMap { js =>
+      val version = js("version").to[Int]
+      if (version != 0) {
+        throw new KafkaException(s"Encountered unknown version $version when parsing migration json $jsonDataAsString")
+      }
+      val controllerId = js("kraft_controller_id").to[Int]
+      val controllerEpoch = js("kraft_controller_epoch").to[Int]
+      val metadataOffset = js("kraft_metadata_offset").to[Long]
+      val metadataEpoch = js("kraft_metadata_epoch").to[Int]
+      Some(new ZkMigrationLeadershipState(controllerId, controllerEpoch, metadataOffset, metadataEpoch, modifyTimeMs, zkVersion, -2))
+    }.getOrElse(throw new KafkaException(s"Failed to parse the migration json $jsonDataAsString"))
+  }
+}
+
 object ZkData {
 
   // Important: it is necessary to add any new top level Zookeeper path to the Seq
@@ -1032,7 +1066,8 @@ object ZkData {
     ProducerIdBlockZNode.path,
     LogDirEventNotificationZNode.path,
     DelegationTokenAuthZNode.path,
-    ExtendedAclZNode.path) ++ ZkAclStore.securePaths
+    ExtendedAclZNode.path,
+    MigrationZNode.path) ++ ZkAclStore.securePaths
 
   // These are persistent ZK paths that should exist on kafka broker startup.
   val PersistentZkPaths = Seq(
diff --git a/core/src/test/java/kafka/test/ClusterTestExtensionsTest.java b/core/src/test/java/kafka/test/ClusterTestExtensionsTest.java
index 63ca1372531..36cbbc82c9a 100644
--- a/core/src/test/java/kafka/test/ClusterTestExtensionsTest.java
+++ b/core/src/test/java/kafka/test/ClusterTestExtensionsTest.java
@@ -117,6 +117,6 @@ public class ClusterTestExtensionsTest {
 
     @ClusterTest
     public void testDefaults(ClusterConfig config) {
-        Assertions.assertEquals(MetadataVersion.IBP_3_3_IV3, config.metadataVersion());
+        Assertions.assertEquals(MetadataVersion.IBP_3_4_IV0, config.metadataVersion());
     }
 }
diff --git a/core/src/test/java/kafka/test/annotation/ClusterTest.java b/core/src/test/java/kafka/test/annotation/ClusterTest.java
index d1d3222a25b..d841996ec18 100644
--- a/core/src/test/java/kafka/test/annotation/ClusterTest.java
+++ b/core/src/test/java/kafka/test/annotation/ClusterTest.java
@@ -41,6 +41,6 @@ public @interface ClusterTest {
     String name() default "";
     SecurityProtocol securityProtocol() default SecurityProtocol.PLAINTEXT;
     String listener() default "";
-    MetadataVersion metadataVersion() default MetadataVersion.IBP_3_3_IV3;
+    MetadataVersion metadataVersion() default MetadataVersion.IBP_3_4_IV0;
     ClusterConfigProperty[] serverProperties() default {};
 }
diff --git a/core/src/test/scala/unit/kafka/admin/FeatureCommandTest.scala b/core/src/test/scala/unit/kafka/admin/FeatureCommandTest.scala
index 3becc7b330e..a8f74af23bf 100644
--- a/core/src/test/scala/unit/kafka/admin/FeatureCommandTest.scala
+++ b/core/src/test/scala/unit/kafka/admin/FeatureCommandTest.scala
@@ -84,7 +84,7 @@ class FeatureCommandTest extends IntegrationTestHarness {
         Array("--bootstrap-server", bootstrapServers(), "describe"), env.out))
       assertEquals(String.format(
         "Feature: metadata.version\tSupportedMinVersion: 3.0-IV1\t" +
-          "SupportedMaxVersion: 3.3-IV3\tFinalizedVersionLevel: 3.3-IV1\t"),
+          "SupportedMaxVersion: 3.4-IV0\tFinalizedVersionLevel: 3.3-IV1\t"),
             env.outputWithoutEpoch())
     }
   }
@@ -145,7 +145,7 @@ class FeatureCommandTest extends IntegrationTestHarness {
       assertEquals(1, FeatureCommand.mainNoExit(Array("--bootstrap-server", bootstrapServers(),
         "disable", "--feature", "metadata.version"), env.out))
       assertEquals("Could not disable metadata.version. Invalid update version 0 for feature " +
-        "metadata.version. Local controller 1000 only supports versions 1-7", env.outputWithoutEpoch())
+        "metadata.version. Local controller 1000 only supports versions 1-8", env.outputWithoutEpoch())
     }
     TestUtils.resource(FeatureCommandTestEnv()) { env =>
       assertEquals(1, FeatureCommand.mainNoExit(Array("--bootstrap-server", bootstrapServers(),
diff --git a/core/src/test/scala/unit/kafka/controller/ControllerChannelManagerTest.scala b/core/src/test/scala/unit/kafka/controller/ControllerChannelManagerTest.scala
index a77b42e46e6..8e9176af2e9 100644
--- a/core/src/test/scala/unit/kafka/controller/ControllerChannelManagerTest.scala
+++ b/core/src/test/scala/unit/kafka/controller/ControllerChannelManagerTest.scala
@@ -17,7 +17,6 @@
 package kafka.controller
 
 import java.util.Properties
-
 import kafka.api.LeaderAndIsr
 import kafka.cluster.{Broker, EndPoint}
 import kafka.server.KafkaConfig
@@ -34,7 +33,7 @@ import org.apache.kafka.common.security.auth.SecurityProtocol
 import org.apache.kafka.common.{TopicPartition, Uuid}
 import org.apache.kafka.metadata.LeaderRecoveryState
 import org.apache.kafka.server.common.MetadataVersion
-import org.apache.kafka.server.common.MetadataVersion.{IBP_0_10_0_IV1, IBP_0_10_2_IV0, IBP_0_9_0, IBP_1_0_IV0, IBP_2_2_IV0, IBP_2_4_IV0, IBP_2_4_IV1, IBP_2_6_IV0, IBP_2_8_IV1, IBP_3_2_IV0}
+import org.apache.kafka.server.common.MetadataVersion.{IBP_0_10_0_IV1, IBP_0_10_2_IV0, IBP_0_9_0, IBP_1_0_IV0, IBP_2_2_IV0, IBP_2_4_IV0, IBP_2_4_IV1, IBP_2_6_IV0, IBP_2_8_IV1, IBP_3_2_IV0, IBP_3_4_IV0}
 import org.junit.jupiter.api.Assertions._
 import org.junit.jupiter.api.Test
 
@@ -164,14 +163,16 @@ class ControllerChannelManagerTest {
     testLeaderAndIsrRequestFollowsInterBrokerProtocolVersion(MetadataVersion.latest, ApiKeys.LEADER_AND_ISR.latestVersion)
 
     for (metadataVersion <- MetadataVersion.VERSIONS) {
-      val leaderAndIsrRequestVersion: Short =
-        if (metadataVersion.isAtLeast(IBP_3_2_IV0)) 6
+      val leaderAndIsrRequestVersion: Short = {
+        if (metadataVersion.isAtLeast(IBP_3_4_IV0)) 7
+        else if (metadataVersion.isAtLeast(IBP_3_2_IV0)) 6
         else if (metadataVersion.isAtLeast(IBP_2_8_IV1)) 5
         else if (metadataVersion.isAtLeast(IBP_2_4_IV1)) 4
         else if (metadataVersion.isAtLeast(IBP_2_4_IV0)) 3
         else if (metadataVersion.isAtLeast(IBP_2_2_IV0)) 2
         else if (metadataVersion.isAtLeast(IBP_1_0_IV0)) 1
         else 0
+      }
 
       testLeaderAndIsrRequestFollowsInterBrokerProtocolVersion(metadataVersion, leaderAndIsrRequestVersion)
     }
@@ -382,7 +383,8 @@ class ControllerChannelManagerTest {
 
     for (metadataVersion <- MetadataVersion.VERSIONS) {
       val updateMetadataRequestVersion: Short =
-        if (metadataVersion.isAtLeast(IBP_2_8_IV1)) 7
+        if (metadataVersion.isAtLeast(IBP_3_4_IV0)) 8
+        else if (metadataVersion.isAtLeast(IBP_2_8_IV1)) 7
         else if (metadataVersion.isAtLeast(IBP_2_4_IV1)) 6
         else if (metadataVersion.isAtLeast(IBP_2_2_IV0)) 5
         else if (metadataVersion.isAtLeast(IBP_1_0_IV0)) 4
@@ -782,8 +784,10 @@ class ControllerChannelManagerTest {
         testStopReplicaFollowsInterBrokerProtocolVersion(metadataVersion, 1.toShort)
       else if (metadataVersion.isLessThan(IBP_2_6_IV0))
         testStopReplicaFollowsInterBrokerProtocolVersion(metadataVersion, 2.toShort)
-      else
+      else if (metadataVersion.isLessThan(IBP_3_4_IV0))
         testStopReplicaFollowsInterBrokerProtocolVersion(metadataVersion, 3.toShort)
+      else
+        testStopReplicaFollowsInterBrokerProtocolVersion(metadataVersion, 4.toShort)
     }
   }
 
diff --git a/metadata/src/main/java/org/apache/kafka/metadata/migration/ZkMigrationLeadershipState.java b/metadata/src/main/java/org/apache/kafka/metadata/migration/ZkMigrationLeadershipState.java
new file mode 100644
index 00000000000..c54cf7d0e4f
--- /dev/null
+++ b/metadata/src/main/java/org/apache/kafka/metadata/migration/ZkMigrationLeadershipState.java
@@ -0,0 +1,125 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.metadata.migration;
+
+import java.util.Objects;
+
+/**
+ * Persistent state needed to recover an ongoing migration. This data is stored in ZooKeeper under the "/migration"
+ * ZNode and is recovered by the active KRaft controller following an election. The absence of this data in ZK indicates
+ * that no migration has been started.
+ */
+public class ZkMigrationLeadershipState {
+
+    public static final ZkMigrationLeadershipState EMPTY = new ZkMigrationLeadershipState(-1, -1, -1, -1, -1, -1, -1);
+
+    private final int kraftControllerId;
+
+    private final int kraftControllerEpoch;
+
+    private final long kraftMetadataOffset;
+
+    private final int kraftMetadataEpoch;
+
+    private final long lastUpdatedTimeMs;
+
+    private final int migrationZkVersion;
+
+    private final int controllerZkVersion;
+
+    public ZkMigrationLeadershipState(int kraftControllerId, int kraftControllerEpoch,
+                                      long kraftMetadataOffset, int kraftMetadataEpoch,
+                                      long lastUpdatedTimeMs, int migrationZkVersion, int controllerZkVersion) {
+        this.kraftControllerId = kraftControllerId;
+        this.kraftControllerEpoch = kraftControllerEpoch;
+        this.kraftMetadataOffset = kraftMetadataOffset;
+        this.kraftMetadataEpoch = kraftMetadataEpoch;
+        this.lastUpdatedTimeMs = lastUpdatedTimeMs;
+        this.migrationZkVersion = migrationZkVersion;
+        this.controllerZkVersion = controllerZkVersion;
+    }
+
+    public ZkMigrationLeadershipState withZkVersion(int zkVersion) {
+        return new ZkMigrationLeadershipState(
+                this.kraftControllerId, this.kraftControllerEpoch, this.kraftMetadataOffset,
+                this.kraftMetadataEpoch, this.lastUpdatedTimeMs, zkVersion, this.controllerZkVersion);
+    }
+
+    public ZkMigrationLeadershipState withNewKRaftController(int controllerId, int controllerEpoch) {
+        return new ZkMigrationLeadershipState(
+                controllerId, controllerEpoch, this.kraftMetadataOffset,
+                this.kraftMetadataEpoch, this.lastUpdatedTimeMs, this.migrationZkVersion, this.controllerZkVersion);
+    }
+
+    public int kraftControllerId() {
+        return kraftControllerId;
+    }
+
+    public int kraftControllerEpoch() {
+        return kraftControllerEpoch;
+    }
+
+    public long kraftMetadataOffset() {
+        return kraftMetadataOffset;
+    }
+
+    public long kraftMetadataEpoch() {
+        return kraftMetadataEpoch;
+    }
+
+    public long lastUpdatedTimeMs() {
+        return lastUpdatedTimeMs;
+    }
+
+    public int migrationZkVersion() {
+        return migrationZkVersion;
+    }
+
+    public int controllerZkVersion() {
+        return controllerZkVersion;
+    }
+
+    public boolean zkMigrationComplete() {
+        return kraftMetadataOffset > 0;
+    }
+
+    @Override
+    public String toString() {
+        return "ZkMigrationLeadershipState{" +
+                "kraftControllerId=" + kraftControllerId +
+                ", kraftControllerEpoch=" + kraftControllerEpoch +
+                ", kraftMetadataOffset=" + kraftMetadataOffset +
+                ", kraftMetadataEpoch=" + kraftMetadataEpoch +
+                ", lastUpdatedTimeMs=" + lastUpdatedTimeMs +
+                ", migrationZkVersion=" + migrationZkVersion +
+                ", controllerZkVersion=" + controllerZkVersion +
+                '}';
+    }
+
+    @Override
+    public boolean equals(Object o) {
+        if (this == o) return true;
+        if (o == null || getClass() != o.getClass()) return false;
+        ZkMigrationLeadershipState that = (ZkMigrationLeadershipState) o;
+        return kraftControllerId == that.kraftControllerId && kraftControllerEpoch == that.kraftControllerEpoch && kraftMetadataOffset == that.kraftMetadataOffset && kraftMetadataEpoch == that.kraftMetadataEpoch && lastUpdatedTimeMs == that.lastUpdatedTimeMs && migrationZkVersion == that.migrationZkVersion && controllerZkVersion == that.controllerZkVersion;
+    }
+
+    @Override
+    public int hashCode() {
+        return Objects.hash(kraftControllerId, kraftControllerEpoch, kraftMetadataOffset, kraftMetadataEpoch, lastUpdatedTimeMs, migrationZkVersion, controllerZkVersion);
+    }
+}
diff --git a/metadata/src/main/java/org/apache/kafka/metadata/migration/ZkMigrationState.java b/metadata/src/main/java/org/apache/kafka/metadata/migration/ZkMigrationState.java
new file mode 100644
index 00000000000..239ed0ab7d6
--- /dev/null
+++ b/metadata/src/main/java/org/apache/kafka/metadata/migration/ZkMigrationState.java
@@ -0,0 +1,74 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.metadata.migration;
+
+import java.util.Optional;
+
+/**
+ * An enumeration of the possible states of the ZkMigrationState field in ZkMigrationStateRecord.
+ *
+ * @see org.apache.kafka.common.metadata.ZkMigrationStateRecord
+ */
+public enum ZkMigrationState {
+    /**
+     * No migration has been started by the controller. The controller is in regular KRaft mode
+     */
+    NONE((byte) 0),
+
+    /**
+     * A KRaft controller has been elected with "zookeeper.metadata.migration.enable" set to "true".
+     * The controller is now awaiting the pre-conditions for starting the migration.
+     */
+    PRE_MIGRATION((byte) 1),
+
+    /**
+     * The ZK data has been migrated and the KRaft controller is now writing metadata to both ZK and the
+     * metadata log. The controller will remain in this state until all of the brokers have been restarted
+     * in KRaft mode
+     */
+    MIGRATION((byte) 2),
+
+    /**
+     * The migration has been fully completed. The cluster is running in KRaft mode. This state will persist
+     * indefinitely after the migration.
+     */
+    POST_MIGRATION((byte) 3);
+
+    private final byte value;
+
+    ZkMigrationState(byte value) {
+        this.value = value;
+    }
+
+    public byte value() {
+        return value;
+    }
+
+    public static ZkMigrationState of(byte value) {
+        return optionalOf(value)
+            .orElseThrow(() -> new IllegalArgumentException(String.format("Value %s is not a valid Zk migration state", value)));
+    }
+
+    public static Optional<ZkMigrationState> optionalOf(byte value) {
+        for (ZkMigrationState state : ZkMigrationState.values()) {
+            if (state.value == value) {
+                return Optional.of(state);
+            }
+        }
+        return Optional.empty();
+    }
+}
diff --git a/metadata/src/main/resources/common/metadata/RegisterBrokerRecord.json b/metadata/src/main/resources/common/metadata/RegisterBrokerRecord.json
index a32c16d8a60..80657abb5fe 100644
--- a/metadata/src/main/resources/common/metadata/RegisterBrokerRecord.json
+++ b/metadata/src/main/resources/common/metadata/RegisterBrokerRecord.json
@@ -17,11 +17,13 @@
   "apiKey": 0,
   "type": "metadata",
   "name": "RegisterBrokerRecord",
-  "validVersions": "0-1",
+  "validVersions": "0-2",
   "flexibleVersions": "0+",
   "fields": [
     { "name": "BrokerId", "type": "int32", "versions": "0+", "entityType": "brokerId",
       "about": "The broker id." },
+    { "name": "IsMigratingZkBroker", "type": "bool", "versions": "2+", "default": false,
+      "about": "True if the registering broker is a ZK broker." },
     { "name": "IncarnationId", "type": "uuid", "versions": "0+",
       "about": "The incarnation ID of the broker process" },
     { "name": "BrokerEpoch", "type": "int64", "versions": "0+",
diff --git a/clients/src/main/resources/common/message/UpdateMetadataResponse.json b/metadata/src/main/resources/common/metadata/ZkMigrationRecord.json
similarity index 63%
copy from clients/src/main/resources/common/message/UpdateMetadataResponse.json
copy to metadata/src/main/resources/common/metadata/ZkMigrationRecord.json
index 62203229c97..65910dc6a45 100644
--- a/clients/src/main/resources/common/message/UpdateMetadataResponse.json
+++ b/metadata/src/main/resources/common/metadata/ZkMigrationRecord.json
@@ -14,14 +14,15 @@
 // limitations under the License.
 
 {
-  "apiKey": 6,
-  "type": "response",
-  "name": "UpdateMetadataResponse",
-  // Versions 1, 2, 3, 4, and 5 are the same as version 0
-  "validVersions": "0-7",
-  "flexibleVersions": "6+",
+  "apiKey": 21,
+  "type": "metadata",
+  "name": "ZkMigrationStateRecord",
+  // Version 0 adds ZkMigrationState which is used by the KRaft controller to mark the beginning and end
+  // of the ZK to KRaft migration. Possible values are 1 (PreMigration), 2 (Migration), 3 (PostMigration).
+  "validVersions": "0",
+  "flexibleVersions": "0+",
   "fields": [
-      { "name": "ErrorCode", "type": "int16", "versions": "0+",
-        "about": "The error code, or 0 if there was no error." }
+    { "name": "ZkMigrationState", "type": "int8", "versions": "0+",
+      "about": "One of the possible migration states." }
   ]
-}
+}
\ No newline at end of file
diff --git a/metadata/src/test/java/org/apache/kafka/controller/QuorumControllerTest.java b/metadata/src/test/java/org/apache/kafka/controller/QuorumControllerTest.java
index 27b01881972..10921c77fbd 100644
--- a/metadata/src/test/java/org/apache/kafka/controller/QuorumControllerTest.java
+++ b/metadata/src/test/java/org/apache/kafka/controller/QuorumControllerTest.java
@@ -173,7 +173,7 @@ public class QuorumControllerTest {
         ) {
             controlEnv.activeController().registerBroker(ANONYMOUS_CONTEXT,
                 new BrokerRegistrationRequestData().
-                setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_3_IV3)).
+                setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_4_IV0)).
                 setBrokerId(0).
                 setClusterId(logEnv.clusterId())).get();
             testConfigurationOperations(controlEnv.activeController());
@@ -214,7 +214,7 @@ public class QuorumControllerTest {
         ) {
             controlEnv.activeController().registerBroker(ANONYMOUS_CONTEXT,
                 new BrokerRegistrationRequestData().
-                    setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_3_IV3)).
+                    setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_4_IV0)).
                     setBrokerId(0).
                     setClusterId(logEnv.clusterId())).get();
             testDelayedConfigurationOperations(logEnv, controlEnv.activeController());
@@ -545,7 +545,7 @@ public class QuorumControllerTest {
                     setBrokerId(0).
                     setClusterId(active.clusterId()).
                     setIncarnationId(Uuid.fromString("kxAT73dKQsitIedpiPtwBA")).
-                    setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_3_IV3)).
+                    setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_4_IV0)).
                     setListeners(listeners));
             assertEquals(2L, reply.get().epoch());
             CreateTopicsRequestData createTopicsRequestData =
@@ -781,7 +781,7 @@ public class QuorumControllerTest {
                         setBrokerId(i).
                         setRack(null).
                         setClusterId(active.clusterId()).
-                        setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_3_IV3)).
+                        setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_4_IV0)).
                         setIncarnationId(Uuid.fromString("kxAT73dKQsitIedpiPtwB" + i)).
                         setListeners(new ListenerCollection(Arrays.asList(new Listener().
                             setName("PLAINTEXT").setHost("localhost").
@@ -844,7 +844,7 @@ public class QuorumControllerTest {
                         setBrokerId(i).
                         setRack(null).
                         setClusterId(active.clusterId()).
-                        setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_3_IV3)).
+                        setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_4_IV0)).
                         setIncarnationId(Uuid.fromString("kxAT73dKQsitIedpiPtwB" + i)).
                         setListeners(new ListenerCollection(Arrays.asList(new Listener().
                             setName("PLAINTEXT").setHost("localhost").
@@ -882,7 +882,7 @@ public class QuorumControllerTest {
                         setBrokerId(i).
                         setRack(null).
                         setClusterId(active.clusterId()).
-                        setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_3_IV3)).
+                        setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_4_IV0)).
                         setIncarnationId(Uuid.fromString("kxAT73dKQsitIedpiPtwB" + i)).
                         setListeners(new ListenerCollection(Arrays.asList(new Listener().
                             setName("PLAINTEXT").setHost("localhost").
@@ -1270,7 +1270,7 @@ public class QuorumControllerTest {
                     .setBrokerId(brokerId)
                     .setRack(null)
                     .setClusterId(controller.clusterId())
-                    .setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_3_IV3))
+                    .setFeatures(brokerFeatures(MetadataVersion.IBP_3_0_IV1, MetadataVersion.IBP_3_4_IV0))
                     .setIncarnationId(Uuid.fromString("kxAT73dKQsitIedpiPtwB" + brokerId))
                     .setListeners(
                         new ListenerCollection(
diff --git a/server-common/src/main/java/org/apache/kafka/server/common/MetadataVersion.java b/server-common/src/main/java/org/apache/kafka/server/common/MetadataVersion.java
index 295c91c51fd..ffe7f1e316f 100644
--- a/server-common/src/main/java/org/apache/kafka/server/common/MetadataVersion.java
+++ b/server-common/src/main/java/org/apache/kafka/server/common/MetadataVersion.java
@@ -161,10 +161,13 @@ public enum MetadataVersion {
     IBP_3_3_IV2(6, "3.3", "IV2", true),
 
     // Adds InControlledShutdown state to RegisterBrokerRecord and BrokerRegistrationChangeRecord (KIP-841).
-    IBP_3_3_IV3(7, "3.3", "IV3", true);
+    IBP_3_3_IV3(7, "3.3", "IV3", true),
+
+    // Adds ZK to KRaft migration support (KIP-866). This includes ZkMigrationRecord, a new version of RegisterBrokerRecord,
+    // and updates to a handful of RPCs.
+    IBP_3_4_IV0(8, "3.4", "IV0", true);
 
     // NOTE: update the default version in @ClusterTest annotation to point to the latest version
-    
     public static final String FEATURE_NAME = "metadata.version";
 
     /**