Browse Source

commit 760a22749bcd2be9a10c875f56bc5d8cdc6d29ae
Author: Yahoo\! <ltucker@yahoo-inc.com>
Date: Thu Sep 24 16:07:40 2009 -0700

HADOOP-5784. Makes the number of heartbeats that should arrive a second at the JobTracker configurable. Contributed by Amareshwari Sriramadasu.

from: http://issues.apache.org/jira/secure/attachment/12420257/HADOOP-5784_yhadoop20.patch

+++ b/YAHOO-CHANGES.txt
+55. HADOOP-5784. Makes the number of heartbeats that should arrive
+ a second at the JobTracker configurable. Contributed by
+ Amareshwari Sriramadasu.
+


git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-0.20-security-patches@1077003 13f79535-47bb-0310-9956-ffa450edef68

Owen O'Malley 14 years ago
parent
commit
ad1acedb5e

+ 9 - 0
src/mapred/mapred-default.xml

@@ -607,6 +607,15 @@
   hosts are excluded.</description>
 </property>
 
+<property>
+  <name>mapred.heartbeats.in.second</name>
+  <value>100</value>
+  <description>Expert: Approximate number of heart-beats that could arrive 
+               JobTracker in a second. Assuming each RPC can be processed 
+               in 10msec, the default value is made 100 RPCs in a second.
+  </description>
+</property> 
+
 <property>
   <name>mapred.max.tracker.blacklists</name>
   <value>4</value>

+ 6 - 1
src/mapred/org/apache/hadoop/mapred/JobTracker.java

@@ -133,6 +133,9 @@ public class JobTracker implements MRConstants, InterTrackerProtocol,
   // The maximum number of blacklists for a tracker after which the 
   // tracker could be blacklisted across all jobs
   private int MAX_BLACKLISTS_PER_TRACKER = 4;
+  // Approximate number of heartbeats that could arrive JobTracker
+  // in a second
+  private int NUM_HEARTBEATS_IN_SECOND = 100;
   public static enum State { INITIALIZING, RUNNING }
   State state = State.INITIALIZING;
   private static final int FS_ACCESS_RETRY_PERIOD = 10000;
@@ -1905,6 +1908,8 @@ public class JobTracker implements MRConstants, InterTrackerProtocol,
     MAX_COMPLETE_USER_JOBS_IN_MEMORY = conf.getInt("mapred.jobtracker.completeuserjobs.maximum", 100);
     MAX_BLACKLISTS_PER_TRACKER = 
         conf.getInt("mapred.max.tracker.blacklists", 4);
+    NUM_HEARTBEATS_IN_SECOND = 
+        conf.getInt("mapred.heartbeats.in.second", 100);
 
     //This configuration is there solely for tuning purposes and 
     //once this feature has been tested in real clusters and an appropriate
@@ -2982,7 +2987,7 @@ public class JobTracker implements MRConstants, InterTrackerProtocol,
     int clusterSize = getClusterStatus().getTaskTrackers();
     int heartbeatInterval =  Math.max(
                                 (int)(1000 * Math.ceil((double)clusterSize / 
-                                                       CLUSTER_INCREMENT)),
+                                                       NUM_HEARTBEATS_IN_SECOND)),
                                 HEARTBEAT_INTERVAL_MIN) ;
     return heartbeatInterval;
   }

+ 0 - 2
src/mapred/org/apache/hadoop/mapred/MRConstants.java

@@ -27,8 +27,6 @@ interface MRConstants {
   //
   public static final int HEARTBEAT_INTERVAL_MIN = 3 * 1000;
   
-  public static final int CLUSTER_INCREMENT = 100;
-
   public static final long COUNTER_UPDATE_INTERVAL = 60 * 1000;
 
   //

+ 73 - 0
src/test/org/apache/hadoop/mapred/TestMapredHeartbeat.java

@@ -0,0 +1,73 @@
+/**
+ * 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.mapred;
+
+import java.io.IOException;
+
+import junit.framework.TestCase;
+
+import org.apache.hadoop.mapred.JobConf;
+
+public class TestMapredHeartbeat extends TestCase {
+  public void testJobDirCleanup() throws IOException {
+    MiniMRCluster mr = null;
+    try {
+      // test the default heartbeat interval
+      int taskTrackers = 2;
+      JobConf conf = new JobConf();
+      mr = new MiniMRCluster(taskTrackers, "file:///", 3, 
+          null, null, conf);
+      JobClient jc = new JobClient(mr.createJobConf());
+      while(jc.getClusterStatus().getTaskTrackers() != taskTrackers) {
+        UtilsForTests.waitFor(100);
+      }
+      assertEquals(MRConstants.HEARTBEAT_INTERVAL_MIN, 
+        mr.getJobTrackerRunner().getJobTracker().getNextHeartbeatInterval());
+      mr.shutdown(); 
+      
+      // test configured heartbeat interval
+      taskTrackers = 5;
+      conf.setInt("mapred.heartbeats.in.second", 1);
+      mr = new MiniMRCluster(taskTrackers, "file:///", 3, 
+          null, null, conf);
+      jc = new JobClient(mr.createJobConf());
+      while(jc.getClusterStatus().getTaskTrackers() != taskTrackers) {
+        UtilsForTests.waitFor(100);
+      }
+      assertEquals(taskTrackers * 1000, 
+        mr.getJobTrackerRunner().getJobTracker().getNextHeartbeatInterval());
+      mr.shutdown(); 
+      
+      // test configured heartbeat interval is capped with min value
+      taskTrackers = 5;
+      conf.setInt("mapred.heartbeats.in.second", 10);
+      mr = new MiniMRCluster(taskTrackers, "file:///", 3, 
+          null, null, conf);
+      jc = new JobClient(mr.createJobConf());
+      while(jc.getClusterStatus().getTaskTrackers() != taskTrackers) {
+        UtilsForTests.waitFor(100);
+      }
+      assertEquals(MRConstants.HEARTBEAT_INTERVAL_MIN, 
+        mr.getJobTrackerRunner().getJobTracker().getNextHeartbeatInterval());
+    } finally {
+      if (mr != null) { mr.shutdown(); }
+    }
+  }
+}
+
+