瀏覽代碼

YARN-316. YARN container launch may exceed maximum Windows command line length due to long classpath. Contributed by Chris Nauroth.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-trunk-win@1440159 13f79535-47bb-0310-9956-ffa450edef68
Suresh Srinivas 12 年之前
父節點
當前提交
3ba6cff886

+ 94 - 0
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/FileUtil.java

@@ -19,12 +19,19 @@
 package org.apache.hadoop.fs;
 
 import java.io.*;
+import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.Enumeration;
+import java.util.List;
+import java.util.Map;
+import java.util.jar.Attributes;
+import java.util.jar.JarOutputStream;
+import java.util.jar.Manifest;
 import java.util.zip.GZIPInputStream;
 import java.util.zip.ZipEntry;
 import java.util.zip.ZipFile;
 
+import org.apache.commons.collections.map.CaseInsensitiveMap;
 import org.apache.commons.compress.archivers.tar.TarArchiveEntry;
 import org.apache.commons.compress.archivers.tar.TarArchiveInputStream;
 import org.apache.hadoop.classification.InterfaceAudience;
@@ -1009,4 +1016,91 @@ public class FileUtil {
     }
     return fileNames;
   }  
+  
+  /**
+   * Create a jar file at the given path, containing a manifest with a classpath
+   * that references all specified entries.
+   * 
+   * Some platforms may have an upper limit on command line length.  For example,
+   * the maximum command line length on Windows is 8191 characters, but the
+   * length of the classpath may exceed this.  To work around this limitation,
+   * use this method to create a small intermediate jar with a manifest that
+   * contains the full classpath.  It returns the absolute path to the new jar,
+   * which the caller may set as the classpath for a new process.
+   * 
+   * Environment variable evaluation is not supported within a jar manifest, so
+   * this method expands environment variables before inserting classpath entries
+   * to the manifest.  The method parses environment variables according to
+   * platform-specific syntax (%VAR% on Windows, or $VAR otherwise).  On Windows,
+   * environment variables are case-insensitive.  For example, %VAR% and %var%
+   * evaluate to the same value.
+   * 
+   * Specifying the classpath in a jar manifest does not support wildcards, so
+   * this method expands wildcards internally.  Any classpath entry that ends
+   * with * is translated to all files at that path with extension .jar or .JAR.
+   * 
+   * @param inputClassPath String input classpath to bundle into the jar manifest
+   * @param pwd Path to working directory to save jar
+   * @return String absolute path to new jar
+   * @throws IOException if there is an I/O error while writing the jar file
+   */
+  public static String createJarWithClassPath(String inputClassPath, Path pwd)
+      throws IOException {
+    // Replace environment variables, case-insensitive on Windows
+    @SuppressWarnings("unchecked")
+    Map<String, String> env = Shell.WINDOWS ?
+      new CaseInsensitiveMap(System.getenv()) : System.getenv();
+    String[] classPathEntries = inputClassPath.split(File.pathSeparator);
+    for (int i = 0; i < classPathEntries.length; ++i) {
+      classPathEntries[i] = StringUtils.replaceTokens(classPathEntries[i],
+        StringUtils.ENV_VAR_PATTERN, env);
+    }
+    File workingDir = new File(pwd.toString());
+    workingDir.mkdirs();
+
+    // Append all entries
+    List<String> classPathEntryList = new ArrayList<String>(
+      classPathEntries.length);
+    for (String classPathEntry: classPathEntries) {
+      if (classPathEntry.endsWith("*")) {
+        // Append all jars that match the wildcard
+        Path globPath = new Path(classPathEntry).suffix("{.jar,.JAR}");
+        FileStatus[] wildcardJars = FileContext.getLocalFSFileContext().util()
+          .globStatus(globPath);
+        if (wildcardJars != null) {
+          for (FileStatus wildcardJar: wildcardJars) {
+            classPathEntryList.add(wildcardJar.getPath().toUri().toURL()
+              .toExternalForm());
+          }
+        }
+      } else {
+        // Append just this jar
+        classPathEntryList.add(new File(classPathEntry).toURI().toURL()
+          .toExternalForm());
+      }
+    }
+    String jarClassPath = StringUtils.join(" ", classPathEntryList);
+
+    // Create the manifest
+    Manifest jarManifest = new Manifest();
+    jarManifest.getMainAttributes().putValue(
+        Attributes.Name.MANIFEST_VERSION.toString(), "1.0");
+    jarManifest.getMainAttributes().putValue(
+        Attributes.Name.CLASS_PATH.toString(), jarClassPath);
+
+    // Write the manifest to output JAR file
+    File classPathJar = File.createTempFile("classpath-", ".jar", workingDir);
+    FileOutputStream fos = null;
+    BufferedOutputStream bos = null;
+    JarOutputStream jos = null;
+    try {
+      fos = new FileOutputStream(classPathJar);
+      bos = new BufferedOutputStream(fos);
+      jos = new JarOutputStream(bos, jarManifest);
+    } finally {
+      IOUtils.cleanup(LOG, jos, bos, fos);
+    }
+
+    return classPathJar.getCanonicalPath();
+  }
 }

+ 3 - 11
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/RawLocalFileSystem.java

@@ -612,17 +612,9 @@ public class RawLocalFileSystem extends FileSystem {
       NativeIO.POSIX.chmod(pathToFile(p).getCanonicalPath(),
                      permission.toShort());
     } else {
-      execCommand(pathToFile(p), Shell.SET_PERMISSION_COMMAND,
-          String.format("%05o", permission.toShort()));
+      String perm = String.format("%04o", permission.toShort());
+      Shell.execCommand(Shell.getSetPermissionCommand(perm, false,
+        FileUtil.makeShellPath(pathToFile(p), true)));
     }
   }
-
-  private static String execCommand(File f, String... cmd) throws IOException {
-    String[] args = new String[cmd.length + 1];
-    System.arraycopy(cmd, 0, args, 0, cmd.length);
-    args[cmd.length] = FileUtil.makeShellPath(f, true);
-    String output = Shell.execCommand(args);
-    return output;
-  }
-
 }

+ 58 - 0
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/util/StringUtils.java

@@ -32,13 +32,17 @@ import java.util.Date;
 import java.util.Iterator;
 import java.util.List;
 import java.util.Locale;
+import java.util.Map;
 import java.util.StringTokenizer;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
 
 import com.google.common.net.InetAddresses;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.net.NetUtils;
+import org.apache.hadoop.util.Shell;
 
 /**
  * General string utils
@@ -52,6 +56,27 @@ public class StringUtils {
    */
   public static final int SHUTDOWN_HOOK_PRIORITY = 0;
 
+  /**
+   * Shell environment variables: $ followed by one letter or _ followed by
+   * multiple letters, numbers, or underscores.  The group captures the
+   * environment variable name without the leading $.
+   */
+  public static final Pattern SHELL_ENV_VAR_PATTERN =
+    Pattern.compile("\\$([A-Za-z_]{1}[A-Za-z0-9_]*)");
+
+  /**
+   * Windows environment variables: surrounded by %.  The group captures the
+   * environment variable name without the leading and trailing %.
+   */
+  public static final Pattern WIN_ENV_VAR_PATTERN = Pattern.compile("%(.*?)%");
+
+  /**
+   * Regular expression that matches and captures environment variable names
+   * according to platform-specific rules.
+   */
+  public static final Pattern ENV_VAR_PATTERN = Shell.WINDOWS ?
+    WIN_ENV_VAR_PATTERN : SHELL_ENV_VAR_PATTERN;
+
   private static final DecimalFormat decimalFormat;
   static {
           NumberFormat numberFormat = NumberFormat.getNumberInstance(Locale.ENGLISH);
@@ -815,4 +840,37 @@ public class StringUtils {
 
     return sb.toString();
   }
+
+  /**
+   * Matches a template string against a pattern, replaces matched tokens with
+   * the supplied replacements, and returns the result.  The regular expression
+   * must use a capturing group.  The value of the first capturing group is used
+   * to look up the replacement.  If no replacement is found for the token, then
+   * it is replaced with the empty string.
+   * 
+   * For example, assume template is "%foo%_%bar%_%baz%", pattern is "%(.*?)%",
+   * and replacements contains 2 entries, mapping "foo" to "zoo" and "baz" to
+   * "zaz".  The result returned would be "zoo__zaz".
+   * 
+   * @param template String template to receive replacements
+   * @param pattern Pattern to match for identifying tokens, must use a capturing
+   *   group
+   * @param replacements Map<String, String> mapping tokens identified by the
+   *   capturing group to their replacement values
+   * @return String template with replacements
+   */
+  public static String replaceTokens(String template, Pattern pattern,
+      Map<String, String> replacements) {
+    StringBuffer sb = new StringBuffer();
+    Matcher matcher = pattern.matcher(template);
+    while (matcher.find()) {
+      String replacement = replacements.get(matcher.group(1));
+      if (replacement == null) {
+        replacement = "";
+      }
+      matcher.appendReplacement(sb, Matcher.quoteReplacement(replacement));
+    }
+    matcher.appendTail(sb);
+    return sb.toString();
+  }
 }

+ 70 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/fs/TestFileUtil.java

@@ -25,14 +25,19 @@ import java.io.FileOutputStream;
 import java.io.FileReader;
 import java.io.IOException;
 import java.io.PrintWriter;
+import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.Collections;
 import java.util.List;
+import java.util.jar.Attributes;
+import java.util.jar.JarFile;
+import java.util.jar.Manifest;
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.util.Shell;
+import org.apache.hadoop.util.StringUtils;
 import org.junit.After;
 import org.junit.Assert;
 import org.junit.Test;
@@ -727,4 +732,69 @@ public class TestFileUtil {
     doUntarAndVerify(new File(tarGzFileName), untarDir);
     doUntarAndVerify(new File(tarFileName), untarDir);
   }
+
+  @Test
+  public void testCreateJarWithClassPath() throws Exception {
+    // setup test directory for files
+    Assert.assertFalse(tmp.exists());
+    Assert.assertTrue(tmp.mkdirs());
+
+    // create files expected to match a wildcard
+    List<File> wildcardMatches = Arrays.asList(new File(tmp, "wildcard1.jar"),
+      new File(tmp, "wildcard2.jar"), new File(tmp, "wildcard3.JAR"),
+      new File(tmp, "wildcard4.JAR"));
+    for (File wildcardMatch: wildcardMatches) {
+      Assert.assertTrue("failure creating file: " + wildcardMatch,
+        wildcardMatch.createNewFile());
+    }
+
+    // create non-jar files, which we expect to not be included in the classpath
+    Assert.assertTrue(new File(tmp, "text.txt").createNewFile());
+    Assert.assertTrue(new File(tmp, "executable.exe").createNewFile());
+    Assert.assertTrue(new File(tmp, "README").createNewFile());
+
+    // create classpath jar
+    String wildcardPath = tmp.getCanonicalPath() + File.separator + "*";
+    List<String> classPaths = Arrays.asList("cp1.jar", "cp2.jar", wildcardPath,
+      "cp3.jar");
+    String inputClassPath = StringUtils.join(File.pathSeparator, classPaths);
+    String classPathJar = FileUtil.createJarWithClassPath(inputClassPath,
+      new Path(tmp.getCanonicalPath()));
+
+    // verify classpath by reading manifest from jar file
+    JarFile jarFile = null;
+    try {
+      jarFile = new JarFile(classPathJar);
+      Manifest jarManifest = jarFile.getManifest();
+      Assert.assertNotNull(jarManifest);
+      Attributes mainAttributes = jarManifest.getMainAttributes();
+      Assert.assertNotNull(mainAttributes);
+      Assert.assertTrue(mainAttributes.containsKey(Attributes.Name.CLASS_PATH));
+      String classPathAttr = mainAttributes.getValue(Attributes.Name.CLASS_PATH);
+      Assert.assertNotNull(classPathAttr);
+      List<String> expectedClassPaths = new ArrayList<String>();
+      for (String classPath: classPaths) {
+        if (!wildcardPath.equals(classPath)) {
+          expectedClassPaths.add(new File(classPath).toURI().toURL()
+            .toExternalForm());
+        } else {
+          // add wildcard matches
+          for (File wildcardMatch: wildcardMatches) {
+            expectedClassPaths.add(wildcardMatch.toURI().toURL()
+              .toExternalForm());
+          }
+        }
+      }
+      List<String> actualClassPaths = Arrays.asList(classPathAttr.split(" "));
+      Assert.assertEquals(expectedClassPaths, actualClassPaths);
+    } finally {
+      if (jarFile != null) {
+        try {
+          jarFile.close();
+        } catch (IOException e) {
+          LOG.warn("exception closing jarFile: " + classPathJar, e);
+        }
+      }
+    }
+  }
 }

+ 46 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/util/TestStringUtils.java

@@ -23,7 +23,10 @@ import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.fail;
 
 import java.util.ArrayList;
+import java.util.HashMap;
 import java.util.List;
+import java.util.Map;
+import java.util.regex.Pattern;
 
 import org.apache.hadoop.test.UnitTestcaseTimeLimit;
 import org.junit.Test;
@@ -295,6 +298,49 @@ public class TestStringUtils extends UnitTestcaseTimeLimit {
             StringUtils.simpleHostname("10.10.5.68"));
   }
 
+  @Test
+  public void testReplaceTokensShellEnvVars() {
+    Pattern pattern = StringUtils.SHELL_ENV_VAR_PATTERN;
+    Map<String, String> replacements = new HashMap<String, String>();
+    replacements.put("FOO", "one");
+    replacements.put("BAZ", "two");
+    replacements.put("NUMBERS123", "one-two-three");
+    replacements.put("UNDER_SCORES", "___");
+
+    assertEquals("one", StringUtils.replaceTokens("$FOO", pattern,
+      replacements));
+    assertEquals("two", StringUtils.replaceTokens("$BAZ", pattern,
+      replacements));
+    assertEquals("", StringUtils.replaceTokens("$BAR", pattern, replacements));
+    assertEquals("", StringUtils.replaceTokens("", pattern, replacements));
+    assertEquals("one-two-three", StringUtils.replaceTokens("$NUMBERS123",
+      pattern, replacements));
+    assertEquals("___", StringUtils.replaceTokens("$UNDER_SCORES", pattern,
+      replacements));
+    assertEquals("//one//two//", StringUtils.replaceTokens("//$FOO/$BAR/$BAZ//",
+      pattern, replacements));
+  }
+
+  @Test
+  public void testReplaceTokensWinEnvVars() {
+    Pattern pattern = StringUtils.WIN_ENV_VAR_PATTERN;
+    Map<String, String> replacements = new HashMap<String, String>();
+    replacements.put("foo", "zoo");
+    replacements.put("baz", "zaz");
+
+    assertEquals("zoo", StringUtils.replaceTokens("%foo%", pattern,
+      replacements));
+    assertEquals("zaz", StringUtils.replaceTokens("%baz%", pattern,
+      replacements));
+    assertEquals("", StringUtils.replaceTokens("%bar%", pattern,
+      replacements));
+    assertEquals("", StringUtils.replaceTokens("", pattern, replacements));
+    assertEquals("zoo__zaz", StringUtils.replaceTokens("%foo%_%bar%_%baz%",
+      pattern, replacements));
+    assertEquals("begin zoo__zaz end", StringUtils.replaceTokens(
+      "begin %foo%_%bar%_%baz% end", pattern, replacements));
+  }
+
   // Benchmark for StringUtils split
   public static void main(String []args) {
     final String TO_SPLIT = "foo,bar,baz,blah,blah";

+ 3 - 3
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/MiniMRClientClusterFactory.java

@@ -45,7 +45,7 @@ public class MiniMRClientClusterFactory {
 
     FileSystem fs = FileSystem.get(conf);
 
-    Path testRootDir = new Path("target", caller.getName() + "-tmpDir")
+    Path testRootDir = new Path("target", caller.getSimpleName() + "-tmpDir")
         .makeQualified(fs);
     Path appJar = new Path(testRootDir, "MRAppJar.jar");
 
@@ -66,9 +66,9 @@ public class MiniMRClientClusterFactory {
     job.addFileToClassPath(remoteCallerJar);
 
     MiniMRYarnCluster miniMRYarnCluster = new MiniMRYarnCluster(caller
-        .getName(), noOfNMs);
+        .getSimpleName(), noOfNMs);
     job.getConfiguration().set("minimrclientcluster.caller.name",
-        caller.getName());
+        caller.getSimpleName());
     job.getConfiguration().setInt("minimrclientcluster.nodemanagers.number",
         noOfNMs);
     miniMRYarnCluster.init(job.getConfiguration());

+ 2 - 0
hadoop-yarn-project/CHANGES.branch-trunk-win.txt

@@ -22,3 +22,5 @@ branch-trunk-win changes - unreleased
   YARN-259. Fix LocalDirsHandlerService to use Path rather than URIs. (Xuan
   Gong via acmurthy) 
 
+  YARN-316. YARN container launch may exceed maximum Windows command line 
+  length due to long classpath. (Chris Nauroth via suresh)

+ 15 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-nodemanager/src/main/java/org/apache/hadoop/yarn/server/nodemanager/DefaultContainerExecutor.java

@@ -55,6 +55,8 @@ public class DefaultContainerExecutor extends ContainerExecutor {
   private static final Log LOG = LogFactory
       .getLog(DefaultContainerExecutor.class);
 
+  private static final int WIN_MAX_PATH = 260;
+
   private final FileContext lfs;
 
   public DefaultContainerExecutor() {
@@ -148,6 +150,17 @@ public class DefaultContainerExecutor extends ContainerExecutor {
       new WindowsLocalWrapperScriptBuilder(containerIdStr, containerWorkDir) :
       new UnixLocalWrapperScriptBuilder(containerWorkDir);
 
+    // Fail fast if attempting to launch the wrapper script would fail due to
+    // Windows path length limitation.
+    if (Shell.WINDOWS &&
+        sb.getWrapperScriptPath().toString().length() > WIN_MAX_PATH) {
+      throw new IOException(String.format(
+        "Cannot launch container using script at path %s, because it exceeds " +
+        "the maximum supported path length of %d characters.  Consider " +
+        "configuring shorter directories in %s.", sb.getWrapperScriptPath(),
+        WIN_MAX_PATH, YarnConfiguration.NM_LOCAL_DIRS));
+    }
+
     Path pidFile = getPidFilePath(containerId);
     if (pidFile != null) {
       sb.writeLocalWrapperScript(launchDst, pidFile);
@@ -168,7 +181,7 @@ public class DefaultContainerExecutor extends ContainerExecutor {
 
       // Setup command to run
       String[] command = Shell.getRunCommand(
-        sb.getWrapperScriptPath().toUri().getPath().toString(), containerIdStr);
+        sb.getWrapperScriptPath().toString(), containerIdStr);
 
       LOG.info("launchContainer: " + Arrays.toString(command));
       shExec = new ShellCommandExecutor(
@@ -277,7 +290,7 @@ public class DefaultContainerExecutor extends ContainerExecutor {
         ".tmp");
       pout.println("@move /Y " + normalizedPidFile + ".tmp " +
         normalizedPidFile);
-      pout.println("@call " + launchDst.toUri().getPath().toString());
+      pout.println("@call " + launchDst.toString());
     }
   }
 

+ 12 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-nodemanager/src/main/java/org/apache/hadoop/yarn/server/nodemanager/containermanager/launcher/ContainerLaunch.java

@@ -38,6 +38,7 @@ import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileContext;
+import org.apache.hadoop.fs.FileUtil;
 import org.apache.hadoop.fs.LocalDirAllocator;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.io.IOUtils;
@@ -132,7 +133,7 @@ public class ContainerLaunch implements Callable<Integer> {
       for (String str : command) {
         // TODO: Should we instead work via symlinks without this grammar?
         newCmds.add(str.replace(ApplicationConstants.LOG_DIR_EXPANSION_VAR,
-            containerLogDir.toUri().getPath()));
+            containerLogDir.toString()));
       }
       launchContext.setCommands(newCmds);
 
@@ -143,7 +144,7 @@ public class ContainerLaunch implements Callable<Integer> {
         entry.setValue(
             value.replace(
                 ApplicationConstants.LOG_DIR_EXPANSION_VAR,
-                containerLogDir.toUri().getPath())
+                containerLogDir.toString())
             );
       }
       // /////////////////////////// End of variable expansion
@@ -531,7 +532,7 @@ public class ContainerLaunch implements Callable<Integer> {
   }
   
   public void sanitizeEnv(Map<String, String> environment, 
-      Path pwd, List<Path> appDirs) {
+      Path pwd, List<Path> appDirs) throws IOException {
     /**
      * Non-modifiable environment variables
      */
@@ -565,6 +566,14 @@ public class ContainerLaunch implements Callable<Integer> {
       environment.put("JVM_PID", "$$");
     }
 
+    // TODO: Remove Windows check and use this approach on all platforms after
+    // additional testing.  See YARN-358.
+    if (Shell.WINDOWS) {
+      String inputClassPath = environment.get(Environment.CLASSPATH.name());
+      environment.put(Environment.CLASSPATH.name(),
+          FileUtil.createJarWithClassPath(inputClassPath, pwd));
+    }
+
     /**
      * Modifiable environment variables
      */

+ 55 - 4
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-tests/src/test/java/org/apache/hadoop/yarn/server/MiniYARNCluster.java

@@ -29,6 +29,8 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileContext;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
+import org.apache.hadoop.util.Shell;
+import org.apache.hadoop.util.Shell.ShellCommandExecutor;
 import org.apache.hadoop.yarn.YarnException;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.event.Dispatcher;
@@ -83,15 +85,51 @@ public class MiniYARNCluster extends CompositeService {
     super(testName.replace("$", ""));
     this.numLocalDirs = numLocalDirs;
     this.numLogDirs = numLogDirs;
-    this.testWorkDir = new File("target",
-        testName.replace("$", ""));
+    String testSubDir = testName.replace("$", "");
+    File targetWorkDir = new File("target", testSubDir);
     try {
       FileContext.getLocalFSFileContext().delete(
-          new Path(testWorkDir.getAbsolutePath()), true);
+          new Path(targetWorkDir.getAbsolutePath()), true);
     } catch (Exception e) {
       LOG.warn("COULD NOT CLEANUP", e);
       throw new YarnException("could not cleanup test dir", e);
     } 
+
+    if (Shell.WINDOWS) {
+      // The test working directory can exceed the maximum path length supported
+      // by some Windows APIs and cmd.exe (260 characters).  To work around this,
+      // create a symlink in temporary storage with a much shorter path,
+      // targeting the full path to the test working directory.  Then, use the
+      // symlink as the test working directory.
+      String targetPath = targetWorkDir.getAbsolutePath();
+      File link = new File(System.getProperty("java.io.tmpdir"),
+        String.valueOf(System.currentTimeMillis()));
+      String linkPath = link.getAbsolutePath();
+
+      try {
+        FileContext.getLocalFSFileContext().delete(new Path(linkPath), true);
+      } catch (IOException e) {
+        throw new YarnException("could not cleanup symlink: " + linkPath, e);
+      }
+
+      // Guarantee target exists before creating symlink.
+      targetWorkDir.mkdirs();
+
+      ShellCommandExecutor shexec = new ShellCommandExecutor(
+        Shell.getSymlinkCommand(targetPath, linkPath));
+      try {
+        shexec.execute();
+      } catch (IOException e) {
+        throw new YarnException(String.format(
+          "failed to create symlink from %s to %s, shell output: %s", linkPath,
+          targetPath, shexec.getOutput()), e);
+      }
+
+      this.testWorkDir = link;
+    } else {
+      this.testWorkDir = targetWorkDir;
+    }
+
     resourceManagerWrapper = new ResourceManagerWrapper();
     addService(resourceManagerWrapper);
     nodeManagers = new CustomNodeManager[noOfNodeManagers];
@@ -192,6 +230,19 @@ public class MiniYARNCluster extends CompositeService {
         resourceManager.stop();
       }
       super.stop();
+
+      if (Shell.WINDOWS) {
+        // On Windows, clean up the short temporary symlink that was created to
+        // work around path length limitation.
+        String testWorkDirPath = testWorkDir.getAbsolutePath();
+        try {
+          FileContext.getLocalFSFileContext().delete(new Path(testWorkDirPath),
+            true);
+        } catch (IOException e) {
+          LOG.warn("could not cleanup symlink: " +
+            testWorkDir.getAbsolutePath());
+        }
+      }
     }
   }
 
@@ -220,7 +271,7 @@ public class MiniYARNCluster extends CompositeService {
       for (int i = 0; i < numDirs; i++) {
         dirs[i]= new File(testWorkDir, MiniYARNCluster.this.getName()
             + "-" + dirType + "Dir-nm-" + index + "_" + i);
-        dirs[i].mkdir();
+        dirs[i].mkdirs();
         LOG.info("Created " + dirType + "Dir in " + dirs[i].getAbsolutePath());
         String delimiter = (i > 0) ? "," : "";
         dirsString = dirsString.concat(delimiter + dirs[i].getAbsolutePath());