Переглянути джерело

HADOOP-11321. copyToLocal cannot save a file to an SMB share unless the user has Full Control permissions. Contributed by Chris Nauroth.

cnauroth 10 роки тому
батько
коміт
e996a1bfd4

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

@@ -593,6 +593,9 @@ Release 2.7.0 - UNRELEASED
     HADOOP-11412 POMs mention "The Apache Software License" rather than
     "Apache License". (Herve Boutemy via stevel)
 
+    HADOOP-11321. copyToLocal cannot save a file to an SMB share unless the user
+    has Full Control permissions. (cnauroth)
+
 Release 2.6.0 - 2014-11-18
 
   INCOMPATIBLE CHANGES

+ 9 - 10
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/ChecksumFileSystem.java

@@ -379,17 +379,19 @@ public abstract class ChecksumFileSystem extends FilterFileSystem {
                           int bufferSize,
                           short replication,
                           long blockSize,
-                          Progressable progress)
+                          Progressable progress,
+                          FsPermission permission)
       throws IOException {
       super(DataChecksum.newDataChecksum(DataChecksum.Type.CRC32,
           fs.getBytesPerSum()));
       int bytesPerSum = fs.getBytesPerSum();
-      this.datas = fs.getRawFileSystem().create(file, overwrite, bufferSize, 
-                                         replication, blockSize, progress);
+      this.datas = fs.getRawFileSystem().create(file, permission, overwrite,
+                                         bufferSize, replication, blockSize,
+                                         progress);
       int sumBufferSize = fs.getSumBufferSize(bytesPerSum, bufferSize);
-      this.sums = fs.getRawFileSystem().create(fs.getChecksumFile(file), true, 
-                                               sumBufferSize, replication,
-                                               blockSize);
+      this.sums = fs.getRawFileSystem().create(fs.getChecksumFile(file),
+                                               permission, true, sumBufferSize,
+                                               replication, blockSize, null);
       sums.write(CHECKSUM_VERSION, 0, CHECKSUM_VERSION.length);
       sums.writeInt(bytesPerSum);
     }
@@ -448,7 +450,7 @@ public abstract class ChecksumFileSystem extends FilterFileSystem {
     if (writeChecksum) {
       out = new FSDataOutputStream(
           new ChecksumFSOutputSummer(this, f, overwrite, bufferSize, replication,
-              blockSize, progress), null);
+              blockSize, progress, permission), null);
     } else {
       out = fs.create(f, permission, overwrite, bufferSize, replication,
           blockSize, progress);
@@ -458,9 +460,6 @@ public abstract class ChecksumFileSystem extends FilterFileSystem {
         fs.delete(checkFile, true);
       }
     }
-    if (permission != null) {
-      setPermission(f, permission);
-    }
     return out;
   }
 

+ 82 - 52
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/RawLocalFileSystem.java

@@ -43,6 +43,7 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.io.IOUtils;
 import org.apache.hadoop.io.nativeio.NativeIO;
+import org.apache.hadoop.io.nativeio.NativeIOException;
 import org.apache.hadoop.util.Progressable;
 import org.apache.hadoop.util.Shell;
 import org.apache.hadoop.util.StringUtils;
@@ -208,8 +209,28 @@ public class RawLocalFileSystem extends FileSystem {
   class LocalFSFileOutputStream extends OutputStream {
     private FileOutputStream fos;
     
-    private LocalFSFileOutputStream(Path f, boolean append) throws IOException {
-      this.fos = new FileOutputStream(pathToFile(f), append);
+    private LocalFSFileOutputStream(Path f, boolean append,
+        FsPermission permission) throws IOException {
+      File file = pathToFile(f);
+      if (permission == null) {
+        this.fos = new FileOutputStream(file, append);
+      } else {
+        if (Shell.WINDOWS && NativeIO.isAvailable()) {
+          this.fos = NativeIO.Windows.createFileOutputStreamWithMode(file,
+              append, permission.toShort());
+        } else {
+          this.fos = new FileOutputStream(file, append);
+          boolean success = false;
+          try {
+            setPermission(f, permission);
+            success = true;
+          } finally {
+            if (!success) {
+              IOUtils.cleanup(LOG, this.fos);
+            }
+          }
+        }
+      }
     }
     
     /*
@@ -248,19 +269,20 @@ public class RawLocalFileSystem extends FileSystem {
       throw new IOException("Cannot append to a diretory (=" + f + " )");
     }
     return new FSDataOutputStream(new BufferedOutputStream(
-        new LocalFSFileOutputStream(f, true), bufferSize), statistics);
+        createOutputStreamWithMode(f, true, null), bufferSize), statistics);
   }
 
   @Override
   public FSDataOutputStream create(Path f, boolean overwrite, int bufferSize,
     short replication, long blockSize, Progressable progress)
     throws IOException {
-    return create(f, overwrite, true, bufferSize, replication, blockSize, progress);
+    return create(f, overwrite, true, bufferSize, replication, blockSize,
+        progress, null);
   }
 
   private FSDataOutputStream create(Path f, boolean overwrite,
       boolean createParent, int bufferSize, short replication, long blockSize,
-      Progressable progress) throws IOException {
+      Progressable progress, FsPermission permission) throws IOException {
     if (exists(f) && !overwrite) {
       throw new FileAlreadyExistsException("File already exists: " + f);
     }
@@ -269,12 +291,18 @@ public class RawLocalFileSystem extends FileSystem {
       throw new IOException("Mkdirs failed to create " + parent.toString());
     }
     return new FSDataOutputStream(new BufferedOutputStream(
-        createOutputStream(f, false), bufferSize), statistics);
+        createOutputStreamWithMode(f, false, permission), bufferSize),
+        statistics);
   }
   
   protected OutputStream createOutputStream(Path f, boolean append) 
       throws IOException {
-    return new LocalFSFileOutputStream(f, append); 
+    return createOutputStreamWithMode(f, append, null);
+  }
+
+  protected OutputStream createOutputStreamWithMode(Path f, boolean append,
+      FsPermission permission) throws IOException {
+    return new LocalFSFileOutputStream(f, append, permission);
   }
   
   @Override
@@ -286,7 +314,8 @@ public class RawLocalFileSystem extends FileSystem {
       throw new FileAlreadyExistsException("File already exists: " + f);
     }
     return new FSDataOutputStream(new BufferedOutputStream(
-        new LocalFSFileOutputStream(f, false), bufferSize), statistics);
+        createOutputStreamWithMode(f, false, permission), bufferSize),
+            statistics);
   }
 
   @Override
@@ -294,18 +323,9 @@ public class RawLocalFileSystem extends FileSystem {
     boolean overwrite, int bufferSize, short replication, long blockSize,
     Progressable progress) throws IOException {
 
-    FSDataOutputStream out = create(f,
-        overwrite, bufferSize, replication, blockSize, progress);
-    boolean success = false;
-    try {
-      setPermission(f, permission);
-      success = true;
-      return out;
-    } finally {
-      if (!success) {
-        IOUtils.cleanup(LOG, out);
-      }
-    }
+    FSDataOutputStream out = create(f, overwrite, true, bufferSize, replication,
+        blockSize, progress, permission);
+    return out;
   }
 
   @Override
@@ -313,18 +333,9 @@ public class RawLocalFileSystem extends FileSystem {
       boolean overwrite,
       int bufferSize, short replication, long blockSize,
       Progressable progress) throws IOException {
-    FSDataOutputStream out = create(f,
-        overwrite, false, bufferSize, replication, blockSize, progress);
-    boolean success = false;
-    try {
-      setPermission(f, permission);
-      success = true;
-      return out;
-    } finally {
-      if (!success) {
-        IOUtils.cleanup(LOG, out);
-      } 
-    }
+    FSDataOutputStream out = create(f, overwrite, false, bufferSize, replication,
+        blockSize, progress, permission);
+    return out;
   }
 
   @Override
@@ -430,7 +441,34 @@ public class RawLocalFileSystem extends FileSystem {
   }
   
   protected boolean mkOneDir(File p2f) throws IOException {
-    return p2f.mkdir();
+    return mkOneDirWithMode(new Path(p2f.getAbsolutePath()), p2f, null);
+  }
+
+  protected boolean mkOneDirWithMode(Path p, File p2f, FsPermission permission)
+      throws IOException {
+    if (permission == null) {
+      return p2f.mkdir();
+    } else {
+      if (Shell.WINDOWS && NativeIO.isAvailable()) {
+        try {
+          NativeIO.Windows.createDirectoryWithMode(p2f, permission.toShort());
+          return true;
+        } catch (IOException e) {
+          if (LOG.isDebugEnabled()) {
+            LOG.debug(String.format(
+                "NativeIO.createDirectoryWithMode error, path = %s, mode = %o",
+                p2f, permission.toShort()), e);
+          }
+          return false;
+        }
+      } else {
+        boolean b = p2f.mkdir();
+        if (b) {
+          setPermission(p, permission);
+        }
+        return b;
+      }
+    }
   }
 
   /**
@@ -439,6 +477,16 @@ public class RawLocalFileSystem extends FileSystem {
    */
   @Override
   public boolean mkdirs(Path f) throws IOException {
+    return mkdirsWithOptionalPermission(f, null);
+  }
+
+  @Override
+  public boolean mkdirs(Path f, FsPermission permission) throws IOException {
+    return mkdirsWithOptionalPermission(f, permission);
+  }
+
+  private boolean mkdirsWithOptionalPermission(Path f, FsPermission permission)
+      throws IOException {
     if(f == null) {
       throw new IllegalArgumentException("mkdirs path arg is null");
     }
@@ -457,25 +505,7 @@ public class RawLocalFileSystem extends FileSystem {
               " and is not a directory: " + p2f.getCanonicalPath());
     }
     return (parent == null || parent2f.exists() || mkdirs(parent)) &&
-      (mkOneDir(p2f) || p2f.isDirectory());
-  }
-
-  @Override
-  public boolean mkdirs(Path f, FsPermission permission) throws IOException {
-    boolean b = mkdirs(f);
-    if(b) {
-      setPermission(f, permission);
-    }
-    return b;
-  }
-  
-
-  @Override
-  protected boolean primitiveMkdir(Path f, FsPermission absolutePermission)
-    throws IOException {
-    boolean b = mkdirs(f);
-    setPermission(f, absolutePermission);
-    return b;
+      (mkOneDirWithMode(f, p2f, permission) || p2f.isDirectory());
   }
   
   

+ 52 - 0
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/nativeio/NativeIO.java

@@ -508,11 +508,63 @@ public class NativeIO {
     
     public static final long FILE_ATTRIBUTE_NORMAL = 0x00000080L;
 
+    /**
+     * Create a directory with permissions set to the specified mode.  By setting
+     * permissions at creation time, we avoid issues related to the user lacking
+     * WRITE_DAC rights on subsequent chmod calls.  One example where this can
+     * occur is writing to an SMB share where the user does not have Full Control
+     * rights, and therefore WRITE_DAC is denied.
+     *
+     * @param path directory to create
+     * @param mode permissions of new directory
+     * @throws IOException if there is an I/O error
+     */
+    public static void createDirectoryWithMode(File path, int mode)
+        throws IOException {
+      createDirectoryWithMode0(path.getAbsolutePath(), mode);
+    }
+
+    /** Wrapper around CreateDirectory() on Windows */
+    private static native void createDirectoryWithMode0(String path, int mode)
+        throws NativeIOException;
+
     /** Wrapper around CreateFile() on Windows */
     public static native FileDescriptor createFile(String path,
         long desiredAccess, long shareMode, long creationDisposition)
         throws IOException;
 
+    /**
+     * Create a file for write with permissions set to the specified mode.  By
+     * setting permissions at creation time, we avoid issues related to the user
+     * lacking WRITE_DAC rights on subsequent chmod calls.  One example where
+     * this can occur is writing to an SMB share where the user does not have
+     * Full Control rights, and therefore WRITE_DAC is denied.
+     *
+     * This method mimics the semantics implemented by the JDK in
+     * {@link java.io.FileOutputStream}.  The file is opened for truncate or
+     * append, the sharing mode allows other readers and writers, and paths
+     * longer than MAX_PATH are supported.  (See io_util_md.c in the JDK.)
+     *
+     * @param path file to create
+     * @param append if true, then open file for append
+     * @param mode permissions of new directory
+     * @return FileOutputStream of opened file
+     * @throws IOException if there is an I/O error
+     */
+    public static FileOutputStream createFileOutputStreamWithMode(File path,
+        boolean append, int mode) throws IOException {
+      long desiredAccess = GENERIC_WRITE;
+      long shareMode = FILE_SHARE_READ | FILE_SHARE_WRITE;
+      long creationDisposition = append ? OPEN_ALWAYS : CREATE_ALWAYS;
+      return new FileOutputStream(createFileWithMode0(path.getAbsolutePath(),
+          desiredAccess, shareMode, creationDisposition, mode));
+    }
+
+    /** Wrapper around CreateFile() with security descriptor on Windows */
+    private static native FileDescriptor createFileWithMode0(String path,
+        long desiredAccess, long shareMode, long creationDisposition, int mode)
+        throws NativeIOException;
+
     /** Wrapper around SetFilePointer() on Windows */
     public static native long setFilePointer(FileDescriptor fd,
         long distanceToMove, long moveMethod) throws IOException;

+ 80 - 0
hadoop-common-project/hadoop-common/src/main/native/src/org/apache/hadoop/io/nativeio/NativeIO.c

@@ -514,6 +514,86 @@ cleanup:
 #endif
 }
 
+/*
+ * Class:     org_apache_hadoop_io_nativeio_NativeIO_Windows
+ * Method:    createDirectoryWithMode0
+ * Signature: (Ljava/lang/String;I)V
+ *
+ * The "00024" in the function name is an artifact of how JNI encodes
+ * special characters. U+0024 is '$'.
+ */
+JNIEXPORT void JNICALL
+  Java_org_apache_hadoop_io_nativeio_NativeIO_00024Windows_createDirectoryWithMode0
+  (JNIEnv *env, jclass clazz, jstring j_path, jint mode)
+{
+#ifdef WINDOWS
+  DWORD dwRtnCode = ERROR_SUCCESS;
+
+  LPCWSTR path = (LPCWSTR) (*env)->GetStringChars(env, j_path, NULL);
+  if (!path) {
+    goto done;
+  }
+
+  dwRtnCode = CreateDirectoryWithMode(path, mode);
+
+done:
+  if (path) {
+    (*env)->ReleaseStringChars(env, j_path, (const jchar*) path);
+  }
+  if (dwRtnCode != ERROR_SUCCESS) {
+    throw_ioe(env, dwRtnCode);
+  }
+#else
+  THROW(env, "java/io/IOException",
+    "The function Windows.createDirectoryWithMode0() is not supported on this platform");
+#endif
+}
+
+/*
+ * Class:     org_apache_hadoop_io_nativeio_NativeIO_Windows
+ * Method:    createFileWithMode0
+ * Signature: (Ljava/lang/String;JJJI)Ljava/io/FileDescriptor;
+ *
+ * The "00024" in the function name is an artifact of how JNI encodes
+ * special characters. U+0024 is '$'.
+ */
+JNIEXPORT jobject JNICALL
+  Java_org_apache_hadoop_io_nativeio_NativeIO_00024Windows_createFileWithMode0
+  (JNIEnv *env, jclass clazz, jstring j_path,
+  jlong desiredAccess, jlong shareMode, jlong creationDisposition, jint mode)
+{
+#ifdef WINDOWS
+  DWORD dwRtnCode = ERROR_SUCCESS;
+  HANDLE hFile = INVALID_HANDLE_VALUE;
+  jobject fd = NULL;
+
+  LPCWSTR path = (LPCWSTR) (*env)->GetStringChars(env, j_path, NULL);
+  if (!path) {
+    goto done;
+  }
+
+  dwRtnCode = CreateFileWithMode(path, desiredAccess, shareMode,
+      creationDisposition, mode, &hFile);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  fd = fd_create(env, (long) hFile);
+
+done:
+  if (path) {
+    (*env)->ReleaseStringChars(env, j_path, (const jchar*) path);
+  }
+  if (dwRtnCode != ERROR_SUCCESS) {
+    throw_ioe(env, dwRtnCode);
+  }
+  return fd;
+#else
+  THROW(env, "java/io/IOException",
+    "The function Windows.createFileWithMode0() is not supported on this platform");
+#endif
+}
+
 /*
  * Class:     org_apache_hadoop_io_nativeio_NativeIO_Windows
  * Method:    createFile

+ 6 - 0
hadoop-common-project/hadoop-common/src/main/winutils/include/winutils.h

@@ -165,6 +165,12 @@ DWORD JunctionPointCheck(__in LPCWSTR pathName, __out LPBOOL result);
 
 DWORD ChangeFileModeByMask(__in LPCWSTR path, INT mode);
 
+DWORD CreateDirectoryWithMode(__in LPCWSTR path, __in INT mode);
+
+DWORD CreateFileWithMode(__in LPCWSTR lpPath, __in DWORD dwDesiredAccess,
+    __in DWORD dwShareMode, __in DWORD dwCreationDisposition, __in INT mode,
+    __out_opt PHANDLE pHFile);
+
 DWORD GetLocalGroupsForUser(__in LPCWSTR user,
   __out LPLOCALGROUP_USERS_INFO_0 *groups, __out LPDWORD entries);
 

+ 264 - 3
hadoop-common-project/hadoop-common/src/main/winutils/libwinutils.c

@@ -138,15 +138,17 @@ DWORD GetFileInformationByName(
 // Function: IsLongWindowsPath
 //
 // Description:
-//  Checks if the path is longer than MAX_PATH in which case it needs to be
-//  prepended with \\?\ for Windows OS to understand it.
+//  Checks if the path is longer than (MAX_PATH - 13) in which case it needs to
+//  be prepended with \\?\ for Windows OS to understand it.  The -13 is to
+//  account for an additional constraint for directories that it must be possible
+//  to append an additional path separator followed by an 8.3 file name.
 //
 // Returns:
 //  TRUE long path
 //  FALSE otherwise
 static BOOL IsLongWindowsPath(__in PCWSTR path)
 {
-  return (wcslen(path) + 1) > MAX_PATH;
+  return (wcslen(path) + 1) > (MAX_PATH - 13);
 }
 
 //----------------------------------------------------------------------------
@@ -1451,6 +1453,265 @@ ChangeFileModeByMaskEnd:
   return ret;
 }
 
+//----------------------------------------------------------------------------
+// Function: GetTokenInformationByClass
+//
+// Description:
+//  Gets a class of information from a token.  On success, this function has
+//  dynamically allocated memory and set the ppTokenInformation parameter to
+//  point to it.  The caller owns this memory and is reponsible for releasing it
+//  by calling LocalFree.
+//
+// Returns:
+//  ERROR_SUCCESS: on success
+//  Error code: otherwise
+//
+static DWORD GetTokenInformationByClass(__in HANDLE hToken,
+    __in TOKEN_INFORMATION_CLASS class, __out_opt LPVOID *ppTokenInformation) {
+  DWORD dwRtnCode = ERROR_SUCCESS;
+  LPVOID pTokenInformation = NULL;
+  DWORD dwSize = 0;
+
+  // Call GetTokenInformation first time to get the required buffer size.
+  if (!GetTokenInformation(hToken, class, NULL, 0, &dwSize)) {
+    dwRtnCode = GetLastError();
+    if (dwRtnCode != ERROR_INSUFFICIENT_BUFFER) {
+      return dwRtnCode;
+    }
+  }
+
+  // Allocate memory.
+  pTokenInformation = LocalAlloc(LPTR, dwSize);
+  if (!pTokenInformation) {
+    return GetLastError();
+  }
+
+  // Call GetTokenInformation second time to fill our buffer with data.
+  if (!GetTokenInformation(hToken, class, pTokenInformation, dwSize, &dwSize)) {
+    LocalFree(pTokenInformation);
+    return GetLastError();
+  }
+
+  *ppTokenInformation = pTokenInformation;
+  return ERROR_SUCCESS;
+}
+
+//----------------------------------------------------------------------------
+// Function: GetWindowsDACLsForCreate
+//
+// Description:
+//  Get the Windows discretionary access control list equivalent to the given
+//  mode, suitable for creating a new file or directory.  Ownership is assumed
+//  to be the current process owner and primary group.  On success, this function
+//  has dynamically allocated memory and set the ppDACL parameter to point to it.
+//  The caller owns this memory and is reponsible for releasing it by calling
+//  LocalFree.
+//
+// Returns:
+//  ERROR_SUCCESS: on success
+//  Error code: otherwise
+//
+static DWORD GetWindowsDACLsForCreate(__in INT mode, __out PACL *ppDACL) {
+  DWORD dwRtnCode = ERROR_SUCCESS;
+  HANDLE hToken = NULL;
+  DWORD dwSize = 0;
+  PTOKEN_OWNER pTokenOwner = NULL;
+  PTOKEN_PRIMARY_GROUP pTokenPrimaryGroup = NULL;
+  PSID pOwnerSid = NULL, pGroupSid = NULL;
+  PACL pDACL = NULL;
+
+  if (!OpenProcessToken(GetCurrentProcess(), TOKEN_QUERY, &hToken)) {
+    dwRtnCode = GetLastError();
+    goto done;
+  }
+
+  dwRtnCode = GetTokenInformationByClass(hToken, TokenOwner, &pTokenOwner);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+  pOwnerSid = pTokenOwner->Owner;
+
+  dwRtnCode = GetTokenInformationByClass(hToken, TokenPrimaryGroup,
+      &pTokenPrimaryGroup);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+  pGroupSid = pTokenPrimaryGroup->PrimaryGroup;
+
+  dwRtnCode = GetWindowsDACLs(mode, pOwnerSid, pGroupSid, &pDACL);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  *ppDACL = pDACL;
+
+done:
+  if (hToken) {
+    CloseHandle(hToken);
+  }
+  LocalFree(pTokenOwner);
+  LocalFree(pTokenPrimaryGroup);
+  return dwRtnCode;
+}
+
+//----------------------------------------------------------------------------
+// Function: CreateSecurityDescriptorForCreate
+//
+// Description:
+//  Creates a security descriptor with the given DACL, suitable for creating a
+//  new file or directory.  On success, this function has dynamically allocated
+//  memory and set the ppSD parameter to point to it.  The caller owns this
+//  memory and is reponsible for releasing it by calling LocalFree.
+//
+// Returns:
+//  ERROR_SUCCESS: on success
+//  Error code: otherwise
+//
+static DWORD CreateSecurityDescriptorForCreate(__in PACL pDACL,
+    __out PSECURITY_DESCRIPTOR *ppSD) {
+  DWORD dwRtnCode = ERROR_SUCCESS;
+  PSECURITY_DESCRIPTOR pSD = NULL;
+
+  pSD = LocalAlloc(LPTR, SECURITY_DESCRIPTOR_MIN_LENGTH);
+  if (!pSD) {
+    dwRtnCode = GetLastError();
+    goto done;
+  }
+
+  if (!InitializeSecurityDescriptor(pSD, SECURITY_DESCRIPTOR_REVISION)) {
+    dwRtnCode = GetLastError();
+    goto done;
+  }
+
+  if (!SetSecurityDescriptorDacl(pSD, TRUE, pDACL, FALSE)) {
+    dwRtnCode = GetLastError();
+    goto done;
+  }
+
+  *ppSD = pSD;
+
+done:
+  if (dwRtnCode != ERROR_SUCCESS) {
+    LocalFree(pSD);
+  }
+  return dwRtnCode;
+}
+
+//----------------------------------------------------------------------------
+// Function: CreateDirectoryWithMode
+//
+// Description:
+//  Create a directory with initial security descriptor containing a
+//  discretionary access control list equivalent to the given mode.
+//
+// Returns:
+//  ERROR_SUCCESS: on success
+//  Error code: otherwise
+//
+// Notes:
+//  This function is long path safe, i.e. the path will be converted to long
+//  path format if not already converted. So the caller does not need to do
+//  the conversion before calling the method.
+//
+DWORD CreateDirectoryWithMode(__in LPCWSTR lpPath, __in INT mode) {
+  DWORD dwRtnCode = ERROR_SUCCESS;
+  LPWSTR lpLongPath = NULL;
+  PACL pDACL = NULL;
+  PSECURITY_DESCRIPTOR pSD = NULL;
+  SECURITY_ATTRIBUTES sa;
+
+  dwRtnCode = ConvertToLongPath(lpPath, &lpLongPath);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  dwRtnCode = GetWindowsDACLsForCreate(mode, &pDACL);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  dwRtnCode = CreateSecurityDescriptorForCreate(pDACL, &pSD);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  sa.nLength = sizeof(SECURITY_ATTRIBUTES);
+  sa.lpSecurityDescriptor = pSD;
+  sa.bInheritHandle = FALSE;
+
+  if (!CreateDirectoryW(lpLongPath, &sa)) {
+    dwRtnCode = GetLastError();
+  }
+
+done:
+  LocalFree(lpLongPath);
+  LocalFree(pDACL);
+  LocalFree(pSD);
+  return dwRtnCode;
+}
+
+//----------------------------------------------------------------------------
+// Function: CreateFileWithMode
+//
+// Description:
+//  Create a file with initial security descriptor containing a discretionary
+//  access control list equivalent to the given mode.
+//
+// Returns:
+//  ERROR_SUCCESS: on success
+//  Error code: otherwise
+//
+// Notes:
+//  This function is long path safe, i.e. the path will be converted to long
+//  path format if not already converted. So the caller does not need to do
+//  the conversion before calling the method.
+//
+DWORD CreateFileWithMode(__in LPCWSTR lpPath, __in DWORD dwDesiredAccess,
+    __in DWORD dwShareMode, __in DWORD dwCreationDisposition, __in INT mode,
+    __out PHANDLE pHFile) {
+  DWORD dwRtnCode = ERROR_SUCCESS;
+  LPWSTR lpLongPath = NULL;
+  PACL pDACL = NULL;
+  PSECURITY_DESCRIPTOR pSD = NULL;
+  SECURITY_ATTRIBUTES sa;
+  DWORD dwFlagsAndAttributes = FILE_ATTRIBUTE_NORMAL;
+  HANDLE hFile = INVALID_HANDLE_VALUE;
+
+  dwRtnCode = ConvertToLongPath(lpPath, &lpLongPath);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  dwRtnCode = GetWindowsDACLsForCreate(mode, &pDACL);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  dwRtnCode = CreateSecurityDescriptorForCreate(pDACL, &pSD);
+  if (dwRtnCode != ERROR_SUCCESS) {
+    goto done;
+  }
+
+  sa.nLength = sizeof(SECURITY_ATTRIBUTES);
+  sa.lpSecurityDescriptor = pSD;
+  sa.bInheritHandle = FALSE;
+
+  hFile = CreateFileW(lpLongPath, dwDesiredAccess, dwShareMode, &sa,
+    dwCreationDisposition, dwFlagsAndAttributes, NULL);
+  if (hFile == INVALID_HANDLE_VALUE) {
+    dwRtnCode = GetLastError();
+    goto done;
+  }
+
+  *pHFile = hFile;
+
+done:
+  LocalFree(lpLongPath);
+  LocalFree(pDACL);
+  LocalFree(pSD);
+  return dwRtnCode;
+}
+
 //----------------------------------------------------------------------------
 // Function: GetAccntNameFromSid
 //

+ 22 - 8
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-nodemanager/src/main/java/org/apache/hadoop/yarn/server/nodemanager/WindowsSecureContainerExecutor.java

@@ -45,6 +45,7 @@ import org.apache.hadoop.fs.FsConstants;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.RawLocalFileSystem;
 import org.apache.hadoop.fs.permission.FsPermission;
+import org.apache.hadoop.io.IOUtils;
 import org.apache.hadoop.io.nativeio.NativeIO.Windows;
 import org.apache.hadoop.io.nativeio.NativeIOException;
 import org.apache.hadoop.util.NativeCodeLoader;
@@ -313,21 +314,23 @@ public class WindowsSecureContainerExecutor extends DefaultContainerExecutor {
     private static class ElevatedRawLocalFilesystem extends RawLocalFileSystem {
       
       @Override
-      protected boolean mkOneDir(File p2f) throws IOException {
-        Path path = new Path(p2f.getAbsolutePath());
+      protected boolean mkOneDirWithMode(Path path, File p2f,
+          FsPermission permission) throws IOException {
         if (LOG.isDebugEnabled()) {
-          LOG.debug(String.format("EFS:mkOneDir: %s", path));
+          LOG.debug(String.format("EFS:mkOneDirWithMode: %s %s", path,
+              permission));
         }
         boolean ret = false;
 
         // File.mkdir returns false, does not throw. Must mimic it.
         try {
           Native.Elevated.mkdir(path);
+          setPermission(path, permission);
           ret = true;
         }
         catch(Throwable e) {
           if (LOG.isDebugEnabled()) {
-            LOG.debug(String.format("EFS:mkOneDir: %s", 
+            LOG.debug(String.format("EFS:mkOneDirWithMode: %s",
                 org.apache.hadoop.util.StringUtils.stringifyException(e)));
           }
         }
@@ -354,12 +357,23 @@ public class WindowsSecureContainerExecutor extends DefaultContainerExecutor {
       }
       
       @Override
-      protected OutputStream createOutputStream(Path f, boolean append) 
-          throws IOException {
+      protected OutputStream createOutputStreamWithMode(Path f, boolean append,
+          FsPermission permission) throws IOException {
         if (LOG.isDebugEnabled()) {
-          LOG.debug(String.format("EFS:create: %s %b", f, append));
+          LOG.debug(String.format("EFS:createOutputStreamWithMode: %s %b %s", f,
+              append, permission));
+        }
+        boolean success = false;
+        OutputStream os = Native.Elevated.create(f, append);
+        try {
+          setPermission(f, permission);
+          success = true;
+          return os;
+        } finally {
+          if (!success) {
+            IOUtils.cleanup(LOG, os);
+          }
         }
-        return Native.Elevated.create(f, append); 
       }
       
       @Override

+ 6 - 6
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-nodemanager/src/test/java/org/apache/hadoop/yarn/server/nodemanager/TestLocalDirsHandlerService.java

@@ -31,9 +31,9 @@ import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.service.Service.STATE;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
-import org.junit.AfterClass;
+import org.junit.After;
 import org.junit.Assert;
-import org.junit.BeforeClass;
+import org.junit.Before;
 import org.junit.Test;
 
 public class TestLocalDirsHandlerService {
@@ -41,14 +41,14 @@ public class TestLocalDirsHandlerService {
       TestDirectoryCollection.class.getName()).getAbsoluteFile();
   private static final File testFile = new File(testDir, "testfile");
 
-  @BeforeClass
-  public static void setup() throws IOException {
+  @Before
+  public void setup() throws IOException {
     testDir.mkdirs();
     testFile.createNewFile();
   }
 
-  @AfterClass
-  public static void teardown() {
+  @After
+  public void teardown() {
     FileUtil.fullyDelete(testDir);
   }