cmccabe commented on code in PR #14208:
URL: https://github.com/apache/kafka/pull/14208#discussion_r1297529002


##########
metadata/src/main/java/org/apache/kafka/controller/ActivationRecordsGenerator.java:
##########
@@ -0,0 +1,197 @@
+/*
+ * 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.controller;
+
+import org.apache.kafka.common.metadata.AbortTransactionRecord;
+import org.apache.kafka.common.metadata.BeginTransactionRecord;
+import org.apache.kafka.common.metadata.EndTransactionRecord;
+import org.apache.kafka.metadata.bootstrap.BootstrapMetadata;
+import org.apache.kafka.metadata.migration.ZkMigrationState;
+import org.apache.kafka.server.common.ApiMessageAndVersion;
+import org.apache.kafka.server.common.MetadataVersion;
+import org.slf4j.Logger;
+
+import java.util.ArrayList;
+import java.util.List;
+
+public class ActivationRecordsGenerator {
+
+    static ControllerResult<Void> recordsForEmptyLog(
+        Logger log,
+        long transactionStartOffset,
+        boolean zkMigrationEnabled,
+        BootstrapMetadata bootstrapMetadata,
+        MetadataVersion metadataVersion
+    ) {
+        List<ApiMessageAndVersion> records = new ArrayList<>();
+
+        if (transactionStartOffset != -1L) {
+            // In-flight bootstrap transaction
+            if (!metadataVersion.isMetadataTransactionSupported()) {
+                throw new RuntimeException("Detected partial bootstrap records 
transaction at " +
+                    transactionStartOffset + ", but the metadata.version " + 
metadataVersion +
+                    " does not support transactions. Cannot continue.");
+            } else {
+                log.warn("Detected partial bootstrap records transaction at " 
+ transactionStartOffset +
+                    " during controller activation. Aborting this transaction 
and re-committing bootstrap records");
+                records.add(new ApiMessageAndVersion(
+                    new AbortTransactionRecord().setReason("Controller 
failover"), (short) 0));
+                records.add(new ApiMessageAndVersion(
+                    new BeginTransactionRecord().setName("Bootstrap records"), 
(short) 0));
+            }
+        } else if (metadataVersion.isMetadataTransactionSupported()) {
+            // No in-flight transaction
+            records.add(new ApiMessageAndVersion(
+                new BeginTransactionRecord().setName("Bootstrap records"), 
(short) 0));
+        }
+
+        // If no records have been replayed, we need to write out the 
bootstrap records.
+        // This will include the new metadata.version, as well as things like 
SCRAM
+        // initialization, etc.
+        log.info("The metadata log appears to be empty. Appending {} bootstrap 
record(s) " +
+            "at metadata.version {} from {}.",
+            bootstrapMetadata.records().size(),
+            metadataVersion,
+            bootstrapMetadata.source());
+        records.addAll(bootstrapMetadata.records());
+
+        if (metadataVersion.isMigrationSupported()) {
+            if (zkMigrationEnabled) {
+                log.info("Putting the controller into pre-migration mode. No 
metadata updates will be allowed " +
+                    "until the ZK metadata has been migrated");
+                records.add(ZkMigrationState.PRE_MIGRATION.toRecord());
+            } else {
+                log.debug("Setting the ZK migration state to NONE since this 
is a de-novo KRaft cluster.");
+                records.add(ZkMigrationState.NONE.toRecord());
+            }
+        } else {
+            if (zkMigrationEnabled) {
+                throw new RuntimeException("The bootstrap metadata.version " + 
bootstrapMetadata.metadataVersion() +
+                    " does not support ZK migrations. Cannot continue with ZK 
migrations enabled.");
+            }
+        }
+
+        if (metadataVersion.isMetadataTransactionSupported()) {
+            records.add(new ApiMessageAndVersion(new EndTransactionRecord(), 
(short) 0));
+            return ControllerResult.of(records, null);
+        } else {
+            return ControllerResult.atomicOf(records, null);
+        }
+    }
+
+    static ControllerResult<Void> recordsForNonEmptyLog(
+        Logger log,
+        long transactionStartOffset,
+        boolean zkMigrationEnabled,
+        FeatureControlManager featureControl,
+        MetadataVersion metadataVersion
+    ) {
+        // Logs have been replayed. We need to initialize some things here if 
upgrading from older KRaft versions
+        List<ApiMessageAndVersion> records = new ArrayList<>();
+
+        if (metadataVersion.equals(MetadataVersion.MINIMUM_KRAFT_VERSION)) {
+            log.info("No metadata.version feature level record was found in 
the log. " +
+                "Treating the log as version {}.", 
MetadataVersion.MINIMUM_KRAFT_VERSION);
+        }
+
+        if (metadataVersion.isMigrationSupported()) {
+            log.info("Loaded ZK migration state of {}", 
featureControl.zkMigrationState());
+            switch (featureControl.zkMigrationState()) {
+                case NONE:
+                    // Since this is the default state there may or may not be 
an actual NONE in the log. Regardless,
+                    // it will eventually be persisted in a snapshot, so we 
don't need to explicitly write it here.
+                    if (zkMigrationEnabled) {
+                        throw new RuntimeException("Should not have ZK 
migrations enabled on a cluster that was " +
+                            "created in KRaft mode.");
+                    }
+                    break;
+                case PRE_MIGRATION:
+                    log.warn("Activating pre-migration controller without 
empty log. There may be a partial migration");
+                    break;
+                case MIGRATION:
+                    if (!zkMigrationEnabled) {
+                        // This can happen if controller leadership transfers 
to a controller with migrations enabled
+                        // after another controller had finalized the 
migration. For example, during a rolling restart
+                        // of the controller quorum during which the migration 
config is being set to false.
+                        log.warn("Completing the ZK migration since this 
controller was configured with " +
+                            "'zookeeper.metadata.migration.enable' set to 
'false'.");
+                        
records.add(ZkMigrationState.POST_MIGRATION.toRecord());
+                    } else {
+                        log.info("Staying in the ZK migration since 
'zookeeper.metadata.migration.enable' is still " +

Review Comment:
   "Staying in the ZK migration" -> "Staying in ZK migration mode" ?



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: jira-unsubscr...@kafka.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org

Reply via email to