You are viewing a plain text version of this content. The canonical link for it is here.
Posted to jira@kafka.apache.org by GitBox <gi...@apache.org> on 2020/12/04 18:48:55 UTC

[GitHub] [kafka] mumrah opened a new pull request #9693: Refactor ZK ISR updates to use AlterIsrManager

mumrah opened a new pull request #9693:
URL: https://github.com/apache/kafka/pull/9693


   In an effort to consolidate the ISR write paths, this PR adapts the ZK ISR update into the new AlterIsrManager trait. This will allow us minimize divergence in the ISR update code in Partition.scala. 
   
   TODO:
   * [ ] Remove expandIsr and shrinkIsr from PartitionStateStore
   * [ ] Move ISR propagation thread to ZkIsrManager
   
   


----------------------------------------------------------------
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.

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



[GitHub] [kafka] mumrah commented on a change in pull request #9693: Refactor ZK ISR updates to use AlterIsrManager

Posted by GitBox <gi...@apache.org>.
mumrah commented on a change in pull request #9693:
URL: https://github.com/apache/kafka/pull/9693#discussion_r538342108



##########
File path: core/src/main/scala/kafka/server/AlterIsrManager.scala
##########
@@ -35,8 +36,12 @@ import scala.collection.mutable.ListBuffer
 import scala.jdk.CollectionConverters._
 
 /**
- * Handles the sending of AlterIsr requests to the controller. Updating the ISR is an asynchronous operation,
- * so partitions will learn about updates through LeaderAndIsr messages sent from the controller
+ * Handles updating the ISR by sending AlterIsr requests to the controller (as of 2.7) or by updating ZK directly
+ * (prior to 2.7). Updating the ISR is an asynchronous operation, so partitions will learn about the result of their
+ * request through a callback.
+ *
+ * Note that ISR state changes can still be initiated by the controller and sent to the partitions via LeaderAndIsr
+ * requests.
  */
 trait AlterIsrManager {

Review comment:
       I wonder if we should rename this IsrManager now?




----------------------------------------------------------------
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.

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



[GitHub] [kafka] mumrah closed pull request #9693: KAFKA-10825 Refactor ZK ISR updates to use AlterIsrManager

Posted by GitBox <gi...@apache.org>.
mumrah closed pull request #9693:
URL: https://github.com/apache/kafka/pull/9693


   


----------------------------------------------------------------
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.

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



[GitHub] [kafka] mumrah commented on a change in pull request #9693: Refactor ZK ISR updates to use AlterIsrManager

Posted by GitBox <gi...@apache.org>.
mumrah commented on a change in pull request #9693:
URL: https://github.com/apache/kafka/pull/9693#discussion_r536307517



##########
File path: core/src/main/scala/kafka/server/ZkIsrManager.scala
##########
@@ -0,0 +1,51 @@
+/**
+ * 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 kafka.server
+
+import kafka.utils.{Logging, ReplicationUtils}
+import kafka.zk.KafkaZkClient
+import org.apache.kafka.common.TopicPartition
+import org.apache.kafka.common.protocol.Errors
+
+class ZkIsrManager(zkClient: KafkaZkClient) extends AlterIsrManager with Logging {
+  override def start(): Unit = {
+    // No async processing is done, so nothing to do here
+  }
+
+  override def clearPending(topicPartition: TopicPartition): Unit = {
+    // Since we always immediately process ZK updates and never actually enqueue anything, there is nothing to
+    // clear here so this is a no-op
+  }
+
+  override def enqueue(alterIsrItem: AlterIsrItem): Boolean = {
+    debug(s"Writing new ISR to ZooKeeper")

Review comment:
       Not sure we need this, I just wanted something in the logs to indicate which implementation is being used




----------------------------------------------------------------
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.

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



[GitHub] [kafka] hachikuji commented on a change in pull request #9693: Refactor ZK ISR updates to use AlterIsrManager

Posted by GitBox <gi...@apache.org>.
hachikuji commented on a change in pull request #9693:
URL: https://github.com/apache/kafka/pull/9693#discussion_r537704021



##########
File path: core/src/main/scala/kafka/server/ZkIsrManager.scala
##########
@@ -0,0 +1,51 @@
+/**
+ * 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 kafka.server
+
+import kafka.utils.{Logging, ReplicationUtils}
+import kafka.zk.KafkaZkClient
+import org.apache.kafka.common.TopicPartition
+import org.apache.kafka.common.protocol.Errors
+
+class ZkIsrManager(zkClient: KafkaZkClient) extends AlterIsrManager with Logging {
+  override def start(): Unit = {
+    // No async processing is done, so nothing to do here
+  }
+
+  override def clearPending(topicPartition: TopicPartition): Unit = {
+    // Since we always immediately process ZK updates and never actually enqueue anything, there is nothing to
+    // clear here so this is a no-op
+  }
+
+  override def enqueue(alterIsrItem: AlterIsrItem): Boolean = {
+    debug(s"Writing new ISR to ZooKeeper")

Review comment:
       The logging seems fine, but maybe we can enrich it with details from `alterIsrItem`?




----------------------------------------------------------------
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.

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



[GitHub] [kafka] mumrah commented on pull request #9693: KAFKA-10825 Refactor ZK ISR updates to use AlterIsrManager

Posted by GitBox <gi...@apache.org>.
mumrah commented on pull request #9693:
URL: https://github.com/apache/kafka/pull/9693#issuecomment-740893777


   Closed in favor of #9713 (fixed branch name for JIRA)


----------------------------------------------------------------
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.

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



[GitHub] [kafka] mumrah commented on pull request #9693: Refactor ZK ISR updates to use AlterIsrManager

Posted by GitBox <gi...@apache.org>.
mumrah commented on pull request #9693:
URL: https://github.com/apache/kafka/pull/9693#issuecomment-740123591


   @hachikuji yes indeed. I just pushed a commit which does just that (removes PartitionStateStore)
   
   I also pushed a change which relocates the ISR propagation logic to ZkIsrManager


----------------------------------------------------------------
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.

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