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 2023/01/11 01:44:42 UTC

[GitHub] [kafka] junrao commented on a diff in pull request #13092: KAFKA-14607: Move Scheduler/KafkaScheduler to server-common

junrao commented on code in PR #13092:
URL: https://github.com/apache/kafka/pull/13092#discussion_r1066374013


##########
server-common/src/main/java/org/apache/kafka/server/util/KafkaScheduler.java:
##########
@@ -0,0 +1,188 @@
+/*
+ * 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.server.util;
+
+import org.apache.kafka.common.utils.KafkaThread;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.concurrent.Delayed;
+import java.util.concurrent.ScheduledFuture;
+import java.util.concurrent.ScheduledThreadPoolExecutor;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicInteger;
+
+/**
+ * A scheduler based on java.util.concurrent.ScheduledThreadPoolExecutor
+ *
+ * It has a pool of kafka-scheduler- threads that do the actual work.
+ */
+public class KafkaScheduler implements Scheduler {
+
+    private static class NoOpScheduledFutureTask implements ScheduledFuture<Void> {
+
+        @Override
+        public boolean cancel(boolean mayInterruptIfRunning) {
+            return true;
+        }
+
+        @Override
+        public boolean isCancelled() {
+            return true;
+        }
+
+        @Override
+        public boolean isDone() {
+            return true;
+        }
+        @Override

Review Comment:
   add a newline above.



##########
core/src/main/scala/kafka/log/LogManager.scala:
##########
@@ -1049,16 +1045,13 @@ class LogManager(logDirs: Seq[File],
         error(s"Exception in kafka-delete-logs thread.", e)
     } finally {
       try {
-        scheduler.schedule("kafka-delete-logs",
+        scheduler.scheduleOnce("kafka-delete-logs",
           deleteLogs _,
-          delay = nextDelayMs,
-          unit = TimeUnit.MILLISECONDS)
+          nextDelayMs)
       } catch {
         case e: Throwable =>
-          if (scheduler.isStarted) {

Review Comment:
   Thanks for catching this. #11351 changed the scheduler to return NoOpScheduledFutureTask if the scheduler is not started. But we forgot to change the code here accordingly.



-- 
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-unsubscribe@kafka.apache.org

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