You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hive.apache.org by br...@apache.org on 2014/12/12 22:22:54 UTC

svn commit: r1645078 - in /hive/branches/HIVE-8065: itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/ ql/src/java/org/apache/hadoop/hive/ql/exec/ ql/src/java/org/apache/hadoop/hive/ql/metadata/

Author: brock
Date: Fri Dec 12 21:22:53 2014
New Revision: 1645078

URL: http://svn.apache.org/r1645078
Log:
HIVE-8864 - Fix permission inheritance with HDFS encryption (Szehon via Brock)

Modified:
    hive/branches/HIVE-8065/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/FolderPermissionBase.java
    hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/exec/MoveTask.java
    hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java
    hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Table.java

Modified: hive/branches/HIVE-8065/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/FolderPermissionBase.java
URL: http://svn.apache.org/viewvc/hive/branches/HIVE-8065/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/FolderPermissionBase.java?rev=1645078&r1=1645077&r2=1645078&view=diff
==============================================================================
--- hive/branches/HIVE-8065/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/FolderPermissionBase.java (original)
+++ hive/branches/HIVE-8065/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/security/FolderPermissionBase.java Fri Dec 12 21:22:53 2014
@@ -28,6 +28,7 @@ import org.apache.hadoop.fs.FileStatus;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.PathFilter;
+import org.apache.hadoop.fs.permission.AclStatus;
 import org.apache.hadoop.hive.cli.CliSessionState;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
@@ -205,169 +206,218 @@ public abstract class FolderPermissionBa
 
 
   @Test
-  public void testStaticPartition() throws Exception {
-    String tableName = "staticpart";
-    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 string, part2 string)");
-    Assert.assertEquals(0,ret.getResponseCode());
+  public void testInsertNonPartTable() throws Exception {
+    //case 1 is non-partitioned table.
+    String tableName = "nonpart";
+
+    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string)");
+    Assert.assertEquals(0, ret.getResponseCode());
 
+    String tableLoc = warehouseDir + "/" + tableName;
     assertExistence(warehouseDir + "/" + tableName);
+
+    //case1A: insert into non-partitioned table.
     setPermission(warehouseDir + "/" + tableName);
+    ret = driver.run("insert into table " + tableName + " select key,value from mysrc");
+    Assert.assertEquals(0, ret.getResponseCode());
 
-    ret = driver.run("insert into table " + tableName + " partition(part1='1', part2='1') select key,value from mysrc where part1='1' and part2='1'");
-    Assert.assertEquals(0,ret.getResponseCode());
+    verifyPermission(warehouseDir + "/" + tableName);
+    Assert.assertTrue(listStatus(tableLoc).size() > 0);
+    for (String child : listStatus(tableLoc)) {
+      verifyPermission(child);
+    }
 
-    verifyPermission(warehouseDir + "/" + tableName + "/part1=1");
-    verifyPermission(warehouseDir + "/" + tableName + "/part1=1/part2=1");
+    //case1B: insert overwrite non-partitioned-table
+    setPermission(warehouseDir + "/" + tableName, 1);
+    ret = driver.run("insert overwrite table " + tableName + " select key,value from mysrc");
+    Assert.assertEquals(0, ret.getResponseCode());
 
-    Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=1/part2=1").size() > 0);
-    for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=1/part2=1")) {
-      verifyPermission(child);
+    verifyPermission(warehouseDir + "/" + tableName, 1);
+    Assert.assertTrue(listStatus(tableLoc).size() > 0);
+    for (String child : listStatus(tableLoc)) {
+      verifyPermission(child, 1);
     }
   }
 
   @Test
-  public void testAlterPartition() throws Exception {
-    String tableName = "alterpart";
-    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 int, part2 int, part3 int)");
-    Assert.assertEquals(0,ret.getResponseCode());
+  public void testInsertStaticSinglePartition() throws Exception {
+    String tableName = "singlestaticpart";
+    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 string)");
+    Assert.assertEquals(0, ret.getResponseCode());
 
     assertExistence(warehouseDir + "/" + tableName);
     setPermission(warehouseDir + "/" + tableName);
 
-    ret = driver.run("insert into table " + tableName + " partition(part1='1',part2='1',part3='1') select key,value from mysrc");
-    Assert.assertEquals(0,ret.getResponseCode());
+    //insert into test
+    ret = driver.run("insert into table " + tableName + " partition(part1='1') select key,value from mysrc where part1='1' and part2='1'");
+    Assert.assertEquals(0, ret.getResponseCode());
 
-    assertExistence(warehouseDir + "/" + tableName);
-    setPermission(warehouseDir + "/" + tableName, 1);
+    verifyPermission(warehouseDir + "/" + tableName);
+    verifyPermission(warehouseDir + "/" + tableName + "/part1=1");
 
-    //alter partition
-    ret = driver.run("alter table " + tableName + " partition (part1='1',part2='1',part3='1') rename to partition (part1='2',part2='2',part3='2')");
-    Assert.assertEquals(0,ret.getResponseCode());
+    Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=1").size() > 0);
+    for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=1")) {
+      verifyPermission(child);
+    }
 
-    verifyPermission(warehouseDir + "/" + tableName + "/part1=2", 1);
-    verifyPermission(warehouseDir + "/" + tableName + "/part1=2/part2=2", 1);
-    verifyPermission(warehouseDir + "/" + tableName + "/part1=2/part2=2/part3=2", 1);
+    //insert overwrite test
+    setPermission(warehouseDir + "/" + tableName, 1);
+    ret = driver.run("insert overwrite table " + tableName + " partition(part1='1') select key,value from mysrc where part1='1' and part2='1'");
+    Assert.assertEquals(0, ret.getResponseCode());
 
-    Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=2/part2=2/part3=2").size() > 0);
-    for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=2/part2=2/part3=2")) {
+    verifyPermission(warehouseDir + "/" + tableName, 1);
+    verifyPermission(warehouseDir + "/" + tableName + "/part1=1", 1);
+
+    Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=1").size() > 0);
+    for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=1")) {
       verifyPermission(child, 1);
     }
   }
 
-
   @Test
-  public void testDynamicPartitions() throws Exception {
-    String tableName = "dynamicpart";
-
+  public void testInsertStaticDualPartition() throws Exception {
+    String tableName = "dualstaticpart";
     CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 string, part2 string)");
-    Assert.assertEquals(0,ret.getResponseCode());
+    Assert.assertEquals(0, ret.getResponseCode());
 
     assertExistence(warehouseDir + "/" + tableName);
     setPermission(warehouseDir + "/" + tableName);
 
-    ret = driver.run("insert into table " + tableName + " partition (part1,part2) select key,value,part1,part2 from mysrc");
-    Assert.assertEquals(0,ret.getResponseCode());
+    //insert into test
+    ret = driver.run("insert into table " + tableName + " partition(part1='1', part2='1') select key,value from mysrc where part1='1' and part2='1'");
+    Assert.assertEquals(0, ret.getResponseCode());
 
+    verifyPermission(warehouseDir + "/" + tableName);
     verifyPermission(warehouseDir + "/" + tableName + "/part1=1");
     verifyPermission(warehouseDir + "/" + tableName + "/part1=1/part2=1");
 
-    verifyPermission(warehouseDir + "/" + tableName + "/part1=2");
-    verifyPermission(warehouseDir + "/" + tableName + "/part1=2/part2=2");
-
     Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=1/part2=1").size() > 0);
     for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=1/part2=1")) {
       verifyPermission(child);
     }
 
-    Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=2/part2=2").size() > 0);
-    for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=2/part2=2")) {
-      verifyPermission(child);
+    //insert overwrite test
+    setPermission(warehouseDir + "/" + tableName, 1);
+    ret = driver.run("insert overwrite table " + tableName + " partition(part1='1', part2='1') select key,value from mysrc where part1='1' and part2='1'");
+    Assert.assertEquals(0, ret.getResponseCode());
+
+    verifyPermission(warehouseDir + "/" + tableName, 1);
+    verifyPermission(warehouseDir + "/" + tableName + "/part1=1", 1);
+    verifyPermission(warehouseDir + "/" + tableName + "/part1=1/part2=1", 1);
+
+    Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=1/part2=1").size() > 0);
+    for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=1/part2=1")) {
+      verifyPermission(child, 1);
     }
   }
 
   @Test
-  public void testExternalTable() throws Exception {
-    String tableName = "externaltable";
+  public void testInsertDualDynamicPartitions() throws Exception {
+    String tableName = "dualdynamicpart";
 
-    String myLocation = warehouseDir + "/myfolder";
-    FileSystem fs = FileSystem.get(new URI(myLocation), conf);
-    fs.mkdirs(new Path(myLocation));
-    setPermission(myLocation);
+    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 string, part2 string)");
+    Assert.assertEquals(0, ret.getResponseCode());
+    assertExistence(warehouseDir + "/" + tableName);
 
-    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) LOCATION '" + myLocation + "'");
-    Assert.assertEquals(0,ret.getResponseCode());
+    //Insert into test, with permission set 0.
+    setPermission(warehouseDir + "/" + tableName, 0);
+    ret = driver.run("insert into table " + tableName + " partition (part1,part2) select key,value,part1,part2 from mysrc");
+    Assert.assertEquals(0, ret.getResponseCode());
 
-    ret = driver.run("insert into table " + tableName + " select key,value from mysrc");
-    Assert.assertEquals(0,ret.getResponseCode());
+    verifyDualPartitionTable(warehouseDir + "/" + tableName, 0);
 
-    Assert.assertTrue(listStatus(myLocation).size() > 0);
-    for (String child : listStatus(myLocation)) {
-      verifyPermission(child);
-    }
+    //Insert overwrite test, with permission set 1.
+    setPermission(warehouseDir + "/" + tableName, 1);
+    ret = driver.run("insert overwrite table " + tableName + " partition (part1,part2) select key,value,part1,part2 from mysrc");
+    Assert.assertEquals(0, ret.getResponseCode());
+
+    verifyDualPartitionTable(warehouseDir + "/" + tableName, 1);
   }
 
   @Test
-  public void testInsert() throws Exception {
-    //case 1 is non-partitioned table.
-    String tableName = "insert";
+  public void testInsertSingleDynamicPartition() throws Exception {
+    String tableName = "singledynamicpart";
 
-    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string)");
+    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 string)");
     Assert.assertEquals(0,ret.getResponseCode());
-
     String tableLoc = warehouseDir + "/" + tableName;
-    assertExistence(warehouseDir + "/" + tableName);
+    assertExistence(tableLoc);
 
-    //case1A: insert into non-partitioned table.
+    //Insert into test, with permission set 0.
+    setPermission(tableLoc, 0);
+    ret = driver.run("insert into table " + tableName + " partition (part1) select key,value,part1 from mysrc");
+    Assert.assertEquals(0,ret.getResponseCode());
+    verifySinglePartition(tableLoc, 0);
+
+    //Insert overwrite test, with permission set 1.
+    setPermission(tableLoc, 1);
+    ret = driver.run("insert overwrite table " + tableName + " partition (part1) select key,value,part1 from mysrc");
+    Assert.assertEquals(0,ret.getResponseCode());
+    verifySinglePartition(tableLoc, 1);
+
+    //delete and re-insert using insert overwrite.  There's different code paths insert vs insert overwrite for new tables.
+    ret = driver.run("DROP TABLE " + tableName);
+    Assert.assertEquals(0, ret.getResponseCode());
+    ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 string)");
+    Assert.assertEquals(0, ret.getResponseCode());
+
+    assertExistence(warehouseDir + "/" + tableName);
     setPermission(warehouseDir + "/" + tableName);
-    ret = driver.run("insert into table " + tableName + " select key,value from mysrc");
+
+    ret = driver.run("insert overwrite table " + tableName + " partition (part1) select key,value,part1 from mysrc");
+    Assert.assertEquals(0, ret.getResponseCode());
+
+    verifySinglePartition(tableLoc, 0);
+  }
+
+  @Test
+  public void testAlterPartition() throws Exception {
+    String tableName = "alterpart";
+    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 int, part2 int, part3 int)");
     Assert.assertEquals(0,ret.getResponseCode());
 
-    Assert.assertTrue(listStatus(tableLoc).size() > 0);
-    for (String child : listStatus(tableLoc)) {
-      verifyPermission(child);
-    }
+    assertExistence(warehouseDir + "/" + tableName);
+    setPermission(warehouseDir + "/" + tableName);
 
-    //case1B: insert overwrite non-partitioned-table
+    ret = driver.run("insert into table " + tableName + " partition(part1='1',part2='1',part3='1') select key,value from mysrc");
+    Assert.assertEquals(0,ret.getResponseCode());
+
+    assertExistence(warehouseDir + "/" + tableName);
     setPermission(warehouseDir + "/" + tableName, 1);
-    ret = driver.run("insert overwrite table " + tableName + " select key,value from mysrc");
+
+    //alter partition
+    ret = driver.run("alter table " + tableName + " partition (part1='1',part2='1',part3='1') rename to partition (part1='2',part2='2',part3='2')");
     Assert.assertEquals(0,ret.getResponseCode());
 
-    Assert.assertTrue(listStatus(tableLoc).size() > 0);
-    for (String child : listStatus(tableLoc)) {
+    verifyPermission(warehouseDir + "/" + tableName + "/part1=2", 1);
+    verifyPermission(warehouseDir + "/" + tableName + "/part1=2/part2=2", 1);
+    verifyPermission(warehouseDir + "/" + tableName + "/part1=2/part2=2/part3=2", 1);
+
+    Assert.assertTrue(listStatus(warehouseDir + "/" + tableName + "/part1=2/part2=2/part3=2").size() > 0);
+    for (String child : listStatus(warehouseDir + "/" + tableName + "/part1=2/part2=2/part3=2")) {
       verifyPermission(child, 1);
     }
+  }
 
-    //case 2 is partitioned table.
-    tableName = "insertpartition";
+  @Test
+  public void testExternalTable() throws Exception {
+    String tableName = "externaltable";
 
-    ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) partitioned by (part1 int, part2 int)");
-    Assert.assertEquals(0,ret.getResponseCode());
+    String myLocation = warehouseDir + "/myfolder";
+    FileSystem fs = FileSystem.get(new URI(myLocation), conf);
+    fs.mkdirs(new Path(myLocation));
+    setPermission(myLocation);
 
-    ret = driver.run("insert overwrite table " + tableName + " partition(part1='1',part2='1') select key,value from mysrc");
+    CommandProcessorResponse ret = driver.run("CREATE TABLE " + tableName + " (key string, value string) LOCATION '" + myLocation + "'");
     Assert.assertEquals(0,ret.getResponseCode());
 
-    String partLoc = warehouseDir + "/" + tableName + "/part1=1/part2=1";
-    assertExistence(partLoc);
-
-    //case 2A: insert into partitioned table.
-    setPermission(partLoc);
-    ret = driver.run("insert overwrite table " + tableName + " partition(part1='1',part2='1') select key,value from mysrc");
+    ret = driver.run("insert into table " + tableName + " select key,value from mysrc");
     Assert.assertEquals(0,ret.getResponseCode());
 
-    Assert.assertTrue(listStatus(partLoc).size() > 0);
-    for (String child : listStatus(partLoc)) {
+    Assert.assertTrue(listStatus(myLocation).size() > 0);
+    for (String child : listStatus(myLocation)) {
       verifyPermission(child);
     }
-
-    //case 2B: insert into non-partitioned table.
-    setPermission(partLoc, 1);
-    ret = driver.run("insert overwrite table " + tableName + " partition(part1='1',part2='1') select key,value from mysrc");
-    Assert.assertEquals(0,ret.getResponseCode());
-
-    Assert.assertTrue(listStatus(tableLoc).size() > 0);
-    for (String child : listStatus(partLoc)) {
-      verifyPermission(child, 1);
-    }
   }
 
   @Test
@@ -422,7 +472,7 @@ public abstract class FolderPermissionBa
     }
 
     //case 2B: insert data overwrite into non-partitioned table.
-    setPermission(partLoc, 1);
+    setPermission(tableLoc, 1);
     ret = driver.run("LOAD DATA LOCAL INPATH '" + dataFilePath + "' OVERWRITE INTO TABLE " + tableName + " PARTITION (part1='1',part2='1')");
     Assert.assertEquals(0,ret.getResponseCode());
 
@@ -487,7 +537,7 @@ public abstract class FolderPermissionBa
     }
 
     //case 2B: insert data overwrite into non-partitioned table.
-    setPermission(partLoc, 1);
+    setPermission(tableLoc, 1);
     fs.copyFromLocalFile(dataFilePath, new Path(location));
     ret = driver.run("LOAD DATA INPATH '" + location + "' OVERWRITE INTO TABLE " + tableName + " PARTITION (part1='1',part2='1')");
     Assert.assertEquals(0,ret.getResponseCode());
@@ -596,6 +646,40 @@ public abstract class FolderPermissionBa
     }
   }
 
+  private void verifySinglePartition(String tableLoc, int index) throws Exception {
+    verifyPermission(tableLoc + "/part1=1", index);
+    verifyPermission(tableLoc + "/part1=2", index);
+
+    Assert.assertTrue(listStatus(tableLoc + "/part1=1").size() > 0);
+    for (String child : listStatus(tableLoc + "/part1=1")) {
+      verifyPermission(child, index);
+    }
+
+    Assert.assertTrue(listStatus(tableLoc + "/part1=2").size() > 0);
+    for (String child : listStatus(tableLoc + "/part1=2")) {
+      verifyPermission(child, index);
+    }
+  }
+
+  private void verifyDualPartitionTable(String baseTablePath, int index) throws Exception {
+    verifyPermission(baseTablePath, index);
+    verifyPermission(baseTablePath + "/part1=1", index);
+    verifyPermission(baseTablePath + "/part1=1/part2=1", index);
+
+    verifyPermission(baseTablePath + "/part1=2", index);
+    verifyPermission(baseTablePath + "/part1=2/part2=2", index);
+
+    Assert.assertTrue(listStatus(baseTablePath + "/part1=1/part2=1").size() > 0);
+    for (String child : listStatus(baseTablePath + "/part1=1/part2=1")) {
+      verifyPermission(child, index);
+    }
+
+    Assert.assertTrue(listStatus(baseTablePath + "/part1=2/part2=2").size() > 0);
+    for (String child : listStatus(baseTablePath + "/part1=2/part2=2")) {
+      verifyPermission(child, index);
+    }
+  }
+
   private void assertExistence(String locn) throws Exception {
     Assert.assertTrue(fs.exists(new Path(locn)));
   }

Modified: hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/exec/MoveTask.java
URL: http://svn.apache.org/viewvc/hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/exec/MoveTask.java?rev=1645078&r1=1645077&r2=1645078&view=diff
==============================================================================
--- hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/exec/MoveTask.java (original)
+++ hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/exec/MoveTask.java Fri Dec 12 21:22:53 2014
@@ -59,6 +59,8 @@ import org.apache.hadoop.hive.ql.plan.Ma
 import org.apache.hadoop.hive.ql.plan.MoveWork;
 import org.apache.hadoop.hive.ql.plan.api.StageType;
 import org.apache.hadoop.hive.ql.session.SessionState;
+import org.apache.hadoop.hive.shims.HadoopShims;
+import org.apache.hadoop.hive.shims.ShimLoader;
 import org.apache.hadoop.util.StringUtils;
 
 import java.io.IOException;
@@ -159,8 +161,14 @@ public class MoveTask extends Task<MoveW
         actualPath = actualPath.getParent();
       }
       fs.mkdirs(mkDirPath);
+      HadoopShims shims = ShimLoader.getHadoopShims();
       if (HiveConf.getBoolVar(conf, HiveConf.ConfVars.HIVE_WAREHOUSE_SUBDIR_INHERIT_PERMS)) {
-        fs.setPermission(mkDirPath, fs.getFileStatus(actualPath).getPermission());
+        try {
+          HadoopShims.HdfsFileStatus status = shims.getFullFileStatus(conf, fs, actualPath);
+          shims.setFullFileStatus(conf, status, fs, actualPath);
+        } catch (Exception e) {
+          LOG.warn("Error setting permissions or group of " + actualPath, e);
+        }
       }
     }
     return deletePath;

Modified: hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java
URL: http://svn.apache.org/viewvc/hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java?rev=1645078&r1=1645077&r2=1645078&view=diff
==============================================================================
--- hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java (original)
+++ hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Hive.java Fri Dec 12 21:22:53 2014
@@ -36,6 +36,7 @@ import java.util.HashSet;
 import java.util.Iterator;
 import java.util.LinkedHashMap;
 import java.util.LinkedHashSet;
+import java.util.LinkedList;
 import java.util.List;
 import java.util.Map;
 import java.util.Map.Entry;
@@ -1344,7 +1345,7 @@ public class Hive {
       }
 
       if (replace) {
-        Hive.replaceFiles(loadPath, newPartPath, oldPartPath, getConf(),
+        Hive.replaceFiles(tbl.getPath(), loadPath, newPartPath, oldPartPath, getConf(),
             isSrcLocal);
       } else {
         FileSystem fs = tbl.getDataLocation().getFileSystem(conf);
@@ -2551,6 +2552,7 @@ private void constructOneLBLocationMap(F
    * srcf, destf, and tmppath should resident in the same DFS, but the oldPath can be in a
    * different DFS.
    *
+   * @param tablePath path of the table.  Used to identify permission inheritance.
    * @param srcf
    *          Source directory to be renamed to tmppath. It should be a
    *          leaf directory where the final data files reside. However it
@@ -2558,13 +2560,15 @@ private void constructOneLBLocationMap(F
    * @param destf
    *          The directory where the final data needs to go
    * @param oldPath
-   *          The directory where the old data location, need to be cleaned up.
+   *          The directory where the old data location, need to be cleaned up.  Most of time, will be the same
+   *          as destf, unless its across FileSystem boundaries.
    * @param isSrcLocal
    *          If the source directory is LOCAL
    */
-  static protected void replaceFiles(Path srcf, Path destf, Path oldPath,
-      HiveConf conf, boolean isSrcLocal) throws HiveException {
+  protected static void replaceFiles(Path tablePath, Path srcf, Path destf, Path oldPath, HiveConf conf,
+          boolean isSrcLocal) throws HiveException {
     try {
+
       FileSystem destFs = destf.getFileSystem(conf);
       boolean inheritPerms = HiveConf.getBoolVar(conf,
           HiveConf.ConfVars.HIVE_WAREHOUSE_SUBDIR_INHERIT_PERMS);
@@ -2585,6 +2589,7 @@ private void constructOneLBLocationMap(F
       List<List<Path[]>> result = checkPaths(conf, destFs, srcs, srcFs, destf,
           true);
 
+      HadoopShims shims = ShimLoader.getHadoopShims();
       if (oldPath != null) {
         try {
           FileSystem fs2 = oldPath.getFileSystem(conf);
@@ -2595,10 +2600,13 @@ private void constructOneLBLocationMap(F
             if (FileUtils.isSubDir(oldPath, destf, fs2)) {
               FileUtils.trashFilesUnderDir(fs2, oldPath, conf);
             }
+            if (inheritPerms) {
+              inheritFromTable(tablePath, destf, conf, destFs);
+            }
           }
         } catch (Exception e) {
           //swallow the exception
-          LOG.warn("Directory " + oldPath.toString() + " cannot be removed: " + StringUtils.stringifyException(e), e);
+          LOG.warn("Directory " + oldPath.toString() + " cannot be removed: " + e, e);
         }
       }
 
@@ -2612,9 +2620,7 @@ private void constructOneLBLocationMap(F
             LOG.warn("Error creating directory " + destf.toString());
           }
           if (inheritPerms && success) {
-            FsPermission perm = destFs.getFileStatus(destfp.getParent()).getPermission();
-            LOG.debug("Setting permissions on " + destfp + " to " + perm);
-            destFs.setPermission(destfp, perm);
+            inheritFromTable(tablePath, destfp, conf, destFs);
           }
         }
 
@@ -2630,9 +2636,7 @@ private void constructOneLBLocationMap(F
                 LOG.warn("Error creating directory " + destParent);
               }
               if (inheritPerms && success) {
-                FsPermission perm = destFs.getFileStatus(destfp.getParent()).getPermission();
-                LOG.debug("Setting permissions on " + destfp + " to " + perm);
-                destFs.setPermission(destfp, perm);
+                inheritFromTable(tablePath, destParent, conf, destFs);
               }
             }
             if (!moveFile(conf, sdpair[0], sdpair[1], destFs, true, isSrcLocal)) {
@@ -2648,9 +2652,7 @@ private void constructOneLBLocationMap(F
             LOG.warn("Error creating directory " + destf.toString());
           }
           if (inheritPerms && success) {
-            FsPermission perm = destFs.getFileStatus(destf.getParent()).getPermission();
-            LOG.debug("Setting permissions on " + destf + " to " + perm);
-            destFs.setPermission(destf, perm);
+            inheritFromTable(tablePath, destf, conf, destFs);
           }
         }
         // srcs must be a list of files -- ensured by LoadSemanticAnalyzer
@@ -2668,6 +2670,38 @@ private void constructOneLBLocationMap(F
     }
   }
 
+  /**
+   * This method sets all paths from tablePath to destf (including destf) to have same permission as tablePath.
+   * @param tablePath path of table
+   * @param destf path of table-subdir.
+   * @param conf
+   * @param fs
+   */
+  private static void inheritFromTable(Path tablePath, Path destf, HiveConf conf, FileSystem fs) {
+    if (!FileUtils.isSubDir(destf, tablePath, fs)) {
+      //partition may not be under the parent.
+      return;
+    }
+    HadoopShims shims = ShimLoader.getHadoopShims();
+    //Calculate all the paths from the table dir, to destf
+    //At end of this loop, currPath is table dir, and pathsToSet contain list of all those paths.
+    Path currPath = destf;
+    List<Path> pathsToSet = new LinkedList<Path>();
+    while (!currPath.equals(tablePath)) {
+      pathsToSet.add(currPath);
+      currPath = currPath.getParent();
+    }
+
+    try {
+      HadoopShims.HdfsFileStatus fullFileStatus = shims.getFullFileStatus(conf, fs, currPath);
+      for (Path pathToSet : pathsToSet) {
+        shims.setFullFileStatus(conf, fullFileStatus, fs, pathToSet);
+      }
+    } catch (Exception e) {
+      LOG.warn("Error setting permissions or group of " + destf, e);
+    }
+  }
+
   public static boolean isHadoop1() {
     return ShimLoader.getMajorVersion().startsWith("0.20");
   }

Modified: hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Table.java
URL: http://svn.apache.org/viewvc/hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Table.java?rev=1645078&r1=1645077&r2=1645078&view=diff
==============================================================================
--- hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Table.java (original)
+++ hive/branches/HIVE-8065/ql/src/java/org/apache/hadoop/hive/ql/metadata/Table.java Fri Dec 12 21:22:53 2014
@@ -663,7 +663,7 @@ public class Table implements Serializab
   protected void replaceFiles(Path srcf, boolean isSrcLocal)
       throws HiveException {
     Path tableDest = getPath();
-    Hive.replaceFiles(srcf, tableDest, tableDest, Hive.get().getConf(),
+    Hive.replaceFiles(tableDest, srcf, tableDest, tableDest, Hive.get().getConf(),
         isSrcLocal);
   }