Explorar el Código

YARN-4403. (AM/NM/Container)LivelinessMonitor should use monotonic time when calculating period. Contributed by Junping Du

Jian He hace 9 años
padre
commit
1cb3299b48

+ 3 - 0
hadoop-yarn-project/CHANGES.txt

@@ -1129,6 +1129,9 @@ Release 2.8.0 - UNRELEASED
     YARN-4421. Remove dead code in RmAppImpl.RMAppRecoveredTransition.
     (Daniel Templeton via rohithsharmaks)
 
+    YARN-4403. (AM/NM/Container)LivelinessMonitor should use monotonic time
+    when calculating period. (Junping Du via jianhe)
+
 Release 2.7.3 - UNRELEASED
 
   INCOMPATIBLE CHANGES

+ 4 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/util/AbstractLivelinessMonitor.java

@@ -56,6 +56,10 @@ public abstract class AbstractLivelinessMonitor<O> extends AbstractService {
     this.clock = clock;
   }
 
+  public AbstractLivelinessMonitor(String name) {
+    this(name, new MonotonicClock());
+  }
+
   @Override
   protected void serviceStart() throws Exception {
     assert !stopped : "starting when already stopped";

+ 46 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/util/MonotonicClock.java

@@ -0,0 +1,46 @@
+/**
+ * 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.hadoop.yarn.util;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Evolving;
+import org.apache.hadoop.util.Time;
+
+/**
+ * A monotonic clock from some arbitrary time base in the past, counting in
+ * milliseconds, and not affected by settimeofday or similar system clock
+ * changes.
+ * This is appropriate to use when computing how much longer to wait for an
+ * interval to expire.
+ * This function can return a negative value and it must be handled correctly
+ * by callers. See the documentation of System#nanoTime for caveats.
+ */
+@Public
+@Evolving
+public class MonotonicClock implements Clock {
+
+  /**
+   * Get current time from some arbitrary time base in the past, counting in
+   * milliseconds, and not affected by settimeofday or similar system clock
+   * changes.
+   * @return a monotonic clock that counts in milliseconds.
+   */
+  public long getTime() {
+    return Time.monotonicNow();
+  }
+}

+ 4 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/util/SystemClock.java

@@ -23,6 +23,10 @@ import org.apache.hadoop.classification.InterfaceStability.Stable;
 /**
  * Implementation of {@link Clock} that gives the current time from the system
  * clock in milliseconds.
+ * 
+ * NOTE: Do not use this to calculate a duration of expire or interval to sleep,
+ * because it will be broken by settimeofday. Please use {@link MonotonicClock}
+ * instead.
  */
 @Public
 @Stable

+ 1 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/NMLivelinessMonitor.java

@@ -26,14 +26,13 @@ import org.apache.hadoop.yarn.event.EventHandler;
 import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEvent;
 import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEventType;
 import org.apache.hadoop.yarn.util.AbstractLivelinessMonitor;
-import org.apache.hadoop.yarn.util.SystemClock;
 
 public class NMLivelinessMonitor extends AbstractLivelinessMonitor<NodeId> {
 
   private EventHandler dispatcher;
   
   public NMLivelinessMonitor(Dispatcher d) {
-    super("NMLivelinessMonitor", new SystemClock());
+    super("NMLivelinessMonitor");
     this.dispatcher = d.getEventHandler();
   }
 

+ 1 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/rmapp/attempt/AMLivelinessMonitor.java

@@ -25,14 +25,13 @@ import org.apache.hadoop.yarn.event.Dispatcher;
 import org.apache.hadoop.yarn.event.EventHandler;
 import org.apache.hadoop.yarn.util.AbstractLivelinessMonitor;
 import org.apache.hadoop.yarn.util.Clock;
-import org.apache.hadoop.yarn.util.SystemClock;
 
 public class AMLivelinessMonitor extends AbstractLivelinessMonitor<ApplicationAttemptId> {
 
   private EventHandler dispatcher;
   
   public AMLivelinessMonitor(Dispatcher d) {
-    super("AMLivelinessMonitor", new SystemClock());
+    super("AMLivelinessMonitor");
     this.dispatcher = d.getEventHandler();
   }
 

+ 1 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/rmcontainer/ContainerAllocationExpirer.java

@@ -25,7 +25,6 @@ import org.apache.hadoop.yarn.event.Dispatcher;
 import org.apache.hadoop.yarn.event.EventHandler;
 import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.ContainerExpiredSchedulerEvent;
 import org.apache.hadoop.yarn.util.AbstractLivelinessMonitor;
-import org.apache.hadoop.yarn.util.SystemClock;
 
 @SuppressWarnings({"unchecked", "rawtypes"})
 public class ContainerAllocationExpirer extends
@@ -34,7 +33,7 @@ public class ContainerAllocationExpirer extends
   private EventHandler dispatcher;
 
   public ContainerAllocationExpirer(Dispatcher d) {
-    super(ContainerAllocationExpirer.class.getName(), new SystemClock());
+    super(ContainerAllocationExpirer.class.getName());
     this.dispatcher = d.getEventHandler();
   }