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 tu...@apache.org on 2012/12/10 19:10:23 UTC

svn commit: r1419619 - in /hadoop/common/branches/branch-2/hadoop-mapreduce-project: ./ hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/

Author: tucu
Date: Mon Dec 10 18:10:22 2012
New Revision: 1419619

URL: http://svn.apache.org/viewvc?rev=1419619&view=rev
Log:
MAPREDUCE-4703. Add the ability to start the MiniMRClientCluster using the configurations used before it is being stopped. (ahmed.radwan via tucu)

Modified:
    hadoop/common/branches/branch-2/hadoop-mapreduce-project/CHANGES.txt
    hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientCluster.java
    hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientClusterFactory.java
    hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRYarnClusterAdapter.java
    hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/TestMiniMRClientCluster.java

Modified: hadoop/common/branches/branch-2/hadoop-mapreduce-project/CHANGES.txt
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-2/hadoop-mapreduce-project/CHANGES.txt?rev=1419619&r1=1419618&r2=1419619&view=diff
==============================================================================
--- hadoop/common/branches/branch-2/hadoop-mapreduce-project/CHANGES.txt (original)
+++ hadoop/common/branches/branch-2/hadoop-mapreduce-project/CHANGES.txt Mon Dec 10 18:10:22 2012
@@ -20,6 +20,9 @@ Release 2.0.3-alpha - Unreleased
     HADOOP-8911. CRLF characters in source and text files.
     (Raja Aluri via suresh)
 
+    MAPREDUCE-4703. Add the ability to start the MiniMRClientCluster using 
+    the configurations used before it is being stopped. (ahmed.radwan via tucu)
+
   OPTIMIZATIONS
 
   BUG FIXES

Modified: hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientCluster.java
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientCluster.java?rev=1419619&r1=1419618&r2=1419619&view=diff
==============================================================================
--- hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientCluster.java (original)
+++ hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientCluster.java Mon Dec 10 18:10:22 2012
@@ -31,6 +31,11 @@ public interface MiniMRClientCluster {
 
   public void start() throws IOException;
 
+  /**
+   * Stop and start back the cluster using the same configuration.
+   */
+  public void restart() throws IOException;
+
   public void stop() throws IOException;
 
   public Configuration getConfig() throws IOException;

Modified: hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientClusterFactory.java
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientClusterFactory.java?rev=1419619&r1=1419618&r2=1419619&view=diff
==============================================================================
--- hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientClusterFactory.java (original)
+++ hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientClusterFactory.java Mon Dec 10 18:10:22 2012
@@ -67,6 +67,10 @@ public class MiniMRClientClusterFactory 
 
     MiniMRYarnCluster miniMRYarnCluster = new MiniMRYarnCluster(caller
         .getName(), noOfNMs);
+    job.getConfiguration().set("minimrclientcluster.caller.name",
+        caller.getName());
+    job.getConfiguration().setInt("minimrclientcluster.nodemanagers.number",
+        noOfNMs);
     miniMRYarnCluster.init(job.getConfiguration());
     miniMRYarnCluster.start();
 

Modified: hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRYarnClusterAdapter.java
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRYarnClusterAdapter.java?rev=1419619&r1=1419618&r2=1419619&view=diff
==============================================================================
--- hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRYarnClusterAdapter.java (original)
+++ hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRYarnClusterAdapter.java Mon Dec 10 18:10:22 2012
@@ -18,8 +18,13 @@
 
 package org.apache.hadoop.mapred;
 
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.mapreduce.v2.MiniMRYarnCluster;
+import org.apache.hadoop.mapreduce.v2.jobhistory.JHAdminConfig;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.apache.hadoop.yarn.service.Service.STATE;
 
 /**
  * An adapter for MiniMRYarnCluster providing a MiniMRClientCluster interface.
@@ -29,6 +34,8 @@ public class MiniMRYarnClusterAdapter im
 
   private MiniMRYarnCluster miniMRYarnCluster;
 
+  private static final Log LOG = LogFactory.getLog(MiniMRYarnClusterAdapter.class);
+
   public MiniMRYarnClusterAdapter(MiniMRYarnCluster miniMRYarnCluster) {
     this.miniMRYarnCluster = miniMRYarnCluster;
   }
@@ -48,4 +55,22 @@ public class MiniMRYarnClusterAdapter im
     miniMRYarnCluster.stop();
   }
 
+  @Override
+  public void restart() {
+    if (!miniMRYarnCluster.getServiceState().equals(STATE.STARTED)){
+      LOG.warn("Cannot restart the mini cluster, start it first");
+      return;
+    }
+    Configuration oldConf = new Configuration(getConfig());
+    String callerName = oldConf.get("minimrclientcluster.caller.name",
+        this.getClass().getName());
+    int noOfNMs = oldConf.getInt("minimrclientcluster.nodemanagers.number", 1);
+    oldConf.setBoolean(YarnConfiguration.YARN_MINICLUSTER_FIXED_PORTS, true);
+    oldConf.setBoolean(JHAdminConfig.MR_HISTORY_MINICLUSTER_FIXED_PORTS, true);
+    stop();
+    miniMRYarnCluster = new MiniMRYarnCluster(callerName, noOfNMs);
+    miniMRYarnCluster.init(oldConf);
+    miniMRYarnCluster.start();
+  }
+
 }

Modified: hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/TestMiniMRClientCluster.java
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/TestMiniMRClientCluster.java?rev=1419619&r1=1419618&r2=1419619&view=diff
==============================================================================
--- hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/TestMiniMRClientCluster.java (original)
+++ hadoop/common/branches/branch-2/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/TestMiniMRClientCluster.java Mon Dec 10 18:10:22 2012
@@ -32,6 +32,8 @@ import org.apache.hadoop.io.IntWritable;
 import org.apache.hadoop.io.Text;
 import org.apache.hadoop.mapreduce.Counters;
 import org.apache.hadoop.mapreduce.Job;
+import org.apache.hadoop.mapreduce.v2.jobhistory.JHAdminConfig;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.junit.AfterClass;
 import org.junit.BeforeClass;
 import org.junit.Test;
@@ -92,6 +94,65 @@ public class TestMiniMRClientCluster {
   }
 
   @Test
+  public void testRestart() throws Exception {
+
+    String rmAddress1 = mrCluster.getConfig().get(YarnConfiguration.RM_ADDRESS);
+    String rmAdminAddress1 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_ADMIN_ADDRESS);
+    String rmSchedAddress1 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_SCHEDULER_ADDRESS);
+    String rmRstrackerAddress1 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_RESOURCE_TRACKER_ADDRESS);
+    String rmWebAppAddress1 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_WEBAPP_ADDRESS);
+
+    String mrHistAddress1 = mrCluster.getConfig().get(
+        JHAdminConfig.MR_HISTORY_ADDRESS);
+    String mrHistWebAppAddress1 = mrCluster.getConfig().get(
+        JHAdminConfig.MR_HISTORY_WEBAPP_ADDRESS);
+
+    mrCluster.restart();
+
+    String rmAddress2 = mrCluster.getConfig().get(YarnConfiguration.RM_ADDRESS);
+    String rmAdminAddress2 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_ADMIN_ADDRESS);
+    String rmSchedAddress2 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_SCHEDULER_ADDRESS);
+    String rmRstrackerAddress2 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_RESOURCE_TRACKER_ADDRESS);
+    String rmWebAppAddress2 = mrCluster.getConfig().get(
+        YarnConfiguration.RM_WEBAPP_ADDRESS);
+
+    String mrHistAddress2 = mrCluster.getConfig().get(
+        JHAdminConfig.MR_HISTORY_ADDRESS);
+    String mrHistWebAppAddress2 = mrCluster.getConfig().get(
+        JHAdminConfig.MR_HISTORY_WEBAPP_ADDRESS);
+
+    assertEquals("Address before restart: " + rmAddress1
+        + " is different from new address: " + rmAddress2, rmAddress1,
+        rmAddress2);
+    assertEquals("Address before restart: " + rmAdminAddress1
+        + " is different from new address: " + rmAdminAddress2,
+        rmAdminAddress1, rmAdminAddress2);
+    assertEquals("Address before restart: " + rmSchedAddress1
+        + " is different from new address: " + rmSchedAddress2,
+        rmSchedAddress1, rmSchedAddress2);
+    assertEquals("Address before restart: " + rmRstrackerAddress1
+        + " is different from new address: " + rmRstrackerAddress2,
+        rmRstrackerAddress1, rmRstrackerAddress2);
+    assertEquals("Address before restart: " + rmWebAppAddress1
+        + " is different from new address: " + rmWebAppAddress2,
+        rmWebAppAddress1, rmWebAppAddress2);
+    assertEquals("Address before restart: " + mrHistAddress1
+        + " is different from new address: " + mrHistAddress2, mrHistAddress1,
+        mrHistAddress2);
+    assertEquals("Address before restart: " + mrHistWebAppAddress1
+        + " is different from new address: " + mrHistWebAppAddress2,
+        mrHistWebAppAddress1, mrHistWebAppAddress2);
+
+  }
+
+  @Test
   public void testJob() throws Exception {
     final Job job = createJob();
     org.apache.hadoop.mapreduce.lib.input.FileInputFormat.setInputPaths(job,