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

svn commit: r1622556 - in /hive/trunk: common/src/java/org/apache/hadoop/hive/common/ common/src/java/org/apache/hadoop/hive/conf/ itests/hive-unit/src/main/java/org/apache/hive/jdbc/miniHS2/ itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/ it...

Author: vgumashta
Date: Thu Sep  4 20:25:49 2014
New Revision: 1622556

URL: http://svn.apache.org/r1622556
Log:
HIVE-6847: Improve / fix bugs in Hive scratch dir setup (Vaibhav Gumashta reviewed by Jason Dere, Szehon Ho, Thejas Nair, Venki Korukanti)

Removed:
    hive/trunk/itests/hive-unit/src/test/java/org/apache/hadoop/hive/ql/TestUtilitiesDfs.java
    hive/trunk/service/src/test/org/apache/hive/service/cli/TestScratchDir.java
Modified:
    hive/trunk/common/src/java/org/apache/hadoop/hive/common/ServerUtils.java
    hive/trunk/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
    hive/trunk/itests/hive-unit/src/main/java/org/apache/hive/jdbc/miniHS2/MiniHS2.java
    hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniHS2.java
    hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniMr.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/Context.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezSessionState.java
    hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/session/SessionState.java
    hive/trunk/ql/src/test/org/apache/hadoop/hive/ql/exec/TestUtilities.java
    hive/trunk/service/src/java/org/apache/hive/service/cli/CLIService.java
    hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionBase.java
    hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionImpl.java
    hive/trunk/service/src/java/org/apache/hive/service/cli/session/SessionManager.java

Modified: hive/trunk/common/src/java/org/apache/hadoop/hive/common/ServerUtils.java
URL: http://svn.apache.org/viewvc/hive/trunk/common/src/java/org/apache/hadoop/hive/common/ServerUtils.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/common/src/java/org/apache/hadoop/hive/common/ServerUtils.java (original)
+++ hive/trunk/common/src/java/org/apache/hadoop/hive/common/ServerUtils.java Thu Sep  4 20:25:49 2014
@@ -25,9 +25,7 @@ import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.hive.conf.HiveConf;
 
 /**
- *
- * ServerUtils.
- *
+ * ServerUtils (specific to HiveServer version 1)
  */
 public class ServerUtils {
 

Modified: hive/trunk/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java
URL: http://svn.apache.org/viewvc/hive/trunk/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java (original)
+++ hive/trunk/common/src/java/org/apache/hadoop/hive/conf/HiveConf.java Thu Sep  4 20:25:49 2014
@@ -206,11 +206,18 @@ public class HiveConf extends Configurat
     PLAN_SERIALIZATION("hive.plan.serialization.format", "kryo",
         "Query plan format serialization between client and task nodes. \n" +
         "Two supported values are : kryo and javaXML. Kryo is default."),
-    SCRATCHDIR("hive.exec.scratchdir", "/tmp/hive-${system:user.name}", "Scratch space for Hive jobs"),
+    SCRATCHDIR("hive.exec.scratchdir", "/tmp/hive", 
+        "HDFS root scratch dir for Hive jobs which gets created with 777 permission. " +
+        "For each connecting user, an HDFS scratch dir: ${hive.exec.scratchdir}/<username> is created, " +
+        "with ${hive.scratch.dir.permission}."),
     LOCALSCRATCHDIR("hive.exec.local.scratchdir",
         "${system:java.io.tmpdir}" + File.separator + "${system:user.name}",
         "Local scratch space for Hive jobs"),
-    SCRATCHDIRPERMISSION("hive.scratch.dir.permission", "700", ""),
+    DOWNLOADED_RESOURCES_DIR("hive.downloaded.resources.dir",
+        "${system:java.io.tmpdir}" + File.separator + "${hive.session.id}_resources",
+        "Temporary local directory for added resources in the remote file system."),
+    SCRATCHDIRPERMISSION("hive.scratch.dir.permission", "700", 
+        "The permission for the user specific scratch directories that get created."),
     SUBMITVIACHILD("hive.exec.submitviachild", false, ""),
     SUBMITLOCALTASKVIACHILD("hive.exec.submit.local.task.via.child", true,
         "Determines whether local tasks (typically mapjoin hashtable generation phase) runs in \n" +
@@ -274,9 +281,6 @@ public class HiveConf extends Configurat
         "Maximum number of dynamic partitions allowed to be created in each mapper/reducer node."),
     MAXCREATEDFILES("hive.exec.max.created.files", 100000L,
         "Maximum number of HDFS files created by all mappers/reducers in a MapReduce job."),
-    DOWNLOADED_RESOURCES_DIR("hive.downloaded.resources.dir",
-        "${system:java.io.tmpdir}" + File.separator + "${hive.session.id}_resources",
-        "Temporary local directory for added resources in the remote file system."),
     DEFAULTPARTITIONNAME("hive.exec.default.partition.name", "__HIVE_DEFAULT_PARTITION__",
         "The default partition name in case the dynamic partition column value is null/empty string or any other values that cannot be escaped. \n" +
         "This value must not contain any special character used in HDFS URI (e.g., ':', '%', '/' etc). \n" +

Modified: hive/trunk/itests/hive-unit/src/main/java/org/apache/hive/jdbc/miniHS2/MiniHS2.java
URL: http://svn.apache.org/viewvc/hive/trunk/itests/hive-unit/src/main/java/org/apache/hive/jdbc/miniHS2/MiniHS2.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/itests/hive-unit/src/main/java/org/apache/hive/jdbc/miniHS2/MiniHS2.java (original)
+++ hive/trunk/itests/hive-unit/src/main/java/org/apache/hive/jdbc/miniHS2/MiniHS2.java Thu Sep  4 20:25:49 2014
@@ -31,7 +31,6 @@ import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
-import org.apache.hadoop.hive.metastore.HiveMetaStore;
 import org.apache.hadoop.hive.metastore.MetaStoreUtils;
 import org.apache.hadoop.hive.shims.HadoopShims.MiniDFSShim;
 import org.apache.hadoop.hive.shims.HadoopShims.MiniMrShim;
@@ -57,6 +56,7 @@ public class MiniHS2 extends AbstractHiv
   private static final AtomicLong hs2Counter = new AtomicLong();
   private MiniMrShim mr;
   private MiniDFSShim dfs;
+  private FileSystem localFS;
   private boolean useMiniMR = false;
   private boolean useMiniKdc = false;
   private final String serverPrincipal;
@@ -137,6 +137,10 @@ public class MiniHS2 extends AbstractHiv
     this.dfs = dfs;
   }
 
+  public FileSystem getLocalFS() {
+    return localFS;
+  }
+
   public boolean isUseMiniMR() {
     return useMiniMR;
   }
@@ -157,7 +161,8 @@ public class MiniHS2 extends AbstractHiv
     this.serverPrincipal = serverPrincipal;
     this.serverKeytab = serverKeytab;
     this.isMetastoreRemote = isMetastoreRemote;
-    baseDir =  Files.createTempDir();
+    baseDir = Files.createTempDir();
+    localFS = FileSystem.getLocal(hiveConf);
     FileSystem fs;
     if (useMiniMR) {
       dfs = ShimLoader.getHadoopShims().getMiniDfs(hiveConf, 4, true, null);
@@ -371,7 +376,7 @@ public class MiniHS2 extends AbstractHiv
           getMiniKdc().loginUser(getMiniKdc().getDefaultUserPrincipal());
           sessionConf.put("principal", serverPrincipal);
         }
-        */
+         */
         sessionHandle = hs2Client.openSession("foo", "bar", sessionConf);
       } catch (Exception e) {
         // service not started yet

Modified: hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniHS2.java
URL: http://svn.apache.org/viewvc/hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniHS2.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniHS2.java (original)
+++ hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniHS2.java Thu Sep  4 20:25:49 2014
@@ -31,7 +31,9 @@ import java.sql.Statement;
 import java.util.HashMap;
 import java.util.Map;
 
+import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
+import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
 import org.apache.hive.jdbc.miniHS2.MiniHS2;
@@ -41,255 +43,357 @@ import org.junit.Before;
 import org.junit.BeforeClass;
 import org.junit.Test;
 
-  public class TestJdbcWithMiniHS2 {
-    private static MiniHS2 miniHS2 = null;
-    private static Path dataFilePath;
-
-    private Connection hs2Conn = null;
-
-    @BeforeClass
-    public static void beforeTest() throws Exception {
-      Class.forName(MiniHS2.getJdbcDriverName());
-      HiveConf conf = new HiveConf();
-      conf.setBoolVar(ConfVars.HIVE_SUPPORT_CONCURRENCY, false);
-      miniHS2 = new MiniHS2(conf);
-      String dataFileDir = conf.get("test.data.files").replace('\\', '/')
-          .replace("c:", "");
-      dataFilePath = new Path(dataFileDir, "kv1.txt");
-      Map<String, String> confOverlay = new HashMap<String, String>();
-      miniHS2.start(confOverlay);
-    }
+public class TestJdbcWithMiniHS2 {
+  private static MiniHS2 miniHS2 = null;
+  private static Path dataFilePath;
+
+  private Connection hs2Conn = null;
+
+  @BeforeClass
+  public static void beforeTest() throws Exception {
+    Class.forName(MiniHS2.getJdbcDriverName());
+    HiveConf conf = new HiveConf();
+    conf.setBoolVar(ConfVars.HIVE_SUPPORT_CONCURRENCY, false);
+    miniHS2 = new MiniHS2(conf);
+    String dataFileDir = conf.get("test.data.files").replace('\\', '/')
+        .replace("c:", "");
+    dataFilePath = new Path(dataFileDir, "kv1.txt");
+    Map<String, String> confOverlay = new HashMap<String, String>();
+    miniHS2.start(confOverlay);
+  }
 
-    @Before
-    public void setUp() throws Exception {
-      hs2Conn = getConnection(miniHS2.getJdbcURL(), System.getProperty("user.name"), "bar");
-    }
+  @Before
+  public void setUp() throws Exception {
+    hs2Conn = getConnection(miniHS2.getJdbcURL(), System.getProperty("user.name"), "bar");
+  }
 
-    private Connection getConnection(String jdbcURL, String user, String pwd) throws SQLException {
-      Connection conn = DriverManager.getConnection(jdbcURL, user, pwd);
-      conn.createStatement().execute("set hive.support.concurrency = false");
-      return conn;
-    }
+  private Connection getConnection(String jdbcURL, String user, String pwd) throws SQLException {
+    Connection conn = DriverManager.getConnection(jdbcURL, user, pwd);
+    conn.createStatement().execute("set hive.support.concurrency = false");
+    return conn;
+  }
 
-    @After
-    public void tearDown() throws Exception {
-      hs2Conn.close();
+  @After
+  public void tearDown() throws Exception {
+    hs2Conn.close();
+  }
+
+  @AfterClass
+  public static void afterTest() throws Exception {
+    if (miniHS2.isStarted()) {
+      miniHS2.stop();
     }
+  }
+
+  @Test
+  public void testConnection() throws Exception {
+    String tableName = "testTab1";
+    Statement stmt = hs2Conn.createStatement();
+
+    // create table
+    stmt.execute("DROP TABLE IF EXISTS " + tableName);
+    stmt.execute("CREATE TABLE " + tableName
+        + " (under_col INT COMMENT 'the under column', value STRING) COMMENT ' test table'");
+
+    // load data
+    stmt.execute("load data local inpath '"
+        + dataFilePath.toString() + "' into table " + tableName);
+
+    ResultSet res = stmt.executeQuery("SELECT * FROM " + tableName);
+    assertTrue(res.next());
+    assertEquals("val_238", res.getString(2));
+    res.close();
+    stmt.close();
+  }
+
 
-    @AfterClass
-    public static void afterTest() throws Exception {
-      if (miniHS2.isStarted())
-        miniHS2.stop();
+  /**   This test is to connect to any database without using the command "Use <<DB>>"
+   *  1)connect to default database.
+   *  2) Create a new DB test_default.
+   *  3) Connect to test_default database.
+   *  4) Connect and create table under test_default_test.
+   *  5) Connect and display all tables.
+   *  6) Connect to default database and shouldn't find table test_default_test.
+   *  7) Connect and drop test_default_test.
+   *  8) drop test_default database.
+   */
+
+  @Test
+  public void testURIDatabaseName() throws Exception{
+
+    String  jdbcUri  = miniHS2.getJdbcURL().substring(0, miniHS2.getJdbcURL().indexOf("default"));
+
+    hs2Conn= getConnection(jdbcUri+"default", System.getProperty("user.name"),"bar");
+    String dbName="test_connection_non_default_db";
+    String tableInNonDefaultSchema="table_in_non_default_schema";
+    Statement stmt = hs2Conn.createStatement();
+    stmt.execute("create database  if not exists "+dbName);
+    stmt.close();
+    hs2Conn.close();
+
+    hs2Conn = getConnection(jdbcUri+dbName,System.getProperty("user.name"),"bar");
+    stmt = hs2Conn .createStatement();
+    boolean expected = stmt.execute(" create table "+tableInNonDefaultSchema +" (x int)");
+    stmt.close();
+    hs2Conn .close();
+
+    hs2Conn  = getConnection(jdbcUri+dbName,System.getProperty("user.name"),"bar");
+    stmt = hs2Conn .createStatement();
+    ResultSet res = stmt.executeQuery("show tables");
+    boolean testTableExists = false;
+    while (res.next()) {
+      assertNotNull("table name is null in result set", res.getString(1));
+      if (tableInNonDefaultSchema.equalsIgnoreCase(res.getString(1))) {
+        testTableExists = true;
+      }
+    }
+    assertTrue("table name  "+tableInNonDefaultSchema
+        + "   found in SHOW TABLES result set", testTableExists);
+    stmt.close();
+    hs2Conn .close();
+
+    hs2Conn  = getConnection(jdbcUri+"default",System.getProperty("user.name"),"bar");
+    stmt = hs2Conn .createStatement();
+    res = stmt.executeQuery("show tables");
+    testTableExists = false;
+    while (res.next()) {
+      assertNotNull("table name is null in result set", res.getString(1));
+      if (tableInNonDefaultSchema.equalsIgnoreCase(res.getString(1))) {
+        testTableExists = true;
+      }
     }
 
-    @Test
-    public void testConnection() throws Exception {
-      String tableName = "testTab1";
-      Statement stmt = hs2Conn.createStatement();
-
-      // create table
-      stmt.execute("DROP TABLE IF EXISTS " + tableName);
-      stmt.execute("CREATE TABLE " + tableName
-          + " (under_col INT COMMENT 'the under column', value STRING) COMMENT ' test table'");
-
-      // load data
-      stmt.execute("load data local inpath '"
-          + dataFilePath.toString() + "' into table " + tableName);
-
-      ResultSet res = stmt.executeQuery("SELECT * FROM " + tableName);
-      assertTrue(res.next());
-      assertEquals("val_238", res.getString(2));
-      res.close();
-      stmt.close();
+    assertFalse("table name "+tableInNonDefaultSchema
+        + "  NOT  found in SHOW TABLES result set", testTableExists);
+    stmt.close();
+    hs2Conn .close();
+
+    hs2Conn  = getConnection(jdbcUri+dbName,System.getProperty("user.name"),"bar");
+    stmt = hs2Conn .createStatement();
+    stmt.execute("set hive.support.concurrency = false");
+    res = stmt.executeQuery("show tables");
+
+    stmt.execute(" drop table if exists table_in_non_default_schema");
+    expected = stmt.execute("DROP DATABASE "+ dbName);
+    stmt.close();
+
+    hs2Conn  = getConnection(jdbcUri+"default",System.getProperty("user.name"),"bar");
+    stmt = hs2Conn .createStatement();
+    res = stmt.executeQuery("show tables");
+    testTableExists = false;
+    while (res.next()) {
+      assertNotNull("table name is null in result set", res.getString(1));
+      if (tableInNonDefaultSchema.equalsIgnoreCase(res.getString(1))) {
+        testTableExists = true;
+      }
     }
 
+    // test URI with no dbName
+    hs2Conn  = getConnection(jdbcUri, System.getProperty("user.name"),"bar");
+    verifyCurrentDB("default", hs2Conn);
+    hs2Conn.close();
+
+    hs2Conn  = getConnection(jdbcUri + ";", System.getProperty("user.name"),"bar");
+    verifyCurrentDB("default", hs2Conn);
+    hs2Conn.close();
+
+    hs2Conn  = getConnection(jdbcUri + ";/foo=bar;foo1=bar1", System.getProperty("user.name"),"bar");
+    verifyCurrentDB("default", hs2Conn);
+    hs2Conn.close();
+  }
 
-    /**   This test is to connect to any database without using the command "Use <<DB>>"
-     *  1)connect to default database.
-     *  2) Create a new DB test_default.
-     *  3) Connect to test_default database.
-     *  4) Connect and create table under test_default_test.
-     *  5) Connect and display all tables.
-     *  6) Connect to default database and shouldn't find table test_default_test.
-     *  7) Connect and drop test_default_test.
-     *  8) drop test_default database.
+  @Test
+  public void testConnectionSchemaAPIs() throws Exception {
+    String db1 = "DB1";
+    /**
+     * get/set Schema are new in JDK7 and not available in java.sql.Connection in JDK6.
+     * Hence the test uses HiveConnection object to call these methods so that test will run with older JDKs
      */
+    HiveConnection hiveConn = (HiveConnection)hs2Conn;
 
-     @Test
-    public void testURIDatabaseName() throws Exception{
+    assertEquals("default", hiveConn.getSchema());
+    Statement stmt = hs2Conn.createStatement();
+    stmt.execute("DROP DATABASE IF EXISTS " + db1 + " CASCADE");
+    stmt.execute("CREATE DATABASE " + db1);
+    assertEquals("default", hiveConn.getSchema());
+
+    stmt.execute("USE " + db1);
+    assertEquals(db1, hiveConn.getSchema());
+
+    stmt.execute("USE default");
+    assertEquals("default", hiveConn.getSchema());
+
+    hiveConn.setSchema(db1);
+    assertEquals(db1, hiveConn.getSchema());
+    hiveConn.setSchema("default");
+    assertEquals("default", hiveConn.getSchema());
+
+    assertTrue(hiveConn.getCatalog().isEmpty());
+    hiveConn.setCatalog("foo");
+    assertTrue(hiveConn.getCatalog().isEmpty());
+  }
 
-     String  jdbcUri  = miniHS2.getJdbcURL().substring(0, miniHS2.getJdbcURL().indexOf("default"));
+  /**
+   * verify that the current db is the one expected. first create table as <db>.tab and then
+   * describe that table to check if <db> is the current database
+   * @param expectedDbName
+   * @param hs2Conn
+   * @throws Exception
+   */
+  private void verifyCurrentDB(String expectedDbName, Connection hs2Conn) throws Exception {
+    String verifyTab = "miniHS2DbVerificationTable";
+    Statement stmt = hs2Conn.createStatement();
+    stmt.execute("DROP TABLE IF EXISTS " + expectedDbName + "." + verifyTab);
+    stmt.execute("CREATE TABLE " + expectedDbName + "." + verifyTab + "(id INT)");
+    stmt.execute("DESCRIBE " + verifyTab);
+    stmt.execute("DROP TABLE IF EXISTS " + expectedDbName + "." + verifyTab);
+    stmt.close();
+  }
 
-     hs2Conn= getConnection(jdbcUri+"default",System.getProperty("user.name"),"bar");
-     String dbName="test_connection_non_default_db";
-     String tableInNonDefaultSchema="table_in_non_default_schema";
-     Statement stmt = hs2Conn.createStatement();
-     stmt.execute("create database  if not exists "+dbName);
-     stmt.close();
-     hs2Conn.close();
-
-     hs2Conn = getConnection(jdbcUri+dbName,System.getProperty("user.name"),"bar");
-     stmt = hs2Conn .createStatement();
-     boolean expected = stmt.execute(" create table "+tableInNonDefaultSchema +" (x int)");
-     stmt.close();
-     hs2Conn .close();
-
-     hs2Conn  = getConnection(jdbcUri+dbName,System.getProperty("user.name"),"bar");
-     stmt = hs2Conn .createStatement();
-     ResultSet res = stmt.executeQuery("show tables");
-     boolean testTableExists = false;
-     while (res.next()) {
-        assertNotNull("table name is null in result set", res.getString(1));
-        if (tableInNonDefaultSchema.equalsIgnoreCase(res.getString(1))) {
-          testTableExists = true;
-        }
-     }
-     assertTrue("table name  "+tableInNonDefaultSchema
-           + "   found in SHOW TABLES result set", testTableExists);
-     stmt.close();
-     hs2Conn .close();
-
-     hs2Conn  = getConnection(jdbcUri+"default",System.getProperty("user.name"),"bar");
-     stmt = hs2Conn .createStatement();
-     res = stmt.executeQuery("show tables");
-     testTableExists = false;
-     while (res.next()) {
-       assertNotNull("table name is null in result set", res.getString(1));
-       if (tableInNonDefaultSchema.equalsIgnoreCase(res.getString(1))) {
-         testTableExists = true;
-        }
-     }
-
-     assertFalse("table name "+tableInNonDefaultSchema
-           + "  NOT  found in SHOW TABLES result set", testTableExists);
-     stmt.close();
-     hs2Conn .close();
-
-     hs2Conn  = getConnection(jdbcUri+dbName,System.getProperty("user.name"),"bar");
-     stmt = hs2Conn .createStatement();
-     stmt.execute("set hive.support.concurrency = false");
-     res = stmt.executeQuery("show tables");
-
-     stmt.execute(" drop table if exists table_in_non_default_schema");
-     expected = stmt.execute("DROP DATABASE "+ dbName);
-     stmt.close();
-
-     hs2Conn  = getConnection(jdbcUri+"default",System.getProperty("user.name"),"bar");
-     stmt = hs2Conn .createStatement();
-     res = stmt.executeQuery("show tables");
-     testTableExists = false;
-     while (res.next()) {
-       assertNotNull("table name is null in result set", res.getString(1));
-       if (tableInNonDefaultSchema.equalsIgnoreCase(res.getString(1))) {
-         testTableExists = true;
-        }
-     }
-
-     // test URI with no dbName
-     hs2Conn  = getConnection(jdbcUri, System.getProperty("user.name"),"bar");
-     verifyCurrentDB("default", hs2Conn);
-     hs2Conn.close();
-
-     hs2Conn  = getConnection(jdbcUri + ";", System.getProperty("user.name"),"bar");
-     verifyCurrentDB("default", hs2Conn);
-     hs2Conn.close();
-
-     hs2Conn  = getConnection(jdbcUri + ";/foo=bar;foo1=bar1", System.getProperty("user.name"),"bar");
-     verifyCurrentDB("default", hs2Conn);
-     hs2Conn.close();
-     }
-
-     @Test
-     public void testConnectionSchemaAPIs() throws Exception {
-       String db1 = "DB1";
-       /**
-        * get/set Schema are new in JDK7 and not available in java.sql.Connection in JDK6.
-        * Hence the test uses HiveConnection object to call these methods so that test will run with older JDKs
-        */
-       HiveConnection hiveConn = (HiveConnection)hs2Conn;
-
-       assertEquals("default", hiveConn.getSchema());
-       Statement stmt = hs2Conn.createStatement();
-       stmt.execute("DROP DATABASE IF EXISTS " + db1 + " CASCADE");
-       stmt.execute("CREATE DATABASE " + db1);
-       assertEquals("default", hiveConn.getSchema());
-
-       stmt.execute("USE " + db1);
-       assertEquals(db1, hiveConn.getSchema());
-
-       stmt.execute("USE default");
-       assertEquals("default", hiveConn.getSchema());
-
-       hiveConn.setSchema(db1);
-       assertEquals(db1, hiveConn.getSchema());
-       hiveConn.setSchema("default");
-       assertEquals("default", hiveConn.getSchema());
-
-       assertTrue(hiveConn.getCatalog().isEmpty());
-       hiveConn.setCatalog("foo");
-       assertTrue(hiveConn.getCatalog().isEmpty());
-     }
-
-   /**
-    * verify that the current db is the one expected. first create table as <db>.tab and then
-    * describe that table to check if <db> is the current database
-    * @param expectedDbName
-    * @param hs2Conn
-    * @throws Exception
-    */
-   private void verifyCurrentDB(String expectedDbName, Connection hs2Conn) throws Exception {
-     String verifyTab = "miniHS2DbVerificationTable";
-     Statement stmt = hs2Conn.createStatement();
-     stmt.execute("DROP TABLE IF EXISTS " + expectedDbName + "." + verifyTab);
-     stmt.execute("CREATE TABLE " + expectedDbName + "." + verifyTab + "(id INT)");
-     stmt.execute("DESCRIBE " + verifyTab);
-     stmt.execute("DROP TABLE IF EXISTS " + expectedDbName + "." + verifyTab);
-     stmt.close();
-   }
-
-   /**
-    * This method tests whether while creating a new connection, the config
-    * variables specified in the JDBC URI are properly set for the connection.
-    * This is a test for HiveConnection#configureConnection.
-    *
-    * @throws Exception
-    */
-   @Test
-   public void testNewConnectionConfiguration() throws Exception {
-
-     // Set some conf parameters
-     String hiveConf = "hive.cli.print.header=true;hive.server2.async.exec.shutdown.timeout=20;"
-         + "hive.server2.async.exec.threads=30;hive.server2.thrift.http.max.worker.threads=15";
-     // Set some conf vars
-     String hiveVar = "stab=salesTable;icol=customerID";
-     String jdbcUri = miniHS2.getJdbcURL() + "?" + hiveConf + "#" + hiveVar;
-
-     // Open a new connection with these conf & vars
-     Connection con1 = DriverManager.getConnection(jdbcUri);
-
-     // Execute "set" command and retrieve values for the conf & vars specified
-     // above
-     // Assert values retrieved
-     Statement stmt = con1.createStatement();
-
-     // Verify that the property has been properly set while creating the
-     // connection above
-     verifyConfProperty(stmt, "hive.cli.print.header", "true");
-     verifyConfProperty(stmt, "hive.server2.async.exec.shutdown.timeout", "20");
-     verifyConfProperty(stmt, "hive.server2.async.exec.threads", "30");
-     verifyConfProperty(stmt, "hive.server2.thrift.http.max.worker.threads",
-         "15");
-     verifyConfProperty(stmt, "stab", "salesTable");
-     verifyConfProperty(stmt, "icol", "customerID");
-     con1.close();
-   }
-
-   private void verifyConfProperty(Statement stmt, String property,
-       String expectedValue) throws Exception {
-     ResultSet res = stmt.executeQuery("set " + property);
-     while (res.next()) {
-       String resultValues[] = res.getString(1).split("=");
-       assertEquals(resultValues[1], expectedValue);
-     }
-   }
+  /**
+   * This method tests whether while creating a new connection, the config
+   * variables specified in the JDBC URI are properly set for the connection.
+   * This is a test for HiveConnection#configureConnection.
+   *
+   * @throws Exception
+   */
+  @Test
+  public void testNewConnectionConfiguration() throws Exception {
+
+    // Set some conf parameters
+    String hiveConf = "hive.cli.print.header=true;hive.server2.async.exec.shutdown.timeout=20;"
+        + "hive.server2.async.exec.threads=30;hive.server2.thrift.http.max.worker.threads=15";
+    // Set some conf vars
+    String hiveVar = "stab=salesTable;icol=customerID";
+    String jdbcUri = miniHS2.getJdbcURL() + "?" + hiveConf + "#" + hiveVar;
+
+    // Open a new connection with these conf & vars
+    Connection con1 = DriverManager.getConnection(jdbcUri);
+
+    // Execute "set" command and retrieve values for the conf & vars specified
+    // above
+    // Assert values retrieved
+    Statement stmt = con1.createStatement();
+
+    // Verify that the property has been properly set while creating the
+    // connection above
+    verifyConfProperty(stmt, "hive.cli.print.header", "true");
+    verifyConfProperty(stmt, "hive.server2.async.exec.shutdown.timeout", "20");
+    verifyConfProperty(stmt, "hive.server2.async.exec.threads", "30");
+    verifyConfProperty(stmt, "hive.server2.thrift.http.max.worker.threads",
+        "15");
+    verifyConfProperty(stmt, "stab", "salesTable");
+    verifyConfProperty(stmt, "icol", "customerID");
+    con1.close();
+  }
+
+  private void verifyConfProperty(Statement stmt, String property,
+      String expectedValue) throws Exception {
+    ResultSet res = stmt.executeQuery("set " + property);
+    while (res.next()) {
+      String resultValues[] = res.getString(1).split("=");
+      assertEquals(resultValues[1], expectedValue);
+    }
+  }
+
+  /**
+   * Tests the creation of the 3 scratch dirs: hdfs, local, downloaded resources (which is also local).
+   * 1. Test with doAs=false: open a new JDBC session and verify the presence of directories/permissions
+   * 2. Test with doAs=true: open a new JDBC session and verify the presence of directories/permissions
+   * @throws Exception
+   */
+  @Test
+  public void testScratchDirs() throws Exception {
+    // Stop HiveServer2
+    if (miniHS2.isStarted()) {
+      miniHS2.stop();
+    }
+    HiveConf conf = new HiveConf();
+    String userName;
+    Path scratchDirPath;
+    // 1. Test with doAs=false
+    conf.setBoolean("hive.server2.enable.doAs", false);
+    // Set a custom prefix for hdfs scratch dir path
+    conf.set("hive.exec.scratchdir", "/tmp/hs2");
+    // Set a scratch dir permission
+    String fsPermissionStr = "700";
+    conf.set("hive.scratch.dir.permission", fsPermissionStr);
+    // Start an instance of HiveServer2 which uses miniMR
+    miniHS2 = new MiniHS2(conf);
+    Map<String, String> confOverlay = new HashMap<String, String>();
+    miniHS2.start(confOverlay);
+    userName = System.getProperty("user.name");
+    hs2Conn = getConnection(miniHS2.getJdbcURL(), userName, "password");
+    // FS
+    FileSystem fs = miniHS2.getLocalFS();
+
+    // Verify scratch dir paths and permission
+    // HDFS scratch dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIR) + "/" + userName);
+    verifyScratchDir(conf, fs, scratchDirPath, userName, false);
+
+    // Local scratch dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.LOCALSCRATCHDIR));
+    verifyScratchDir(conf, fs, scratchDirPath, userName, true);
+
+    // Downloaded resources dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.DOWNLOADED_RESOURCES_DIR));
+    verifyScratchDir(conf, fs, scratchDirPath, userName, true);
+
+    // 2. Test with doAs=true
+    // Restart HiveServer2 with doAs=true
+    if (miniHS2.isStarted()) {
+      miniHS2.stop();
+    }
+    conf.setBoolean("hive.server2.enable.doAs", true);
+    // Start HS2
+    miniHS2 = new MiniHS2(conf);
+    miniHS2.start(confOverlay);
+    // Test for user "neo"
+    userName = "neo";
+    hs2Conn = getConnection(miniHS2.getJdbcURL(), userName, "the-one");
+
+    // Verify scratch dir paths and permission
+    // HDFS scratch dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIR) + "/" + userName);
+    verifyScratchDir(conf, fs, scratchDirPath, userName, false);
+
+    // Local scratch dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.LOCALSCRATCHDIR));
+    verifyScratchDir(conf, fs, scratchDirPath, userName, true);
+
+    // Downloaded resources dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.DOWNLOADED_RESOURCES_DIR));
+    verifyScratchDir(conf, fs, scratchDirPath, userName, true);
+
+    // Test for user "trinity"
+    userName = "trinity";
+    hs2Conn = getConnection(miniHS2.getJdbcURL(), userName, "the-one");
+
+    // Verify scratch dir paths and permission
+    // HDFS scratch dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIR) + "/" + userName);
+    verifyScratchDir(conf, fs, scratchDirPath, userName, false);
+
+    // Local scratch dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.LOCALSCRATCHDIR));
+    verifyScratchDir(conf, fs, scratchDirPath, userName, true);
+
+    // Downloaded resources dir
+    scratchDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.DOWNLOADED_RESOURCES_DIR));
+    verifyScratchDir(conf, fs, scratchDirPath, userName, true);
+  }
+
+  private void verifyScratchDir(HiveConf conf, FileSystem fs, Path scratchDirPath,
+      String userName, boolean isLocal) throws Exception {
+    String dirType = isLocal ? "Local" : "DFS";
+    FsPermission expectedFSPermission = new FsPermission(HiveConf.getVar(conf,
+        HiveConf.ConfVars.SCRATCHDIRPERMISSION));
+    assertTrue("The expected " + dirType + " scratch dir does not exist for the user: " +
+        userName, fs.exists(scratchDirPath));
+    if (fs.exists(scratchDirPath) && !isLocal) {
+      assertEquals("DFS scratch dir permissions don't match", expectedFSPermission,
+          fs.getFileStatus(scratchDirPath).getPermission());
+    }
   }
+}
\ No newline at end of file

Modified: hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniMr.java
URL: http://svn.apache.org/viewvc/hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniMr.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniMr.java (original)
+++ hive/trunk/itests/hive-unit/src/test/java/org/apache/hive/jdbc/TestJdbcWithMiniMr.java Thu Sep  4 20:25:49 2014
@@ -19,16 +19,16 @@
 package org.apache.hive.jdbc;
 
 import static org.junit.Assert.assertEquals;
-import static org.junit.Assert.assertTrue;
 import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
 
-import java.util.Map;
-import java.util.HashMap;
 import java.sql.Connection;
 import java.sql.DriverManager;
 import java.sql.ResultSet;
 import java.sql.SQLException;
 import java.sql.Statement;
+import java.util.HashMap;
+import java.util.Map;
 
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
@@ -48,11 +48,11 @@ public class TestJdbcWithMiniMr {
   public static final String TEST_TAG = "miniHS2.miniMr.tag";
   public static final String TEST_TAG_VALUE = "miniHS2.miniMr.value";
   public static class MiniMrTestSessionHook implements HiveSessionHook {
-     @Override
-     public void run(HiveSessionHookContext sessionHookContext) throws HiveSQLException {
-       sessionHookContext.getSessionConf().set(TEST_TAG, TEST_TAG_VALUE);
-     }
-   }
+    @Override
+    public void run(HiveSessionHookContext sessionHookContext) throws HiveSQLException {
+      sessionHookContext.getSessionConf().set(TEST_TAG, TEST_TAG_VALUE);
+    }
+  }
 
   private static MiniHS2 miniHS2 = null;
   private static HiveConf conf;
@@ -93,7 +93,7 @@ public class TestJdbcWithMiniMr {
   @Before
   public void setUp() throws Exception {
     hs2Conn = DriverManager.getConnection(miniHS2.getJdbcURL(dbName),
-            System.getProperty("user.name"), "bar");
+        System.getProperty("user.name"), "bar");
     stmt = hs2Conn.createStatement();
     stmt.execute("USE " + dbName);
   }
@@ -225,7 +225,7 @@ public class TestJdbcWithMiniMr {
     String queryStr = "SELECT * FROM " + tempTableName +
         " where value = '" + resultVal + "'";
     verifyResult(queryStr, resultVal, 2);
-    
+
     // A second connection should not be able to see the table
     Connection conn2 = DriverManager.getConnection(miniHS2.getJdbcURL(dbName),
         System.getProperty("user.name"), "bar");

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/Context.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/Context.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/Context.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/Context.java Thu Sep  4 20:25:49 2014
@@ -18,6 +18,18 @@
 
 package org.apache.hadoop.hive.ql;
 
+import java.io.DataInput;
+import java.io.FileNotFoundException;
+import java.io.IOException;
+import java.net.URI;
+import java.text.SimpleDateFormat;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.Random;
+import java.util.concurrent.ConcurrentHashMap;
+
 import org.antlr.runtime.TokenRewriteStream;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
@@ -30,7 +42,6 @@ import org.apache.hadoop.fs.permission.F
 import org.apache.hadoop.hive.common.FileUtils;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.ql.exec.TaskRunner;
-import org.apache.hadoop.hive.ql.exec.Utilities;
 import org.apache.hadoop.hive.ql.hooks.WriteEntity;
 import org.apache.hadoop.hive.ql.lockmgr.HiveLock;
 import org.apache.hadoop.hive.ql.lockmgr.HiveLockManager;
@@ -41,18 +52,6 @@ import org.apache.hadoop.hive.ql.session
 import org.apache.hadoop.hive.shims.ShimLoader;
 import org.apache.hadoop.util.StringUtils;
 
-import java.io.DataInput;
-import java.io.FileNotFoundException;
-import java.io.IOException;
-import java.net.URI;
-import java.text.SimpleDateFormat;
-import java.util.Date;
-import java.util.HashMap;
-import java.util.List;
-import java.util.Map;
-import java.util.Random;
-import java.util.concurrent.ConcurrentHashMap;
-
 /**
  * Context for Semantic Analyzers. Usage: not reusable - construct a new one for
  * each query should call clear() at end of use to remove temporary folders
@@ -191,7 +190,7 @@ public class Context {
    * @param scratchDir path of tmp directory
    */
   private Path getScratchDir(String scheme, String authority,
-                               boolean mkdir, String scratchDir) {
+      boolean mkdir, String scratchDir) {
 
     String fileSystem =  scheme + ":" + authority;
     Path dir = fsScratchDirs.get(fileSystem + "-" + TaskRunner.getTaskRunnerID());
@@ -203,11 +202,11 @@ public class Context {
         try {
           FileSystem fs = dirPath.getFileSystem(conf);
           dirPath = new Path(fs.makeQualified(dirPath).toString());
-          FsPermission fsPermission = new FsPermission(Short.parseShort(scratchDirPermission.trim(), 8));
+          FsPermission fsPermission = new FsPermission(scratchDirPermission);
 
-          if (!Utilities.createDirsWithPermission(conf, dirPath, fsPermission)) {
+          if (!fs.mkdirs(dirPath, fsPermission)) {
             throw new RuntimeException("Cannot make directory: "
-                                       + dirPath.toString());
+                + dirPath.toString());
           }
           if (isHDFSCleanup) {
             fs.deleteOnExit(dirPath);
@@ -233,7 +232,7 @@ public class Context {
       FileSystem fs = FileSystem.getLocal(conf);
       URI uri = fs.getUri();
       return getScratchDir(uri.getScheme(), uri.getAuthority(),
-                           mkdir, localScratchDir);
+          mkdir, localScratchDir);
     } catch (IOException e) {
       throw new RuntimeException (e);
     }
@@ -257,7 +256,7 @@ public class Context {
       URI uri = dir.toUri();
 
       Path newScratchDir = getScratchDir(uri.getScheme(), uri.getAuthority(),
-                           !explain, uri.getPath());
+          !explain, uri.getPath());
       LOG.info("New scratch dir is " + newScratchDir);
       return newScratchDir;
     } catch (IOException e) {
@@ -270,7 +269,7 @@ public class Context {
 
   private Path getExternalScratchDir(URI extURI) {
     return getScratchDir(extURI.getScheme(), extURI.getAuthority(),
-                         !explain, nonLocalScratchPath.toUri().getPath());
+        !explain, nonLocalScratchPath.toUri().getPath());
   }
 
   /**
@@ -283,7 +282,7 @@ public class Context {
         p.getFileSystem(conf).delete(p, true);
       } catch (Exception e) {
         LOG.warn("Error Removing Scratch: "
-                 + StringUtils.stringifyException(e));
+            + StringUtils.stringifyException(e));
       }
     }
     fsScratchDirs.clear();
@@ -305,7 +304,7 @@ public class Context {
    */
   public boolean isMRTmpFileURI(String uriStr) {
     return (uriStr.indexOf(executionId) != -1) &&
-      (uriStr.indexOf(MR_PREFIX) != -1);
+        (uriStr.indexOf(MR_PREFIX) != -1);
   }
 
   /**
@@ -315,7 +314,7 @@ public class Context {
    */
   public Path getMRTmpPath() {
     return new Path(getMRScratchDir(), MR_PREFIX +
-      nextPathId());
+        nextPathId());
   }
 
   /**
@@ -343,7 +342,7 @@ public class Context {
       return getExtTmpPathRelTo(path.getParent());
     }
     return new Path(getExternalScratchDir(extURI), EXT_PREFIX +
-      nextPathId());
+        nextPathId());
   }
 
   /**
@@ -353,8 +352,8 @@ public class Context {
    */
   public Path getExtTmpPathRelTo(Path path) {
     URI uri = path.toUri();
-    return new Path (getScratchDir(uri.getScheme(), uri.getAuthority(), !explain, 
-    uri.getPath() + Path.SEPARATOR + "_" + this.executionId), EXT_PREFIX + nextPathId());
+    return new Path (getScratchDir(uri.getScheme(), uri.getAuthority(), !explain,
+        uri.getPath() + Path.SEPARATOR + "_" + this.executionId), EXT_PREFIX + nextPathId());
   }
 
   /**

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/Utilities.java Thu Sep  4 20:25:49 2014
@@ -92,7 +92,6 @@ 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.FsPermission;
 import org.apache.hadoop.hive.common.HiveInterruptCallback;
 import org.apache.hadoop.hive.common.HiveInterruptUtils;
 import org.apache.hadoop.hive.common.HiveStatsUtils;
@@ -3355,7 +3354,6 @@ public final class Utilities {
   private static void createTmpDirs(Configuration conf,
       List<Operator<? extends OperatorDesc>> ops) throws IOException {
 
-    FsPermission fsPermission = new FsPermission((short)00777);
     while (!ops.isEmpty()) {
       Operator<? extends OperatorDesc> op = ops.remove(0);
 
@@ -3365,7 +3363,8 @@ public final class Utilities {
 
         if (tempDir != null) {
           Path tempPath = Utilities.toTempPath(tempDir);
-          createDirsWithPermission(conf, tempPath, fsPermission);
+          FileSystem fs = tempPath.getFileSystem(conf);
+          fs.mkdirs(tempPath);
         }
       }
 
@@ -3501,76 +3500,6 @@ public final class Utilities {
   }
 
   /**
-   * @param conf the configuration used to derive the filesystem to create the path
-   * @param mkdir the path to be created
-   * @param fsPermission ignored if it is hive server session and doAs is enabled
-   * @return true if successfully created the directory else false
-   * @throws IOException if hdfs experiences any error conditions
-   */
-  public static boolean createDirsWithPermission(Configuration conf, Path mkdir,
-      FsPermission fsPermission) throws IOException {
-
-    boolean recursive = false;
-    if (SessionState.get() != null) {
-      recursive = SessionState.get().isHiveServerQuery() &&
-          conf.getBoolean(HiveConf.ConfVars.HIVE_SERVER2_ENABLE_DOAS.varname,
-              HiveConf.ConfVars.HIVE_SERVER2_ENABLE_DOAS.defaultBoolVal);
-      // we reset the permission in case of hive server and doAs enabled because
-      // currently scratch directory uses /tmp/hive-hive as the scratch directory.
-      // However, with doAs enabled, the first user to create this directory would
-      // own the directory and subsequent users cannot access the scratch directory.
-      // The right fix is to have scratch dir per user.
-      fsPermission = new FsPermission((short)00777);
-    }
-
-    // if we made it so far without exception we are good!
-    return createDirsWithPermission(conf, mkdir, fsPermission, recursive);
-  }
-
-  private static void resetConfAndCloseFS (Configuration conf, boolean unsetUmask,
-      String origUmask, FileSystem fs) throws IOException {
-    if (unsetUmask) {
-      if (origUmask != null) {
-        conf.set(FsPermission.UMASK_LABEL, origUmask);
-      } else {
-        conf.unset(FsPermission.UMASK_LABEL);
-      }
-    }
-
-    fs.close();
-  }
-
-  public static boolean createDirsWithPermission(Configuration conf, Path mkdirPath,
-      FsPermission fsPermission, boolean recursive) throws IOException {
-    String origUmask = null;
-    LOG.debug("Create dirs " + mkdirPath + " with permission " + fsPermission + " recursive " +
-        recursive);
-
-    if (recursive) {
-      origUmask = conf.get(FsPermission.UMASK_LABEL);
-      // this umask is required because by default the hdfs mask is 022 resulting in
-      // all parents getting the fsPermission & !(022) permission instead of fsPermission
-      conf.set(FsPermission.UMASK_LABEL, "000");
-    }
-
-    FileSystem fs = ShimLoader.getHadoopShims().getNonCachedFileSystem(mkdirPath.toUri(), conf);
-    boolean retval = false;
-    try {
-      retval = fs.mkdirs(mkdirPath, fsPermission);
-      resetConfAndCloseFS(conf, recursive, origUmask, fs);
-    } catch (IOException ioe) {
-      try {
-        resetConfAndCloseFS(conf, recursive, origUmask, fs);
-      }
-      catch (IOException e) {
-        // do nothing - double failure
-      }
-    }
-    return retval;
-  }
-
-
-  /**
    * Convert path to qualified path.
    *
    * @param conf

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezSessionState.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezSessionState.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezSessionState.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/exec/tez/TezSessionState.java Thu Sep  4 20:25:49 2014
@@ -41,12 +41,9 @@ import org.apache.hadoop.fs.FileStatus;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.permission.FsPermission;
-import org.apache.hadoop.hdfs.DistributedFileSystem;
-import org.apache.hadoop.hive.common.FileUtils;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
-import org.apache.hadoop.hive.ql.ErrorMsg;
-import org.apache.hadoop.hive.ql.exec.Utilities;
+import org.apache.hadoop.hive.ql.session.SessionState;
 import org.apache.hadoop.hive.shims.ShimLoader;
 import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.yarn.api.records.LocalResource;
@@ -300,11 +297,11 @@ public class TezSessionState {
     throws IOException {
 
     // tez needs its own scratch dir (per session)
-    Path tezDir = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIR), TEZ_DIR);
+    Path tezDir = new Path(SessionState.get().getHdfsScratchDirURIString(), TEZ_DIR);
     tezDir = new Path(tezDir, sessionId);
     FileSystem fs = tezDir.getFileSystem(conf);
-    FsPermission fsPermission = new FsPermission((short)00777);
-    Utilities.createDirsWithPermission(conf, tezDir, fsPermission, true);
+    FsPermission fsPermission = new FsPermission(HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIRPERMISSION));
+    fs.mkdirs(tezDir, fsPermission);
     // Make sure the path is normalized (we expect validation to pass since we just created it).
     tezDir = DagUtils.validateTargetDir(tezDir, conf).getPath();
     // don't keep the directory around on non-clean exit

Modified: hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/session/SessionState.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/session/SessionState.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/session/SessionState.java (original)
+++ hive/trunk/ql/src/java/org/apache/hadoop/hive/ql/session/SessionState.java Thu Sep  4 20:25:49 2014
@@ -45,7 +45,6 @@ import org.apache.hadoop.fs.permission.F
 import org.apache.hadoop.hive.common.JavaUtils;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
-import org.apache.hadoop.hive.metastore.api.ColumnStatistics;
 import org.apache.hadoop.hive.metastore.api.ColumnStatisticsObj;
 import org.apache.hadoop.hive.ql.MapRedStats;
 import org.apache.hadoop.hive.ql.exec.Utilities;
@@ -69,6 +68,7 @@ import org.apache.hadoop.hive.ql.securit
 import org.apache.hadoop.hive.ql.security.authorization.plugin.HiveMetastoreClientFactoryImpl;
 import org.apache.hadoop.hive.ql.util.DosToUnix;
 import org.apache.hadoop.hive.shims.ShimLoader;
+import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.util.ReflectionUtils;
 
 import com.google.common.base.Preconditions;
@@ -208,6 +208,8 @@ public class SessionState {
    */
   private Path localSessionPath;
 
+  private String hdfsScratchDirURIString;
+
   /**
    * Get the lineage state stored in this session.
    *
@@ -344,36 +346,39 @@ public class SessionState {
 
     setCurrentSessionState(startSs);
 
-    if(startSs.hiveHist == null){
+    if (startSs.hiveHist == null){
       if (startSs.getConf().getBoolVar(HiveConf.ConfVars.HIVE_SESSION_HISTORY_ENABLED)) {
         startSs.hiveHist = new HiveHistoryImpl(startSs);
-      }else {
-        //Hive history is disabled, create a no-op proxy
+      } else {
+        // Hive history is disabled, create a no-op proxy
         startSs.hiveHist = HiveHistoryProxyHandler.getNoOpHiveHistoryProxy();
       }
     }
 
-    if (startSs.getTmpOutputFile() == null) {
-      // set temp file containing results to be sent to HiveClient
-      try {
-        startSs.setTmpOutputFile(createTempFile(startSs.getConf()));
-      } catch (IOException e) {
-        throw new RuntimeException(e);
-      }
-    }
-
     // Get the following out of the way when you start the session these take a
     // while and should be done when we start up.
     try {
-      //Hive object instance should be created with a copy of the conf object. If the conf is
+      // Hive object instance should be created with a copy of the conf object. If the conf is
       // shared with SessionState, other parts of the code might update the config, but
       // Hive.get(HiveConf) would not recognize the case when it needs refreshing
       Hive.get(new HiveConf(startSs.conf)).getMSC();
-      ShimLoader.getHadoopShims().getUGIForConf(startSs.conf);
+      UserGroupInformation sessionUGI = ShimLoader.getHadoopShims().getUGIForConf(startSs.conf);
       FileSystem.get(startSs.conf);
-      startSs.createSessionPaths(startSs.conf);
+
+      // Create scratch dirs for this session
+      startSs.createSessionDirs(sessionUGI.getShortUserName());
+
+      // Set temp file containing results to be sent to HiveClient
+      if (startSs.getTmpOutputFile() == null) {
+        try {
+          startSs.setTmpOutputFile(createTempFile(startSs.getConf()));
+        } catch (IOException e) {
+          throw new RuntimeException(e);
+        }
+      }
+
     } catch (Exception e) {
-      // catch-all due to some exec time dependencies on session state
+      // Catch-all due to some exec time dependencies on session state
       // that would cause ClassNoFoundException otherwise
       throw new RuntimeException(e);
     }
@@ -396,6 +401,88 @@ public class SessionState {
     return startSs;
   }
 
+  /**
+   * Create dirs & session paths for this session:
+   * 1. HDFS scratch dir
+   * 2. Local scratch dir
+   * 3. Local downloaded resource dir
+   * 4. HDFS session path
+   * 5. Local session path
+   * 6. HDFS temp table space
+   * @param userName
+   * @throws IOException
+   */
+  private void createSessionDirs(String userName) throws IOException {
+    HiveConf conf = getConf();
+    // First create the root scratch dir on hdfs (if it doesn't already exist) and make it writable
+    Path rootHDFSDirPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIR));
+    String rootHDFSDirPermission = "777";
+    createPath(conf, rootHDFSDirPath, rootHDFSDirPermission, false, false);
+    // Now create session specific dirs
+    String scratchDirPermission = HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIRPERMISSION);
+    Path path;
+    // 1. HDFS scratch dir
+    path = new Path(rootHDFSDirPath, userName);
+    hdfsScratchDirURIString = path.toUri().toString();
+    createPath(conf, path, scratchDirPermission, false, false);
+    // 2. Local scratch dir
+    path = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.LOCALSCRATCHDIR));
+    createPath(conf, path, scratchDirPermission, true, false);
+    // 3. Download resources dir
+    path = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.DOWNLOADED_RESOURCES_DIR));
+    createPath(conf, path, scratchDirPermission, true, false);
+    // Finally, create session paths for this session
+    // Local & non-local tmp location is configurable. however it is the same across
+    // all external file systems
+    String sessionId = getSessionId();
+    // 4. HDFS session path
+    hdfsSessionPath = new Path(hdfsScratchDirURIString, sessionId);
+    createPath(conf, hdfsSessionPath, scratchDirPermission, false, true);
+    conf.set(HDFS_SESSION_PATH_KEY, hdfsSessionPath.toUri().toString());
+    // 5. Local session path
+    localSessionPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.LOCALSCRATCHDIR), sessionId);
+    createPath(conf, localSessionPath, scratchDirPermission, true, true);
+    conf.set(LOCAL_SESSION_PATH_KEY, localSessionPath.toUri().toString());
+    // 6. HDFS temp table space
+    hdfsTmpTableSpace = new Path(hdfsSessionPath, TMP_PREFIX);
+    createPath(conf, hdfsTmpTableSpace, scratchDirPermission, false, true);
+    conf.set(TMP_TABLE_SPACE_KEY, hdfsTmpTableSpace.toUri().toString());
+  }
+
+  /**
+   * Create a given path if it doesn't exist.
+   *
+   * @param conf
+   * @param pathString
+   * @param permission
+   * @param isLocal
+   * @param isCleanUp
+   * @return
+   * @throws IOException
+   */
+  private void createPath(HiveConf conf, Path path, String permission, boolean isLocal,
+      boolean isCleanUp) throws IOException {
+    FsPermission fsPermission = new FsPermission(permission);
+    FileSystem fs;
+    if (isLocal) {
+      fs = FileSystem.getLocal(conf);
+    } else {
+      fs = path.getFileSystem(conf);
+    }
+    if (!fs.exists(path)) {
+      fs.mkdirs(path, fsPermission);
+      String dirType = isLocal ? "local" : "HDFS";
+      LOG.info("Created " + dirType + " directory: " + path.toString());
+    }
+    if (isCleanUp) {
+      fs.deleteOnExit(path);
+    }
+  }
+
+  public String getHdfsScratchDirURIString() {
+    return hdfsScratchDirURIString;
+  }
+
   public static Path getLocalSessionPath(Configuration conf) {
     SessionState ss = SessionState.get();
     if (ss == null) {
@@ -448,43 +535,6 @@ public class SessionState {
     }
   }
 
-  private void createSessionPaths(Configuration conf) throws IOException {
-
-    String scratchDirPermission = HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIRPERMISSION);
-    String sessionId = getSessionId();
-
-    // local & non-local tmp location is configurable. however it is the same across
-    // all external file systems
-    hdfsSessionPath =
-      new Path(HiveConf.getVar(conf, HiveConf.ConfVars.SCRATCHDIR),
-               sessionId);
-    createPath(conf, hdfsSessionPath, scratchDirPermission);
-    conf.set(HDFS_SESSION_PATH_KEY, hdfsSessionPath.toUri().toString());
-
-    localSessionPath = new Path(HiveConf.getVar(conf, HiveConf.ConfVars.LOCALSCRATCHDIR),
-                                sessionId);
-    createPath(conf, localSessionPath, scratchDirPermission);
-    conf.set(LOCAL_SESSION_PATH_KEY, localSessionPath.toUri().toString());
-    hdfsTmpTableSpace = new Path(hdfsSessionPath, TMP_PREFIX);
-    createPath(conf, hdfsTmpTableSpace, scratchDirPermission);
-    conf.set(TMP_TABLE_SPACE_KEY, hdfsTmpTableSpace.toUri().toString());
-  }
-
-  private void createPath(Configuration conf, Path p, String perm) throws IOException {
-    FileSystem fs = p.getFileSystem(conf);
-    p = new Path(fs.makeQualified(p).toString());
-    FsPermission fsPermission = new FsPermission(Short.parseShort(perm.trim(), 8));
-
-    if (!Utilities.createDirsWithPermission(conf, p, fsPermission)) {
-      throw new IOException("Cannot create directory: "
-                            + p.toString());
-    }
-
-    // best effort to clean up if we don't shut down properly
-    fs.deleteOnExit(p);
-  }
-
-
   /**
    * Setup authentication and authorization plugins for this session.
    */

Modified: hive/trunk/ql/src/test/org/apache/hadoop/hive/ql/exec/TestUtilities.java
URL: http://svn.apache.org/viewvc/hive/trunk/ql/src/test/org/apache/hadoop/hive/ql/exec/TestUtilities.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/ql/src/test/org/apache/hadoop/hive/ql/exec/TestUtilities.java (original)
+++ hive/trunk/ql/src/test/org/apache/hadoop/hive/ql/exec/TestUtilities.java Thu Sep  4 20:25:49 2014
@@ -20,15 +20,12 @@ package org.apache.hadoop.hive.ql.exec;
 
 import static org.apache.hadoop.hive.ql.exec.Utilities.getFileExtension;
 
-import java.io.IOException;
 import java.sql.Timestamp;
 import java.util.ArrayList;
 import java.util.List;
 
 import junit.framework.TestCase;
 
-import org.apache.hadoop.fs.Path;
-import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.ql.io.HiveIgnoreKeyTextOutputFormat;
 import org.apache.hadoop.hive.ql.metadata.HiveException;
@@ -39,7 +36,6 @@ import org.apache.hadoop.hive.ql.session
 import org.apache.hadoop.hive.ql.udf.generic.GenericUDFFromUtcTimestamp;
 import org.apache.hadoop.hive.serde2.typeinfo.TypeInfoFactory;
 import org.apache.hadoop.mapred.JobConf;
-import org.junit.Test;
 
 public class TestUtilities extends TestCase {
 
@@ -77,9 +73,9 @@ public class TestUtilities extends TestC
     List<ExprNodeDesc> children = new ArrayList<ExprNodeDesc>(1);
     children.add(constant);
     ExprNodeGenericFuncDesc desc = new ExprNodeGenericFuncDesc(TypeInfoFactory.timestampTypeInfo,
-      new GenericUDFFromUtcTimestamp(), children);
+        new GenericUDFFromUtcTimestamp(), children);
     assertEquals(desc.getExprString(), Utilities.deserializeExpression(
-      Utilities.serializeExpression(desc)).getExprString());
+        Utilities.serializeExpression(desc)).getExprString());
   }
 
   public void testgetDbTableName() throws HiveException{
@@ -109,23 +105,4 @@ public class TestUtilities extends TestC
       assertEquals("Invalid table name " + tablename, ex.getMessage());
     }
   }
-
-  @Test
-  public void testFSUmaskReset() throws Exception {
-    // ensure that FS Umask is not reset (HIVE-7001)
-    checkFSUMaskReset(true);
-    checkFSUMaskReset(false);
-  }
-
-  private void checkFSUMaskReset(boolean recursiveArg) throws IllegalArgumentException, IOException {
-    final String FS_MASK_VAL = "055";
-    HiveConf conf = new HiveConf();
-    String dir = System.getProperty("test.tmp.dir") + "/testUtilitiesUMaskReset";
-    conf.set(FsPermission.UMASK_LABEL, FS_MASK_VAL);
-    Utilities.createDirsWithPermission(conf, new Path(dir), new FsPermission((short) 00777),
-        recursiveArg);
-    assertEquals(conf.get(FsPermission.UMASK_LABEL), FS_MASK_VAL);
-  }
-
-
 }

Modified: hive/trunk/service/src/java/org/apache/hive/service/cli/CLIService.java
URL: http://svn.apache.org/viewvc/hive/trunk/service/src/java/org/apache/hive/service/cli/CLIService.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/service/src/java/org/apache/hive/service/cli/CLIService.java (original)
+++ hive/trunk/service/src/java/org/apache/hive/service/cli/CLIService.java Thu Sep  4 20:25:49 2014
@@ -30,12 +30,8 @@ import javax.security.auth.login.LoginEx
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
-import org.apache.hadoop.fs.FileSystem;
-import org.apache.hadoop.fs.Path;
-import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
-import org.apache.hadoop.hive.conf.SystemVariables;
 import org.apache.hadoop.hive.metastore.HiveMetaStoreClient;
 import org.apache.hadoop.hive.metastore.IMetaStoreClient;
 import org.apache.hadoop.hive.ql.metadata.Hive;
@@ -118,15 +114,6 @@ public class CLIService extends Composit
   @Override
   public synchronized void start() {
     super.start();
-
-    try {
-      // make sure that the base scratch directories exists and writable
-      setupStagingDir(hiveConf.getVar(HiveConf.ConfVars.SCRATCHDIR), false);
-      setupStagingDir(hiveConf.getVar(HiveConf.ConfVars.LOCALSCRATCHDIR), true);
-      setupStagingDir(hiveConf.getVar(HiveConf.ConfVars.DOWNLOADED_RESOURCES_DIR), true);
-    } catch (IOException eIO) {
-      throw new ServiceException("Error setting stage directories", eIO);
-    }
     // Initialize and test a connection to the metastore
     IMetaStoreClient metastoreClient = null;
     try {
@@ -460,25 +447,6 @@ public class CLIService extends Composit
     }
   }
 
-  // create the give Path if doesn't exists and make it writable
-  private void setupStagingDir(String dirPath, boolean isLocal) throws IOException {
-    Path scratchDir = getStaticPath(new Path(dirPath));
-    if (scratchDir == null) {
-      return;
-    }
-    FileSystem fs;
-    if (isLocal) {
-      fs = FileSystem.getLocal(hiveConf);
-    } else {
-      fs = scratchDir.getFileSystem(hiveConf);
-    }
-    if (!fs.exists(scratchDir)) {
-      fs.mkdirs(scratchDir);
-    }
-    FsPermission fsPermission = new FsPermission((short)0777);
-    fs.setPermission(scratchDir, fsPermission);
-  }
-
   @Override
   public String getDelegationToken(SessionHandle sessionHandle, HiveAuthFactory authFactory,
       String owner, String renewer) throws HiveSQLException {
@@ -502,16 +470,4 @@ public class CLIService extends Composit
     sessionManager.getSession(sessionHandle).renewDelegationToken(authFactory, tokenStr);
     LOG.info(sessionHandle  + ": renewDelegationToken()");
   }
-
-  // DOWNLOADED_RESOURCES_DIR for example, which is by default ${system:java.io.tmpdir}/${hive.session.id}_resources,
-  // {system:java.io.tmpdir} would be already evaluated but ${hive.session.id} would be not in here.
-  // for that case, this returns evaluatd parts only, in this case, "/tmp"
-  // what for ${hive.session.id}_resources/${system:java.io.tmpdir}? just don't do that.
-  private Path getStaticPath(Path path) {
-    Path current = path;
-    for (; current != null && SystemVariables.containsVar(current.getName());
-        current = current.getParent()) {
-    }
-    return current;
-  }
 }

Modified: hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionBase.java
URL: http://svn.apache.org/viewvc/hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionBase.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionBase.java (original)
+++ hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionBase.java Thu Sep  4 20:25:49 2014
@@ -18,6 +18,8 @@
 
 package org.apache.hive.service.cli.session;
 
+import java.util.Map;
+
 import org.apache.hadoop.hive.conf.HiveConf;
 import org.apache.hadoop.hive.ql.session.SessionState;
 import org.apache.hive.service.cli.SessionHandle;

Modified: hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionImpl.java
URL: http://svn.apache.org/viewvc/hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionImpl.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionImpl.java (original)
+++ hive/trunk/service/src/java/org/apache/hive/service/cli/session/HiveSessionImpl.java Thu Sep  4 20:25:49 2014
@@ -69,7 +69,7 @@ public class HiveSessionImpl implements 
 
   private String username;
   private final String password;
-  private final HiveConf hiveConf;
+  private HiveConf hiveConf;
   private final SessionState sessionState;
   private String ipAddress;
 
@@ -240,6 +240,12 @@ public class HiveSessionImpl implements 
   }
 
   @Override
+  /**
+   * Opens a new HiveServer2 session for the client connection.
+   * Note that if doAs is true, this call goes through a proxy object,
+   * which wraps the method logic in a UserGroupInformation#doAs.
+   * That is why it is important to call SessionState#start here rather than the constructor.
+   */
   public void open() {
     SessionState.start(sessionState);
   }

Modified: hive/trunk/service/src/java/org/apache/hive/service/cli/session/SessionManager.java
URL: http://svn.apache.org/viewvc/hive/trunk/service/src/java/org/apache/hive/service/cli/session/SessionManager.java?rev=1622556&r1=1622555&r2=1622556&view=diff
==============================================================================
--- hive/trunk/service/src/java/org/apache/hive/service/cli/session/SessionManager.java (original)
+++ hive/trunk/service/src/java/org/apache/hive/service/cli/session/SessionManager.java Thu Sep  4 20:25:49 2014
@@ -233,11 +233,13 @@ public class SessionManager extends Comp
       Map<String, String> sessionConf, boolean withImpersonation, String delegationToken)
           throws HiveSQLException {
     HiveSession session;
+    // If doAs is set to true for HiveServer2, we will create a proxy object for the session impl.
+    // Within the proxy object, we wrap the method call in a UserGroupInformation#doAs
     if (withImpersonation) {
-      HiveSessionImplwithUGI hiveSessionUgi = new HiveSessionImplwithUGI(protocol, username, password,
-        hiveConf, ipAddress, delegationToken);
-      session = HiveSessionProxy.getProxy(hiveSessionUgi, hiveSessionUgi.getSessionUgi());
-      hiveSessionUgi.setProxySession(session);
+      HiveSessionImplwithUGI sessionWithUGI = new HiveSessionImplwithUGI(protocol, username, password,
+          hiveConf, ipAddress, delegationToken);
+      session = HiveSessionProxy.getProxy(sessionWithUGI, sessionWithUGI.getSessionUgi());
+      sessionWithUGI.setProxySession(session);
     } else {
       session = new HiveSessionImpl(protocol, username, password, hiveConf, ipAddress);
     }