You are viewing a plain text version of this content. The canonical link for it is here.
Posted to mapreduce-commits@hadoop.apache.org by dd...@apache.org on 2009/08/20 14:56:25 UTC

svn commit: r806152 - in /hadoop/mapreduce/trunk: ./ src/java/org/apache/hadoop/mapred/ src/test/ src/test/mapred/org/apache/hadoop/mapred/

Author: ddas
Date: Thu Aug 20 12:56:25 2009
New Revision: 806152

URL: http://svn.apache.org/viewvc?rev=806152&view=rev
Log:
MAPREDUCE-745. Fixes a testcase problem to do with generation of JobTracker IDs. Contributed by Amar Kamat.

Added:
    hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/TestJobTrackerStart.java
Modified:
    hadoop/mapreduce/trunk/CHANGES.txt
    hadoop/mapreduce/trunk/src/java/org/apache/hadoop/mapred/JobTracker.java
    hadoop/mapreduce/trunk/src/test/findbugsExcludeFile.xml
    hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/MiniMRCluster.java

Modified: hadoop/mapreduce/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/hadoop/mapreduce/trunk/CHANGES.txt?rev=806152&r1=806151&r2=806152&view=diff
==============================================================================
--- hadoop/mapreduce/trunk/CHANGES.txt (original)
+++ hadoop/mapreduce/trunk/CHANGES.txt Thu Aug 20 12:56:25 2009
@@ -413,3 +413,6 @@
     MAPREDUCE-832. Reduce number of warning messages printed when
     deprecated memory variables are used. (Rahul Kumar Singh via yhemanth)
 
+    MAPREDUCE-745. Fixes a testcase problem to do with generation of JobTracker
+    IDs. (Amar Kamat via ddas)
+

Modified: hadoop/mapreduce/trunk/src/java/org/apache/hadoop/mapred/JobTracker.java
URL: http://svn.apache.org/viewvc/hadoop/mapreduce/trunk/src/java/org/apache/hadoop/mapred/JobTracker.java?rev=806152&r1=806151&r2=806152&view=diff
==============================================================================
--- hadoop/mapreduce/trunk/src/java/org/apache/hadoop/mapred/JobTracker.java (original)
+++ hadoop/mapreduce/trunk/src/java/org/apache/hadoop/mapred/JobTracker.java Thu Aug 20 12:56:25 2009
@@ -150,7 +150,9 @@
   final static FsPermission SYSTEM_FILE_PERMISSION =
     FsPermission.createImmutable((short) 0700); // rwx------
   
-  private static Clock clock;
+  private static Clock clock = null;
+  
+  static final Clock DEFAULT_CLOCK = new Clock();
 
   /**
    * A client tried to submit a job before the Job Tracker was ready.
@@ -181,8 +183,13 @@
 
   public static final Log LOG = LogFactory.getLog(JobTracker.class);
     
+  /**
+   * Returns JobTracker's clock. Note that the correct clock implementation will
+   * be obtained only when the JobTracker is initialized. If the JobTracker is
+   * not initialized then the default clock i.e {@link Clock} is returned. 
+   */
   static Clock getClock() {
-    return clock;
+    return clock == null ? DEFAULT_CLOCK : clock;
   }
   
   /**
@@ -198,15 +205,20 @@
   public static JobTracker startTracker(JobConf conf
                                         ) throws IOException,
                                                  InterruptedException {
-    return startTracker(conf, new Clock());
+    return startTracker(conf, DEFAULT_CLOCK);
   }
 
   static JobTracker startTracker(JobConf conf, Clock clock) 
   throws IOException, InterruptedException {
+    return startTracker(conf, clock, generateNewIdentifier());
+  }
+
+  static JobTracker startTracker(JobConf conf, Clock clock, String identifier) 
+  throws IOException, InterruptedException {
     JobTracker result = null;
     while (true) {
       try {
-        result = new JobTracker(conf, clock);
+        result = new JobTracker(conf, clock, identifier);
         result.taskScheduler.setTaskTrackerManager(result);
         break;
       } catch (VersionMismatch e) {
@@ -1104,6 +1116,9 @@
           hasUpdates = true;
           LOG.info("Calling init from RM for job " + jip.getJobID().toString());
           initJob(jip);
+          if (!jip.inited()) {
+            throw new IOException("Failed to initialize job " + jip.getJobID());
+          }
         }
       }
       
@@ -1832,8 +1847,13 @@
    * Start the JobTracker process, listen on the indicated port
    */
   JobTracker(JobConf conf, Clock clock) throws IOException, InterruptedException {
+    this(conf, clock, generateNewIdentifier());
+  }
+
+  JobTracker(JobConf conf, Clock newClock, String jobtrackerIndentifier) 
+  throws IOException, InterruptedException {
     // find the owner of the process
-    this.clock = clock;
+    clock = newClock;
     try {
       mrOwner = UnixUserGroupInformation.login(conf);
     } catch (LoginException e) {
@@ -1928,7 +1948,7 @@
     infoServer.addServlet("reducegraph", "/taskgraph", TaskGraphServlet.class);
     infoServer.start();
     
-    trackerIdentifier = getDateFormat().format(new Date());
+    this.trackerIdentifier = jobtrackerIndentifier;
 
     // Initialize instrumentation
     JobTrackerInstrumentation tmp;
@@ -2037,6 +2057,10 @@
     return new SimpleDateFormat("yyyyMMddHHmm");
   }
 
+  private static String generateNewIdentifier() {
+    return getDateFormat().format(new Date());
+  }
+  
   static boolean validateIdentifier(String id) {
     try {
       // the jobtracker id should be 'date' parseable

Modified: hadoop/mapreduce/trunk/src/test/findbugsExcludeFile.xml
URL: http://svn.apache.org/viewvc/hadoop/mapreduce/trunk/src/test/findbugsExcludeFile.xml?rev=806152&r1=806151&r2=806152&view=diff
==============================================================================
--- hadoop/mapreduce/trunk/src/test/findbugsExcludeFile.xml (original)
+++ hadoop/mapreduce/trunk/src/test/findbugsExcludeFile.xml Thu Aug 20 12:56:25 2009
@@ -80,6 +80,11 @@
        <Bug pattern="DM_EXIT" />
      </Match>
      <Match>
+       <Class name="org.apache.hadoop.mapred.JobTracker" />
+       <Field name="clock" />
+       <Bug pattern="ST_WRITE_TO_STATIC_FROM_INSTANCE_METHOD" />
+     </Match>
+     <Match>
        <Class name="org.apache.hadoop.mapred.Task$TaskReporter" />
        <Method name="run" />
        <Bug pattern="DM_EXIT" />

Modified: hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/MiniMRCluster.java
URL: http://svn.apache.org/viewvc/hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/MiniMRCluster.java?rev=806152&r1=806151&r2=806152&view=diff
==============================================================================
--- hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/MiniMRCluster.java (original)
+++ hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/MiniMRCluster.java Thu Aug 20 12:56:25 2009
@@ -19,7 +19,9 @@
 
 import java.io.File;
 import java.io.IOException;
+import java.text.SimpleDateFormat;
 import java.util.ArrayList;
+import java.util.Date;
 import java.util.Iterator;
 import java.util.List;
 
@@ -67,7 +69,7 @@
     private volatile boolean isActive = true;
     
     JobConf jc = null;
-    Clock clock = null;
+    Clock clock = JobTracker.DEFAULT_CLOCK;
         
     public JobTrackerRunner(JobConf conf) {
       jc = conf;
@@ -108,11 +110,9 @@
         jc.set("mapred.local.dir",f.getAbsolutePath());
         jc.setClass("topology.node.switch.mapping.impl", 
             StaticMapping.class, DNSToSwitchMapping.class);
-        if (clock == null) {
-          tracker = JobTracker.startTracker(jc);
-        } else {
-          tracker = JobTracker.startTracker(jc, clock);
-        }
+        String id = 
+          new SimpleDateFormat("yyyyMMddHHmmssSSS").format(new Date());
+        tracker = JobTracker.startTracker(jc, clock, id);
         tracker.offerService();
       } catch (Throwable e) {
         LOG.error("Job tracker crashed", e);
@@ -328,6 +328,13 @@
     if(conf == null) {
       conf = new JobConf();
     }
+    return configureJobConf(conf, namenode, jobTrackerPort, jobTrackerInfoPort, 
+                            ugi);
+  }
+  
+  static JobConf configureJobConf(JobConf conf, String namenode, 
+                                  int jobTrackerPort, int jobTrackerInfoPort, 
+                                  UnixUserGroupInformation ugi) {
     JobConf result = new JobConf(conf);
     FileSystem.setDefaultUri(result, namenode);
     result.set("mapred.job.tracker", "localhost:"+jobTrackerPort);

Added: hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/TestJobTrackerStart.java
URL: http://svn.apache.org/viewvc/hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/TestJobTrackerStart.java?rev=806152&view=auto
==============================================================================
--- hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/TestJobTrackerStart.java (added)
+++ hadoop/mapreduce/trunk/src/test/mapred/org/apache/hadoop/mapred/TestJobTrackerStart.java Thu Aug 20 12:56:25 2009
@@ -0,0 +1,62 @@
+/**
+ * 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 org.apache.hadoop.mapred.UtilsForTests.FakeClock;
+
+import junit.framework.TestCase;
+
+/**
+ * Test {@link JobTracker} w.r.t config parameters.
+ */
+public class TestJobTrackerStart extends TestCase {
+  
+  public void testJobTrackerStartConfig() throws Exception {
+    JobConf conf = new JobConf();
+    conf = MiniMRCluster.configureJobConf(conf, "file:///", 0, 0, null);
+    
+    // test JobTracker's default clock
+    Clock c = JobTracker.getClock();
+    assertNotNull(c);
+    assertEquals(c, JobTracker.DEFAULT_CLOCK);
+    
+    // test with default values
+    JobTracker jt = JobTracker.startTracker(conf);
+    c = JobTracker.getClock();
+    // test clock
+    assertNotNull(c);
+    assertEquals(c, JobTracker.DEFAULT_CLOCK);
+    // test identifier
+    assertEquals(12, jt.getTrackerIdentifier().length()); // correct upto mins
+    jt.stopTracker();
+    
+    // test with special clock
+    FakeClock myClock = new FakeClock();
+    jt = JobTracker.startTracker(conf, myClock);
+    c = JobTracker.getClock();
+    assertNotNull(c);
+    assertEquals(c, myClock);
+    jt.stopTracker();
+    
+    // test with special identifier
+    String identifier = "test-identifier";
+    jt = JobTracker.startTracker(conf, JobTracker.DEFAULT_CLOCK, identifier);
+    assertEquals(identifier, jt.getTrackerIdentifier());
+    jt.stopTracker();
+  }
+}