You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@zeppelin.apache.org by mo...@apache.org on 2016/11/24 17:17:08 UTC

[2/2] zeppelin git commit: [ZEPPELIN-1567] Let JDBC interpreter use user credential information.

[ZEPPELIN-1567] Let JDBC interpreter use user credential information.

### What is this PR for?

This PR is for the multi-tenant of JDBC Interpreter.

User can create a user/password for JDBC account at the [Credential page](http://zeppelin.apache.org/docs/0.7.0-SNAPSHOT/security/datasource_authorization.html).
The `Entity` of `Credential` is match with JDBC interpreter group name.

If the account for JDBC is not setted in the `Interpreter property` then use `Credential`'s.
### What type of PR is it?

Improvement
### What is the Jira issue?

https://issues.apache.org/jira/browse/ZEPPELIN-1567
### How should this be tested?

Please refer to testMultiTenant() of JDBCInterpreterTest/
### Screenshots (if appropriate)
### Questions:
- Does the licenses files need update? no
- Is there breaking changes for older versions? no
- Does this needs documentation? no

Author: astroshim <hs...@nflabs.com>

Closes #1539 from astroshim/jdbc-impersonation and squashes the following commits:

46fce31 [astroshim] add explanation of InterpreterGroup
7a92236 [astroshim] fix doc and remove persist value.
63f5ea7 [astroshim] Merge branch 'master' into jdbc-impersonation
267277a [astroshim] rebase
649ff6e [astroshim] rebase
872fb49 [astroshim] fix ScioInterpreterTestCase
4387a5b [astroshim] Merge branch 'master' into jdbc-impersonation
47c463f [astroshim] update doc and html
d4eb178 [astroshim] fix docs
59aa9ff [astroshim] Merge branch 'master' into jdbc-impersonation
bf61afd [astroshim] fix testcase
5c0f5d7 [astroshim] rebase
79ba25b [astroshim] Merge branch 'master' into jdbc-impersonation
1f9c2c0 [astroshim] clean redundant code
a2f5687 [astroshim] fix impersonation
9962181 [astroshim] fix InterpreterOutput of PySparkInterpreterTest case
b55aceb [astroshim] Merge branch 'master' into jdbc-impersonation
24a8226 [astroshim] fix doc
086dfda [astroshim] fix testcase
34fe0a6 [astroshim] fix code for more simple.
fee7086 [astroshim] fix build error.
a305eca [astroshim] Merge branch 'master' into jdbc-impersonation
df80741 [astroshim] documentation for credential.
df1b1dc [astroshim] rebase and entity name convention.
63d6a1c [astroshim] change thrift version to 0.9.2
6573c1c [astroshim] change variable name
f311f34 [astroshim] fix typo
722e333 [astroshim] change testcase name
9161937 [astroshim] clean code
3dafdf0 [astroshim] add testcase
373d5f1 [astroshim] pass replName to Interpreter and use credential info for jdbc auth.


Project: http://git-wip-us.apache.org/repos/asf/zeppelin/repo
Commit: http://git-wip-us.apache.org/repos/asf/zeppelin/commit/b7307d49
Tree: http://git-wip-us.apache.org/repos/asf/zeppelin/tree/b7307d49
Diff: http://git-wip-us.apache.org/repos/asf/zeppelin/diff/b7307d49

Branch: refs/heads/master
Commit: b7307d49def37d7a648baa170b26b930ca8aa14b
Parents: caa664d
Author: astroshim <hs...@nflabs.com>
Authored: Thu Nov 24 20:02:17 2016 +0900
Committer: Lee moon soo <mo...@apache.org>
Committed: Thu Nov 24 09:17:01 2016 -0800

----------------------------------------------------------------------
 .../zeppelin/beam/BeamInterpreterTest.java      |   2 +-
 .../zeppelin/img/docs-img/add_credential.png    | Bin 167834 -> 56795 bytes
 docs/interpreter/jdbc.md                        |   6 +-
 docs/security/datasource_authorization.md       |   5 +-
 .../zeppelin/flink/FlinkInterpreterTest.java    |   2 +-
 .../zeppelin/ignite/IgniteInterpreterTest.java  |   2 +-
 .../ignite/IgniteSqlInterpreterTest.java        |   2 +-
 .../apache/zeppelin/jdbc/JDBCInterpreter.java   | 336 +++++++++++--------
 .../zeppelin/jdbc/JDBCUserConfigurations.java   | 100 ++++++
 .../zeppelin/jdbc/JDBCInterpreterTest.java      | 104 +++++-
 .../zeppelin/livy/LivyIntegrationTest.java      |   4 +-
 .../apache/zeppelin/pig/PigInterpreterTest.java |   2 +-
 .../zeppelin/pig/PigQueryInterpreterTest.java   |   2 +-
 .../python/PythonCondaInterpreterTest.java      |   1 +
 .../python/PythonInterpreterMatplotlibTest.java |   2 +-
 .../python/PythonInterpreterPandasSqlTest.java  |   2 +-
 .../scalding/ScaldingInterpreterTest.java       |   2 +-
 .../zeppelin/scio/ScioInterpreterTest.java      |   2 +-
 .../zeppelin/shell/ShellInterpreterTest.java    |   4 +-
 .../zeppelin/spark/DepInterpreterTest.java      |   2 +-
 .../zeppelin/spark/PySparkInterpreterTest.java  |   2 +-
 .../zeppelin/spark/SparkInterpreterTest.java    |   2 +-
 .../zeppelin/spark/SparkSqlInterpreterTest.java |   2 +-
 .../angular/AbstractAngularElemTest.scala       |   2 +-
 .../angular/AbstractAngularModelTest.scala      |   2 +-
 .../interpreter/InterpreterContext.java         |  12 +-
 .../interpreter/remote/RemoteInterpreter.java   |   1 +
 .../remote/RemoteInterpreterServer.java         |   1 +
 .../thrift/InterpreterCompletion.java           |  19 +-
 .../thrift/RemoteApplicationResult.java         |  19 +-
 .../thrift/RemoteInterpreterContext.java        | 198 ++++++++---
 .../thrift/RemoteInterpreterEvent.java          |  19 +-
 .../thrift/RemoteInterpreterEventType.java      |  17 -
 .../thrift/RemoteInterpreterResult.java         |  19 +-
 .../thrift/RemoteInterpreterService.java        |  19 +-
 .../main/thrift/RemoteInterpreterService.thrift |  14 +-
 .../interpreter/InterpreterContextTest.java     |   2 +-
 .../interpreter/LazyOpenInterpreterTest.java    |   2 +-
 .../remote/RemoteAngularObjectTest.java         |   1 +
 .../RemoteInterpreterOutputTestStream.java      |   1 +
 .../remote/RemoteInterpreterTest.java           |  10 +
 .../resource/DistributedResourcePoolTest.java   |   1 +
 .../zeppelin/scheduler/RemoteSchedulerTest.java |   3 +
 .../src/app/credential/credential.controller.js |  29 ++
 zeppelin-web/src/app/credential/credential.html |   4 +-
 .../org/apache/zeppelin/notebook/Paragraph.java |   1 +
 .../interpreter/InterpreterFactoryTest.java     |   2 +-
 47 files changed, 647 insertions(+), 339 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/beam/src/main/test/org/apache/zeppelin/beam/BeamInterpreterTest.java
----------------------------------------------------------------------
diff --git a/beam/src/main/test/org/apache/zeppelin/beam/BeamInterpreterTest.java b/beam/src/main/test/org/apache/zeppelin/beam/BeamInterpreterTest.java
index c24ed41..1fef7a6 100644
--- a/beam/src/main/test/org/apache/zeppelin/beam/BeamInterpreterTest.java
+++ b/beam/src/main/test/org/apache/zeppelin/beam/BeamInterpreterTest.java
@@ -43,7 +43,7 @@ public class BeamInterpreterTest {
     Properties p = new Properties();
     beam = new BeamInterpreter(p);
     beam.open();
-    context = new InterpreterContext(null, null, null, null, null, null, null, null, null, null,
+    context = new InterpreterContext(null, null, null, null, null, null, null, null, null, null, null,
         null);
   }
 

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/docs/assets/themes/zeppelin/img/docs-img/add_credential.png
----------------------------------------------------------------------
diff --git a/docs/assets/themes/zeppelin/img/docs-img/add_credential.png b/docs/assets/themes/zeppelin/img/docs-img/add_credential.png
index dcf4460..102b3ec 100644
Binary files a/docs/assets/themes/zeppelin/img/docs-img/add_credential.png and b/docs/assets/themes/zeppelin/img/docs-img/add_credential.png differ

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/docs/interpreter/jdbc.md
----------------------------------------------------------------------
diff --git a/docs/interpreter/jdbc.md b/docs/interpreter/jdbc.md
index a10ea0b..e7bd4fe 100644
--- a/docs/interpreter/jdbc.md
+++ b/docs/interpreter/jdbc.md
@@ -116,7 +116,11 @@ The JDBC interpreter properties are defined by default like below.
   </tr>
 </table>
 
-If you want to connect other databases such as `Mysql`, `Redshift` and `Hive`, you need to edit the property values. 
+If you want to connect other databases such as `Mysql`, `Redshift` and `Hive`, you need to edit the property values.
+You can also use [Credential](../security/datasource_authorization.html) for JDBC authentication.
+If `default.user` and `default.password` properties are deleted(using X button) for database connection in the interpreter setting page,
+the JDBC interpreter will get the account information from [Credential](../security/datasource_authorization.html).
+
 The below example is for `Mysql` connection.
 
 <img src="../assets/themes/zeppelin/img/docs-img/edit_properties.png" width="600px" />

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/docs/security/datasource_authorization.md
----------------------------------------------------------------------
diff --git a/docs/security/datasource_authorization.md b/docs/security/datasource_authorization.md
index 60bf086..03165c8 100644
--- a/docs/security/datasource_authorization.md
+++ b/docs/security/datasource_authorization.md
@@ -37,7 +37,10 @@ You can add new credentials in the dropdown menu for your data source which can
 
 <img class="img-responsive" src="../assets/themes/zeppelin/img/docs-img/credential_tab.png" width="180px"/>
 
-**Entity** can be the key that distinguishes each credential sets. Type **Username & Password** for your own credentials. ex) user & password of Mysql 
+**Entity** can be the key that distinguishes each credential sets.(We suggest that the convention of the **Entity** is `[Interpreter Group].[Interpreter Name]`.)
+Please see [what is interpreter group](../manual/interpreters.html#what-is-interpreter-group) for the detailed information.
+
+Type **Username & Password** for your own credentials. ex) Mysql user & password of the JDBC Interpreter.
 
 <img class="img-responsive" src="../assets/themes/zeppelin/img/docs-img/add_credential.png" />
 

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/flink/src/test/java/org/apache/zeppelin/flink/FlinkInterpreterTest.java
----------------------------------------------------------------------
diff --git a/flink/src/test/java/org/apache/zeppelin/flink/FlinkInterpreterTest.java b/flink/src/test/java/org/apache/zeppelin/flink/FlinkInterpreterTest.java
index 1d8f437..6657a1e 100644
--- a/flink/src/test/java/org/apache/zeppelin/flink/FlinkInterpreterTest.java
+++ b/flink/src/test/java/org/apache/zeppelin/flink/FlinkInterpreterTest.java
@@ -40,7 +40,7 @@ public class FlinkInterpreterTest {
     Properties p = new Properties();
     flink = new FlinkInterpreter(p);
     flink.open();
-    context = new InterpreterContext(null, null, null, null, null, null, null, null, null, null, null);
+    context = new InterpreterContext(null, null, null, null, null, null, null, null, null, null, null, null);
   }
 
   @AfterClass

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteInterpreterTest.java
----------------------------------------------------------------------
diff --git a/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteInterpreterTest.java b/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteInterpreterTest.java
index f151763..78f86ae 100644
--- a/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteInterpreterTest.java
+++ b/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteInterpreterTest.java
@@ -40,7 +40,7 @@ public class IgniteInterpreterTest {
   private static final String HOST = "127.0.0.1:47500..47509";
 
   private static final InterpreterContext INTP_CONTEXT =
-      new InterpreterContext(null, null, null, null, null, null, null, null, null, null, null);
+      new InterpreterContext(null, null, null, null, null, null, null, null, null, null, null, null);
 
   private IgniteInterpreter intp;
   private Ignite ignite;

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteSqlInterpreterTest.java
----------------------------------------------------------------------
diff --git a/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteSqlInterpreterTest.java b/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteSqlInterpreterTest.java
index 9076c36..07f11f8 100644
--- a/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteSqlInterpreterTest.java
+++ b/ignite/src/test/java/org/apache/zeppelin/ignite/IgniteSqlInterpreterTest.java
@@ -44,7 +44,7 @@ public class IgniteSqlInterpreterTest {
   private static final String HOST = "127.0.0.1:47500..47509";
 
   private static final InterpreterContext INTP_CONTEXT =
-      new InterpreterContext(null, null, null, null, null, null, null, null, null, null, null);
+      new InterpreterContext(null, null, null, null, null, null, null, null, null, null, null, null);
 
   private Ignite ignite;
   private IgniteSqlInterpreter intp;

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCInterpreter.java
----------------------------------------------------------------------
diff --git a/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCInterpreter.java b/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCInterpreter.java
index 0fbbda3..7871f01 100644
--- a/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCInterpreter.java
+++ b/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCInterpreter.java
@@ -39,6 +39,8 @@ import org.apache.zeppelin.interpreter.thrift.InterpreterCompletion;
 import org.apache.zeppelin.jdbc.security.JDBCSecurityImpl;
 import org.apache.zeppelin.scheduler.Scheduler;
 import org.apache.zeppelin.scheduler.SchedulerFactory;
+import org.apache.zeppelin.user.UserCredentials;
+import org.apache.zeppelin.user.UsernamePassword;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -73,6 +75,9 @@ public class JDBCInterpreter extends Interpreter {
 
   private Logger logger = LoggerFactory.getLogger(JDBCInterpreter.class);
 
+  static final String INTERPRETER_NAME = "jdbc";
+  static final String JDBC_DEFAULT_USER_KEY = "default.user";
+  static final String JDBC_DEFAULT_PASSWORD_KEY = "default.password";
   static final String COMMON_KEY = "common";
   static final String MAX_LINE_KEY = "max_count";
   static final String MAX_LINE_DEFAULT = "1000";
@@ -100,16 +105,12 @@ public class JDBCInterpreter extends Interpreter {
 
   static final String EMPTY_COLUMN_VALUE = "";
 
-
   private final String CONCURRENT_EXECUTION_KEY = "zeppelin.jdbc.concurrent.use";
   private final String CONCURRENT_EXECUTION_COUNT = "zeppelin.jdbc.concurrent.max_connection";
-
   private final String DBCP_STRING = "jdbc:apache:commons:dbcp:";
 
-  private final HashMap<String, Properties> propertiesMap;
-  private final Map<String, Statement> paragraphIdStatementMap;
-  private final Map<String, PoolingDriver> poolingDriverMap;
-
+  private final HashMap<String, Properties> basePropretiesMap;
+  private final HashMap<String, JDBCUserConfigurations> jdbcUserConfigurationsMap;
   private final Map<String, SqlCompleter> propertyKeySqlCompleterMap;
 
   private static final Function<CharSequence, InterpreterCompletion> sequenceToStringTransformer =
@@ -123,14 +124,13 @@ public class JDBCInterpreter extends Interpreter {
 
   public JDBCInterpreter(Properties property) {
     super(property);
-    propertiesMap = new HashMap<>();
-    paragraphIdStatementMap = new HashMap<>();
-    poolingDriverMap = new HashMap<>();
+    jdbcUserConfigurationsMap = new HashMap<>();
     propertyKeySqlCompleterMap = new HashMap<>();
+    basePropretiesMap = new HashMap<>();
   }
 
   public HashMap<String, Properties> getPropertiesMap() {
-    return propertiesMap;
+    return basePropretiesMap;
   }
 
   @Override
@@ -140,21 +140,22 @@ public class JDBCInterpreter extends Interpreter {
       String[] keyValue = propertyKey.split("\\.", 2);
       if (2 == keyValue.length) {
         logger.info("key: {}, value: {}", keyValue[0], keyValue[1]);
+
         Properties prefixProperties;
-        if (propertiesMap.containsKey(keyValue[0])) {
-          prefixProperties = propertiesMap.get(keyValue[0]);
+        if (basePropretiesMap.containsKey(keyValue[0])) {
+          prefixProperties = basePropretiesMap.get(keyValue[0]);
         } else {
           prefixProperties = new Properties();
-          propertiesMap.put(keyValue[0], prefixProperties);
+          basePropretiesMap.put(keyValue[0], prefixProperties);
         }
         prefixProperties.put(keyValue[1], property.getProperty(propertyKey));
       }
     }
 
     Set<String> removeKeySet = new HashSet<>();
-    for (String key : propertiesMap.keySet()) {
+    for (String key : basePropretiesMap.keySet()) {
       if (!COMMON_KEY.equals(key)) {
-        Properties properties = propertiesMap.get(key);
+        Properties properties = basePropretiesMap.get(key);
         if (!properties.containsKey(DRIVER_KEY) || !properties.containsKey(URL_KEY)) {
           logger.error("{} will be ignored. {}.{} and {}.{} is mandatory.",
               key, DRIVER_KEY, key, key, URL_KEY);
@@ -164,15 +165,14 @@ public class JDBCInterpreter extends Interpreter {
     }
 
     for (String key : removeKeySet) {
-      propertiesMap.remove(key);
+      basePropretiesMap.remove(key);
     }
-
-    logger.debug("propertiesMap: {}", propertiesMap);
+    logger.debug("JDBC PropretiesMap: {}", basePropretiesMap);
 
     if (!StringUtils.isEmpty(property.getProperty("zeppelin.jdbc.auth.type"))) {
       JDBCSecurityImpl.createSecureConfiguration(property);
     }
-    for (String propertyKey : propertiesMap.keySet()) {
+    for (String propertyKey : basePropretiesMap.keySet()) {
       propertyKeySqlCompleterMap.put(propertyKey, createSqlCompleter(null));
     }
   }
@@ -194,12 +194,110 @@ public class JDBCInterpreter extends Interpreter {
     return completer;
   }
 
-  private boolean isConnectionInPool(String driverName) {
-    if (poolingDriverMap.containsKey(driverName)) return true;
-    return false;
+  private void initStatementMap() {
+    for (JDBCUserConfigurations configurations : jdbcUserConfigurationsMap.values()) {
+      try {
+        configurations.initStatementMap();
+      } catch (Exception e) {
+        logger.error("Error while closing paragraphIdStatementMap statement...", e);
+      }
+    }
+  }
+
+  private void initConnectionPoolMap() {
+    for (JDBCUserConfigurations configurations : jdbcUserConfigurationsMap.values()) {
+      try {
+        configurations.initConnectionPoolMap();
+      } catch (Exception e) {
+        logger.error("Error while closing initConnectionPoolMap...", e);
+      }
+    }
+  }
+
+  @Override
+  public void close() {
+    try {
+      initStatementMap();
+      initConnectionPoolMap();
+    } catch (Exception e) {
+      logger.error("Error while closing...", e);
+    }
+  }
+
+  private String getEntityName(String replName) {
+    StringBuffer entityName = new StringBuffer();
+    entityName.append(INTERPRETER_NAME);
+    entityName.append(".");
+    entityName.append(replName);
+    return entityName.toString();
+  }
+
+  private String getJDBCDriverName(String user, String propertyKey) {
+    StringBuffer driverName = new StringBuffer();
+    driverName.append(DBCP_STRING);
+    driverName.append(propertyKey);
+    driverName.append(user);
+    return driverName.toString();
+  }
+
+  private boolean existAccountInBaseProperty() {
+    return property.containsKey(JDBC_DEFAULT_USER_KEY) &&
+      property.containsKey(JDBC_DEFAULT_PASSWORD_KEY);
+  }
+
+  private UsernamePassword getUsernamePassword(InterpreterContext interpreterContext,
+                                               String replName) {
+    UserCredentials uc = interpreterContext.getAuthenticationInfo().getUserCredentials();
+    if (uc != null) {
+      return uc.getUsernamePassword(replName);
+    }
+    return null;
+  }
+
+  public JDBCUserConfigurations getJDBCConfiguration(String user) {
+    JDBCUserConfigurations jdbcUserConfigurations =
+      jdbcUserConfigurationsMap.get(user);
+
+    if (jdbcUserConfigurations == null) {
+      jdbcUserConfigurations = new JDBCUserConfigurations();
+      jdbcUserConfigurationsMap.put(user, jdbcUserConfigurations);
+    }
+
+    return jdbcUserConfigurations;
+  }
+
+  private void closeDBPool(String user, String propertyKey) throws SQLException {
+    PoolingDriver poolingDriver = getJDBCConfiguration(user).removeDBDriverPool(propertyKey);
+    if (poolingDriver != null) {
+      poolingDriver.closePool(propertyKey + user);
+    }
+  }
+
+  private void setUserProperty(String propertyKey, InterpreterContext interpreterContext)
+      throws SQLException {
+
+    String user = interpreterContext.getAuthenticationInfo().getUser();
+
+    JDBCUserConfigurations jdbcUserConfigurations =
+      getJDBCConfiguration(user);
+    jdbcUserConfigurations.setPropertyMap(propertyKey, basePropretiesMap.get(propertyKey));
+
+    if (existAccountInBaseProperty()) {
+      return;
+    }
+    jdbcUserConfigurations.cleanUserProperty(propertyKey);
+
+    UsernamePassword usernamePassword = getUsernamePassword(interpreterContext,
+      getEntityName(interpreterContext.getReplName()));
+    if (usernamePassword != null) {
+      jdbcUserConfigurations.setUserProperty(propertyKey, usernamePassword);
+    } else {
+      closeDBPool(user, propertyKey);
+    }
   }
 
-  private void createConnectionPool(String url, String propertyKey, Properties properties) {
+  private void createConnectionPool(String url, String user, String propertyKey,
+      Properties properties) throws SQLException, ClassNotFoundException {
     ConnectionFactory connectionFactory =
       new DriverManagerConnectionFactory(url, properties);
 
@@ -208,135 +306,100 @@ public class JDBCInterpreter extends Interpreter {
     ObjectPool connectionPool = new GenericObjectPool(poolableConnectionFactory);
 
     poolableConnectionFactory.setPool(connectionPool);
+    Class.forName(properties.getProperty(DRIVER_KEY));
     PoolingDriver driver = new PoolingDriver();
-    driver.registerPool(propertyKey, connectionPool);
-
-    poolingDriverMap.put(propertyKey, driver);
+    driver.registerPool(propertyKey + user, connectionPool);
+    getJDBCConfiguration(user).saveDBDriverPool(propertyKey, driver);
   }
 
-  private Connection getConnectionFromPool(String url, String propertyKey, Properties properties)
-      throws SQLException {
-    if (!isConnectionInPool(propertyKey)) {
-      createConnectionPool(url, propertyKey, properties);
-    }
+  private Connection getConnectionFromPool(String url, String user, String propertyKey,
+      Properties properties) throws SQLException, ClassNotFoundException {
+    String jdbcDriver = getJDBCDriverName(user, propertyKey);
 
-    return DriverManager.getConnection(DBCP_STRING + propertyKey);
+    if (!getJDBCConfiguration(user).isConnectionInDBDriverPool(propertyKey)) {
+      createConnectionPool(url, user, propertyKey, properties);
+    }
+    return DriverManager.getConnection(jdbcDriver);
   }
 
-  public Connection getConnection(String propertyKey, String user)
+  public Connection getConnection(String propertyKey, InterpreterContext interpreterContext)
       throws ClassNotFoundException, SQLException, InterpreterException {
-    Connection connection = null;
-    if (propertyKey == null || propertiesMap.get(propertyKey) == null) {
+    final String user =  interpreterContext.getAuthenticationInfo().getUser();
+    Connection connection;
+    if (propertyKey == null || basePropretiesMap.get(propertyKey) == null) {
       return null;
     }
-    if (null == connection) {
-      final Properties properties = (Properties) propertiesMap.get(propertyKey).clone();
-      logger.info(properties.getProperty(DRIVER_KEY));
-      Class.forName(properties.getProperty(DRIVER_KEY));
-      final String url = properties.getProperty(URL_KEY);
-
-      if (StringUtils.isEmpty(property.getProperty("zeppelin.jdbc.auth.type"))) {
-        connection = DriverManager.getConnection(url, properties);
-      } else {
-        UserGroupInformation.AuthenticationMethod authType = JDBCSecurityImpl.getAuthtype(property);
-        switch (authType) {
-            case KERBEROS:
-              if (user == null) {
-                connection = getConnectionFromPool(url, propertyKey, properties);
+
+    JDBCUserConfigurations jdbcUserConfigurations = getJDBCConfiguration(user);
+    setUserProperty(propertyKey, interpreterContext);
+
+    final Properties properties = jdbcUserConfigurations.getPropertyMap(propertyKey);
+    final String url = properties.getProperty(URL_KEY);
+
+    if (StringUtils.isEmpty(property.getProperty("zeppelin.jdbc.auth.type"))) {
+      connection = getConnectionFromPool(url, user, propertyKey, properties);
+    } else {
+      UserGroupInformation.AuthenticationMethod authType = JDBCSecurityImpl.getAuthtype(property);
+
+      switch (authType) {
+          case KERBEROS:
+            if (user == null) {
+              connection = getConnectionFromPool(url, user, propertyKey, properties);
+            } else {
+              if ("hive".equalsIgnoreCase(propertyKey)) {
+                connection = getConnectionFromPool(url + ";hive.server2.proxy.user=" + user,
+                  user, propertyKey, properties);
               } else {
-                if ("hive".equalsIgnoreCase(propertyKey)) {
-                  connection = getConnectionFromPool(url + ";hive.server2.proxy.user=" + user,
-                    propertyKey, properties);
-                } else {
-                  UserGroupInformation ugi = null;
-                  try {
-                    ugi = UserGroupInformation.createProxyUser(user,
-                      UserGroupInformation.getCurrentUser());
-                  } catch (Exception e) {
-                    logger.error("Error in createProxyUser", e);
-                    StringBuilder stringBuilder = new StringBuilder();
-                    stringBuilder.append(e.getMessage()).append("\n");
-                    stringBuilder.append(e.getCause());
-                    throw new InterpreterException(stringBuilder.toString());
-                  }
-
-                  final String poolKey = propertyKey;
-                  try {
-                    connection = ugi.doAs(new PrivilegedExceptionAction<Connection>() {
-                      @Override
-                      public Connection run() throws Exception {
-                        return getConnectionFromPool(url, poolKey, properties);
-                      }
-                    });
-                  } catch (Exception e) {
-                    logger.error("Error in doAs", e);
-                    StringBuilder stringBuilder = new StringBuilder();
-                    stringBuilder.append(e.getMessage()).append("\n");
-                    stringBuilder.append(e.getCause());
-                    throw new InterpreterException(stringBuilder.toString());
-                  }
+                UserGroupInformation ugi = null;
+                try {
+                  ugi = UserGroupInformation.createProxyUser(user,
+                    UserGroupInformation.getCurrentUser());
+                } catch (Exception e) {
+                  logger.error("Error in createProxyUser", e);
+                  StringBuilder stringBuilder = new StringBuilder();
+                  stringBuilder.append(e.getMessage()).append("\n");
+                  stringBuilder.append(e.getCause());
+                  throw new InterpreterException(stringBuilder.toString());
+                }
+
+                final String poolKey = propertyKey;
+                try {
+                  connection = ugi.doAs(new PrivilegedExceptionAction<Connection>() {
+                    @Override
+                    public Connection run() throws Exception {
+                      return getConnectionFromPool(url, user, poolKey, properties);
+                    }
+                  });
+                } catch (Exception e) {
+                  logger.error("Error in doAs", e);
+                  StringBuilder stringBuilder = new StringBuilder();
+                  stringBuilder.append(e.getMessage()).append("\n");
+                  stringBuilder.append(e.getCause());
+                  throw new InterpreterException(stringBuilder.toString());
                 }
               }
-              break;
+            }
+            break;
 
-            default:
-              connection = getConnectionFromPool(url, propertyKey, properties);
-        }
+          default:
+            connection = getConnectionFromPool(url, user, propertyKey, properties);
       }
     }
     propertyKeySqlCompleterMap.put(propertyKey, createSqlCompleter(connection));
     return connection;
   }
 
-  private void initStatementMap() {
-    for (Statement statement : paragraphIdStatementMap.values()) {
-      try {
-        statement.close();
-      } catch (Exception e) {
-        logger.error("Error while closing paragraphIdStatementMap statement...", e);
-      }
-    }
-    paragraphIdStatementMap.clear();
-  }
-
-  private void initConnectionPoolMap() throws SQLException {
-    Iterator<String> it = poolingDriverMap.keySet().iterator();
-    while (it.hasNext()) {
-      String driverName = it.next();
-      poolingDriverMap.get(driverName).closePool(driverName);
-      it.remove();
-    }
-    poolingDriverMap.clear();
-  }
-
-  private void saveStatement(String key, Statement statement) throws SQLException {
-    paragraphIdStatementMap.put(key, statement);
-    statement.setMaxRows(getMaxResult());
-  }
-
-  private void removeStatement(String key) {
-    paragraphIdStatementMap.remove(key);
-  }
-
-  @Override
-  public void close() {
-    try {
-      initStatementMap();
-      initConnectionPoolMap();
-    } catch (Exception e) {
-      logger.error("Error while closing...", e);
-    }
-  }
-
   private InterpreterResult executeSql(String propertyKey, String sql,
       InterpreterContext interpreterContext) {
-    String paragraphId = interpreterContext.getParagraphId();
     Connection connection;
     Statement statement;
     ResultSet resultSet = null;
+    String paragraphId = interpreterContext.getParagraphId();
+    String user = interpreterContext.getAuthenticationInfo().getUser();
 
     try {
-      connection = getConnection(propertyKey, interpreterContext.getAuthenticationInfo().getUser());
+      connection = getConnection(propertyKey, interpreterContext);
+
       if (connection == null) {
         return new InterpreterResult(Code.ERROR, "Prefix not found.");
       }
@@ -357,8 +420,7 @@ public class JDBCInterpreter extends Interpreter {
       }
 
       try {
-        saveStatement(paragraphId +
-          interpreterContext.getAuthenticationInfo().getUser(), statement);
+        getJDBCConfiguration(user).saveStatement(paragraphId, statement);
 
         boolean isResultSetAvailable = statement.execute(sql);
 
@@ -416,8 +478,7 @@ public class JDBCInterpreter extends Interpreter {
             connection.close();
           } catch (SQLException e) { /*ignored*/ }
         }
-        removeStatement(paragraphId +
-          interpreterContext.getAuthenticationInfo().getUser());
+        getJDBCConfiguration(user).removeStatement(paragraphId);
       }
       return new InterpreterResult(Code.SUCCESS, msg.toString());
 
@@ -427,6 +488,13 @@ public class JDBCInterpreter extends Interpreter {
       PrintStream ps = new PrintStream(baos);
       e.printStackTrace(ps);
       String errorMsg = new String(baos.toByteArray(), StandardCharsets.UTF_8);
+
+      try {
+        closeDBPool(user, propertyKey);
+      } catch (SQLException e1) {
+        e1.printStackTrace();
+      }
+
       return new InterpreterResult(Code.ERROR, errorMsg);
     }
   }
@@ -458,12 +526,12 @@ public class JDBCInterpreter extends Interpreter {
 
   @Override
   public void cancel(InterpreterContext context) {
-
     logger.info("Cancel current query statement.");
-
     String paragraphId = context.getParagraphId();
+    JDBCUserConfigurations jdbcUserConfigurations =
+      getJDBCConfiguration(context.getAuthenticationInfo().getUser());
     try {
-      paragraphIdStatementMap.get(paragraphId + context.getAuthenticationInfo().getUser()).cancel();
+      jdbcUserConfigurations.cancelStatement(paragraphId);
     } catch (SQLException e) {
       logger.error("Error while cancelling...", e);
     }
@@ -520,7 +588,7 @@ public class JDBCInterpreter extends Interpreter {
 
   public int getMaxResult() {
     return Integer.valueOf(
-        propertiesMap.get(COMMON_KEY).getProperty(MAX_LINE_KEY, MAX_LINE_DEFAULT));
+        basePropretiesMap.get(COMMON_KEY).getProperty(MAX_LINE_KEY, MAX_LINE_DEFAULT));
   }
 
   boolean isConcurrentExecution() {

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCUserConfigurations.java
----------------------------------------------------------------------
diff --git a/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCUserConfigurations.java b/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCUserConfigurations.java
new file mode 100644
index 0000000..e23145b
--- /dev/null
+++ b/jdbc/src/main/java/org/apache/zeppelin/jdbc/JDBCUserConfigurations.java
@@ -0,0 +1,100 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more contributor license
+ * agreements. See the NOTICE file distributed with this work for additional information regarding
+ * copyright ownership. The ASF licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License. You may obtain a
+ * copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software distributed under the License
+ * is distributed on an "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express
+ * or implied. See the License for the specific language governing permissions and limitations under
+ * the License.
+ */
+package org.apache.zeppelin.jdbc;
+
+import org.apache.commons.dbcp2.PoolingDriver;
+import org.apache.zeppelin.user.UsernamePassword;
+
+import java.sql.SQLException;
+import java.sql.Statement;
+import java.util.HashMap;
+import java.util.Iterator;
+import java.util.Map;
+import java.util.Properties;
+
+/**
+ * UserConfigurations for JDBC impersonation.
+ */
+public class JDBCUserConfigurations {
+  private final Map<String, Statement> paragraphIdStatementMap;
+  private final Map<String, PoolingDriver> poolingDriverMap;
+  private final HashMap<String, Properties> propertiesMap;
+
+  public JDBCUserConfigurations() {
+    paragraphIdStatementMap = new HashMap<>();
+    poolingDriverMap = new HashMap<>();
+    propertiesMap = new HashMap<>();
+  }
+
+  public void initStatementMap() throws SQLException {
+    for (Statement statement : paragraphIdStatementMap.values()) {
+      statement.close();
+    }
+    paragraphIdStatementMap.clear();
+  }
+
+  public void initConnectionPoolMap() throws SQLException {
+    Iterator<String> it = poolingDriverMap.keySet().iterator();
+    while (it.hasNext()) {
+      String driverName = it.next();
+      poolingDriverMap.get(driverName).closePool(driverName);
+      it.remove();
+    }
+    poolingDriverMap.clear();
+  }
+
+  public void setPropertyMap(String key, Properties properties) {
+    Properties p = (Properties) properties.clone();
+    propertiesMap.put(key, p);
+  }
+
+  public Properties getPropertyMap(String key) {
+    return propertiesMap.get(key);
+  }
+
+  public void cleanUserProperty(String propertyKey) {
+    propertiesMap.get(propertyKey).remove("user");
+    propertiesMap.get(propertyKey).remove("password");
+  }
+
+  public void setUserProperty(String propertyKey, UsernamePassword usernamePassword) {
+    propertiesMap.get(propertyKey).setProperty("user", usernamePassword.getUsername());
+    propertiesMap.get(propertyKey).setProperty("password", usernamePassword.getPassword());
+  }
+
+  public void saveStatement(String key, Statement statement) throws SQLException {
+    paragraphIdStatementMap.put(key, statement);
+  }
+
+  public void cancelStatement(String key) throws SQLException {
+    paragraphIdStatementMap.get(key).cancel();
+  }
+
+  public void removeStatement(String key) {
+    paragraphIdStatementMap.remove(key);
+  }
+
+  public void saveDBDriverPool(String key, PoolingDriver driver) throws SQLException {
+    poolingDriverMap.put(key, driver);
+  }
+  public PoolingDriver removeDBDriverPool(String key) throws SQLException {
+    return poolingDriverMap.remove(key);
+  }
+
+  public boolean isConnectionInDBDriverPool(String key) {
+    return poolingDriverMap.containsKey(key);
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/jdbc/src/test/java/org/apache/zeppelin/jdbc/JDBCInterpreterTest.java
----------------------------------------------------------------------
diff --git a/jdbc/src/test/java/org/apache/zeppelin/jdbc/JDBCInterpreterTest.java b/jdbc/src/test/java/org/apache/zeppelin/jdbc/JDBCInterpreterTest.java
index bd5bae6..070ea19 100644
--- a/jdbc/src/test/java/org/apache/zeppelin/jdbc/JDBCInterpreterTest.java
+++ b/jdbc/src/test/java/org/apache/zeppelin/jdbc/JDBCInterpreterTest.java
@@ -16,20 +16,20 @@ package org.apache.zeppelin.jdbc;
 
 import static java.lang.String.format;
 import static org.apache.zeppelin.interpreter.Interpreter.logger;
-import static org.junit.Assert.assertEquals;
+import static org.apache.zeppelin.interpreter.Interpreter.register;
 import static org.apache.zeppelin.jdbc.JDBCInterpreter.DEFAULT_KEY;
 import static org.apache.zeppelin.jdbc.JDBCInterpreter.DEFAULT_DRIVER;
 import static org.apache.zeppelin.jdbc.JDBCInterpreter.DEFAULT_PASSWORD;
 import static org.apache.zeppelin.jdbc.JDBCInterpreter.DEFAULT_USER;
 import static org.apache.zeppelin.jdbc.JDBCInterpreter.DEFAULT_URL;
 import static org.apache.zeppelin.jdbc.JDBCInterpreter.COMMON_MAX_LINE;
-import static org.junit.Assert.assertFalse;
-import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.*;
 
 import java.io.IOException;
 import java.nio.file.Files;
 import java.nio.file.Path;
 import java.sql.*;
+import java.util.HashMap;
 import java.util.List;
 import java.util.Properties;
 
@@ -41,6 +41,9 @@ import org.apache.zeppelin.scheduler.FIFOScheduler;
 import org.apache.zeppelin.scheduler.ParallelScheduler;
 import org.apache.zeppelin.scheduler.Scheduler;
 import org.apache.zeppelin.user.AuthenticationInfo;
+import org.apache.zeppelin.user.Credentials;
+import org.apache.zeppelin.user.UserCredentials;
+import org.apache.zeppelin.user.UsernamePassword;
 import org.junit.Before;
 import org.junit.Test;
 
@@ -75,7 +78,6 @@ public class JDBCInterpreterTest extends BasicJDBCTestCaseAdapter {
   
   @Before
   public void setUp() throws Exception {
-
     Class.forName("org.h2.Driver");
     Connection connection = DriverManager.getConnection(getJdbcConnection());
     Statement statement = connection.createStatement();
@@ -86,7 +88,7 @@ public class JDBCInterpreterTest extends BasicJDBCTestCaseAdapter {
     PreparedStatement insertStatement = connection.prepareStatement("insert into test_table(id, name) values ('a', 'a_name'),('b', 'b_name'),('c', ?);");
     insertStatement.setString(1, null);
     insertStatement.execute();
-    interpreterContext = new InterpreterContext("", "1", "", "", new AuthenticationInfo(), null, null, null, null,
+    interpreterContext = new InterpreterContext("", "1", null, "", "", new AuthenticationInfo(), null, null, null, null,
         null, null);
   }
 
@@ -251,7 +253,7 @@ public class JDBCInterpreterTest extends BasicJDBCTestCaseAdapter {
     jdbcInterpreter.interpret("", interpreterContext);
 
     List<InterpreterCompletion> completionList = jdbcInterpreter.completion("SEL", 0);
-    
+
     InterpreterCompletion correctCompletionKeyword = new InterpreterCompletion("SELECT", "SELECT");
 
     assertEquals(2, completionList.size());
@@ -259,4 +261,92 @@ public class JDBCInterpreterTest extends BasicJDBCTestCaseAdapter {
     assertEquals(0, jdbcInterpreter.completion("SEL", 100).size());
   }
 
-}
+  private Properties getDBProperty(String dbUser, String dbPassowrd) throws IOException {
+    Properties properties = new Properties();
+    properties.setProperty("common.max_count", "1000");
+    properties.setProperty("common.max_retry", "3");
+    properties.setProperty("default.driver", "org.h2.Driver");
+    properties.setProperty("default.url", getJdbcConnection());
+    if (dbUser != null) {
+      properties.setProperty("default.user", dbUser);
+    }
+    if (dbPassowrd != null) {
+      properties.setProperty("default.password", dbPassowrd);
+    }
+    return properties;
+  }
+
+  private AuthenticationInfo getUserAuth(String user, String entityName, String dbUser, String dbPassword){
+    UserCredentials userCredentials = new UserCredentials();
+    if (entityName != null && dbUser != null && dbPassword != null) {
+      UsernamePassword up = new UsernamePassword(dbUser, dbPassword);
+      userCredentials.putUsernamePassword(entityName, up);
+    }
+    AuthenticationInfo authInfo = new AuthenticationInfo();
+    authInfo.setUserCredentials(userCredentials);
+    authInfo.setUser(user);
+    return authInfo;
+  }
+
+  @Test
+  public void testMultiTenant() throws SQLException, IOException {
+
+    /**
+     * assume that the database user is 'dbuser' and password is 'dbpassword'
+     * 'jdbc1' interpreter has user('dbuser')/password('dbpassword') property
+     * 'jdbc2' interpreter doesn't have user/password property
+     * 'user1' doesn't have Credential information.
+     * 'user2' has 'jdbc2' Credential information that is same with database account.
+     */
+
+    JDBCInterpreter jdbc1 = new JDBCInterpreter(getDBProperty("dbuser", "dbpassword"));
+    JDBCInterpreter jdbc2 = new JDBCInterpreter(getDBProperty(null, null));
+
+    AuthenticationInfo user1Credential = getUserAuth("user1", null, null, null);
+    AuthenticationInfo user2Credential = getUserAuth("user2", "jdbc.jdbc2", "dbuser", "dbpassword");
+
+    // user1 runs jdbc1
+    jdbc1.open();
+    InterpreterContext ctx1 = new InterpreterContext("", "1", "jdbc.jdbc1", "", "", user1Credential,
+      null, null, null, null, null, null);
+    jdbc1.interpret("", ctx1);
+
+    JDBCUserConfigurations user1JDBC1Conf = jdbc1.getJDBCConfiguration("user1");
+    assertEquals("dbuser", user1JDBC1Conf.getPropertyMap("default").get("user"));
+    assertEquals("dbpassword", user1JDBC1Conf.getPropertyMap("default").get("password"));
+    jdbc1.close();
+
+    // user1 runs jdbc2
+    jdbc2.open();
+    InterpreterContext ctx2 = new InterpreterContext("", "1", "jdbc.jdbc2", "", "", user1Credential,
+      null, null, null, null, null, null);
+    jdbc2.interpret("", ctx2);
+
+    JDBCUserConfigurations user1JDBC2Conf = jdbc2.getJDBCConfiguration("user1");
+    assertNull(user1JDBC2Conf.getPropertyMap("default").get("user"));
+    assertNull(user1JDBC2Conf.getPropertyMap("default").get("password"));
+    jdbc2.close();
+
+    // user2 runs jdbc1
+    jdbc1.open();
+    InterpreterContext ctx3 = new InterpreterContext("", "1", "jdbc.jdbc1", "", "", user2Credential,
+      null, null, null, null, null, null);
+    jdbc1.interpret("", ctx3);
+
+    JDBCUserConfigurations user2JDBC1Conf = jdbc1.getJDBCConfiguration("user2");
+    assertEquals("dbuser", user2JDBC1Conf.getPropertyMap("default").get("user"));
+    assertEquals("dbpassword", user2JDBC1Conf.getPropertyMap("default").get("password"));
+    jdbc1.close();
+
+    // user2 runs jdbc2
+    jdbc2.open();
+    InterpreterContext ctx4 = new InterpreterContext("", "1", "jdbc.jdbc2", "", "", user2Credential,
+      null, null, null, null, null, null);
+    jdbc2.interpret("", ctx4);
+
+    JDBCUserConfigurations user2JDBC2Conf = jdbc2.getJDBCConfiguration("user2");
+    assertNull(user2JDBC2Conf.getPropertyMap("default").get("user"));
+    assertNull(user2JDBC2Conf.getPropertyMap("default").get("password"));
+    jdbc2.close();
+  }
+ }

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/livy/src/test/java/org/apache/zeppelin/livy/LivyIntegrationTest.java
----------------------------------------------------------------------
diff --git a/livy/src/test/java/org/apache/zeppelin/livy/LivyIntegrationTest.java b/livy/src/test/java/org/apache/zeppelin/livy/LivyIntegrationTest.java
index 6df3700..9d38d94 100644
--- a/livy/src/test/java/org/apache/zeppelin/livy/LivyIntegrationTest.java
+++ b/livy/src/test/java/org/apache/zeppelin/livy/LivyIntegrationTest.java
@@ -83,7 +83,7 @@ public class LivyIntegrationTest {
     AuthenticationInfo authInfo = new AuthenticationInfo("user1");
     MyInterpreterOutputListener outputListener = new MyInterpreterOutputListener();
     InterpreterOutput output = new InterpreterOutput(outputListener);
-    InterpreterContext context = new InterpreterContext("noteId", "paragraphId", "title",
+    InterpreterContext context = new InterpreterContext("noteId", "paragraphId", null, "title",
         "text", authInfo, null, null, null, null, null, output);
     sparkInterpreter.open();
     InterpreterResult result = sparkInterpreter.interpret("sc.version", context);
@@ -177,7 +177,7 @@ public class LivyIntegrationTest {
     AuthenticationInfo authInfo = new AuthenticationInfo("user1");
     MyInterpreterOutputListener outputListener = new MyInterpreterOutputListener();
     InterpreterOutput output = new InterpreterOutput(outputListener);
-    InterpreterContext context = new InterpreterContext("noteId", "paragraphId", "title",
+    InterpreterContext context = new InterpreterContext("noteId", "paragraphId", null, "title",
             "text", authInfo, null, null, null, null, null, output);
     pysparkInterpreter.open();
     InterpreterResult result = pysparkInterpreter.interpret("sc.version", context);

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/pig/src/test/java/org/apache/zeppelin/pig/PigInterpreterTest.java
----------------------------------------------------------------------
diff --git a/pig/src/test/java/org/apache/zeppelin/pig/PigInterpreterTest.java b/pig/src/test/java/org/apache/zeppelin/pig/PigInterpreterTest.java
index 3d062d6..cb16063 100644
--- a/pig/src/test/java/org/apache/zeppelin/pig/PigInterpreterTest.java
+++ b/pig/src/test/java/org/apache/zeppelin/pig/PigInterpreterTest.java
@@ -47,7 +47,7 @@ public class PigInterpreterTest {
     properties.put("zeppelin.pig.execType", "local");
     pigInterpreter = new PigInterpreter(properties);
     pigInterpreter.open();
-    context = new InterpreterContext(null, "paragraph_id", null, null, null, null, null, null, null,
+    context = new InterpreterContext(null, "paragraph_id", null, null, null, null, null, null, null, null,
             null, null);
   }
 

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/pig/src/test/java/org/apache/zeppelin/pig/PigQueryInterpreterTest.java
----------------------------------------------------------------------
diff --git a/pig/src/test/java/org/apache/zeppelin/pig/PigQueryInterpreterTest.java b/pig/src/test/java/org/apache/zeppelin/pig/PigQueryInterpreterTest.java
index 00ece44..ef282cd 100644
--- a/pig/src/test/java/org/apache/zeppelin/pig/PigQueryInterpreterTest.java
+++ b/pig/src/test/java/org/apache/zeppelin/pig/PigQueryInterpreterTest.java
@@ -65,7 +65,7 @@ public class PigQueryInterpreterTest {
     pigInterpreter.open();
     pigQueryInterpreter.open();
 
-    context = new InterpreterContext(null, "paragraph_id", null, null, null, null, null, null, null,
+    context = new InterpreterContext(null, "paragraph_id", null, null, null, null, null, null, null, null,
             null, null);
   }
 

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/python/src/test/java/org/apache/zeppelin/python/PythonCondaInterpreterTest.java
----------------------------------------------------------------------
diff --git a/python/src/test/java/org/apache/zeppelin/python/PythonCondaInterpreterTest.java b/python/src/test/java/org/apache/zeppelin/python/PythonCondaInterpreterTest.java
index c0c1be5..d9a1133 100644
--- a/python/src/test/java/org/apache/zeppelin/python/PythonCondaInterpreterTest.java
+++ b/python/src/test/java/org/apache/zeppelin/python/PythonCondaInterpreterTest.java
@@ -93,6 +93,7 @@ public class PythonCondaInterpreterTest implements InterpreterOutputListener {
     return new InterpreterContext(
         "noteId",
         "paragraphId",
+        null,
         "paragraphTitle",
         "paragraphText",
         new AuthenticationInfo(),

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterMatplotlibTest.java
----------------------------------------------------------------------
diff --git a/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterMatplotlibTest.java b/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterMatplotlibTest.java
index 7ee4c2d..03868c4 100644
--- a/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterMatplotlibTest.java
+++ b/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterMatplotlibTest.java
@@ -76,7 +76,7 @@ public class PythonInterpreterMatplotlibTest {
     interpreters.add(python);
     intpGroup.put("note", interpreters);
 
-    context = new InterpreterContext("note", "id", "title", "text", new AuthenticationInfo(),
+    context = new InterpreterContext("note", "id", null, "title", "text", new AuthenticationInfo(),
         new HashMap<String, Object>(), new GUI(),
         new AngularObjectRegistry(intpGroup.getId(), null), null,
         new LinkedList<InterpreterContextRunner>(), new InterpreterOutput(

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterPandasSqlTest.java
----------------------------------------------------------------------
diff --git a/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterPandasSqlTest.java b/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterPandasSqlTest.java
index 9154394..7fbe1d7 100644
--- a/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterPandasSqlTest.java
+++ b/python/src/test/java/org/apache/zeppelin/python/PythonInterpreterPandasSqlTest.java
@@ -78,7 +78,7 @@ public class PythonInterpreterPandasSqlTest {
 
     intpGroup.put("note", Arrays.asList(python, sql));
 
-    context = new InterpreterContext("note", "id", "title", "text", new AuthenticationInfo(),
+    context = new InterpreterContext("note", "id", null, "title", "text", new AuthenticationInfo(),
         new HashMap<String, Object>(), new GUI(),
         new AngularObjectRegistry(intpGroup.getId(), null), null,
         new LinkedList<InterpreterContextRunner>(), new InterpreterOutput(

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/scalding/src/test/java/org/apache/zeppelin/scalding/ScaldingInterpreterTest.java
----------------------------------------------------------------------
diff --git a/scalding/src/test/java/org/apache/zeppelin/scalding/ScaldingInterpreterTest.java b/scalding/src/test/java/org/apache/zeppelin/scalding/ScaldingInterpreterTest.java
index 7ffbd97..eb57b14 100644
--- a/scalding/src/test/java/org/apache/zeppelin/scalding/ScaldingInterpreterTest.java
+++ b/scalding/src/test/java/org/apache/zeppelin/scalding/ScaldingInterpreterTest.java
@@ -64,7 +64,7 @@ public class ScaldingInterpreterTest {
     }
 
     InterpreterGroup intpGroup = new InterpreterGroup();
-    context = new InterpreterContext("note", "id", "title", "text", new AuthenticationInfo(),
+    context = new InterpreterContext("note", "id", null, "title", "text", new AuthenticationInfo(),
         new HashMap<String, Object>(), new GUI(), new AngularObjectRegistry(
             intpGroup.getId(), null), null,
         new LinkedList<InterpreterContextRunner>(), null);

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/scio/src/test/java/org/apache/zeppelin/scio/ScioInterpreterTest.java
----------------------------------------------------------------------
diff --git a/scio/src/test/java/org/apache/zeppelin/scio/ScioInterpreterTest.java b/scio/src/test/java/org/apache/zeppelin/scio/ScioInterpreterTest.java
index bd33886..37733a1 100644
--- a/scio/src/test/java/org/apache/zeppelin/scio/ScioInterpreterTest.java
+++ b/scio/src/test/java/org/apache/zeppelin/scio/ScioInterpreterTest.java
@@ -40,7 +40,7 @@ public class ScioInterpreterTest {
   private final String newline = "\n";
 
   private InterpreterContext getNewContext() {
-    return new InterpreterContext("note", "id", "title", "text",
+    return new InterpreterContext("note", "id", null, "title", "text",
         new AuthenticationInfo(),
         new HashMap<String, Object>(),
         new GUI(),

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/shell/src/test/java/org/apache/zeppelin/shell/ShellInterpreterTest.java
----------------------------------------------------------------------
diff --git a/shell/src/test/java/org/apache/zeppelin/shell/ShellInterpreterTest.java b/shell/src/test/java/org/apache/zeppelin/shell/ShellInterpreterTest.java
index acdb65c..a796ac5 100644
--- a/shell/src/test/java/org/apache/zeppelin/shell/ShellInterpreterTest.java
+++ b/shell/src/test/java/org/apache/zeppelin/shell/ShellInterpreterTest.java
@@ -47,7 +47,7 @@ public class ShellInterpreterTest {
   @Test
   public void test() {
     shell.open();
-    InterpreterContext context = new InterpreterContext("", "1", "", "", null, null, null, null, null, null, null);
+    InterpreterContext context = new InterpreterContext("", "1", null, "", "", null, null, null, null, null, null, null);
     InterpreterResult result = new InterpreterResult(Code.ERROR);
     if (System.getProperty("os.name").startsWith("Windows")) {
       result = shell.interpret("dir", context);
@@ -64,7 +64,7 @@ public class ShellInterpreterTest {
   @Test
   public void testInvalidCommand(){
     shell.open();
-    InterpreterContext context = new InterpreterContext("","1","","",null,null,null,null,null,null,null);
+    InterpreterContext context = new InterpreterContext("","1",null,"","",null,null,null,null,null,null,null);
     InterpreterResult result = new InterpreterResult(Code.ERROR);
     if (System.getProperty("os.name").startsWith("Windows")) {
       result = shell.interpret("invalid_command\ndir",context);

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/spark/src/test/java/org/apache/zeppelin/spark/DepInterpreterTest.java
----------------------------------------------------------------------
diff --git a/spark/src/test/java/org/apache/zeppelin/spark/DepInterpreterTest.java b/spark/src/test/java/org/apache/zeppelin/spark/DepInterpreterTest.java
index 03ecb9e..7bb660d 100644
--- a/spark/src/test/java/org/apache/zeppelin/spark/DepInterpreterTest.java
+++ b/spark/src/test/java/org/apache/zeppelin/spark/DepInterpreterTest.java
@@ -64,7 +64,7 @@ public class DepInterpreterTest {
     intpGroup.get("note").add(dep);
     dep.setInterpreterGroup(intpGroup);
 
-    context = new InterpreterContext("note", "id", "title", "text", new AuthenticationInfo(),
+    context = new InterpreterContext("note", "id", null, "title", "text", new AuthenticationInfo(),
         new HashMap<String, Object>(), new GUI(),
         new AngularObjectRegistry(intpGroup.getId(), null),
         null,

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/spark/src/test/java/org/apache/zeppelin/spark/PySparkInterpreterTest.java
----------------------------------------------------------------------
diff --git a/spark/src/test/java/org/apache/zeppelin/spark/PySparkInterpreterTest.java b/spark/src/test/java/org/apache/zeppelin/spark/PySparkInterpreterTest.java
index 6a60fef..85cc46e 100644
--- a/spark/src/test/java/org/apache/zeppelin/spark/PySparkInterpreterTest.java
+++ b/spark/src/test/java/org/apache/zeppelin/spark/PySparkInterpreterTest.java
@@ -100,7 +100,7 @@ public class PySparkInterpreterTest {
       pySparkInterpreter.open();
     }
 
-    context = new InterpreterContext("note", "id", "title", "text",
+    context = new InterpreterContext("note", "id", null, "title", "text",
       new AuthenticationInfo(),
       new HashMap<String, Object>(),
       new GUI(),

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/spark/src/test/java/org/apache/zeppelin/spark/SparkInterpreterTest.java
----------------------------------------------------------------------
diff --git a/spark/src/test/java/org/apache/zeppelin/spark/SparkInterpreterTest.java b/spark/src/test/java/org/apache/zeppelin/spark/SparkInterpreterTest.java
index ff26e6a..fe127a6 100644
--- a/spark/src/test/java/org/apache/zeppelin/spark/SparkInterpreterTest.java
+++ b/spark/src/test/java/org/apache/zeppelin/spark/SparkInterpreterTest.java
@@ -92,7 +92,7 @@ public class SparkInterpreterTest {
       repl.open();
     }
 
-    context = new InterpreterContext("note", "id", "title", "text",
+    context = new InterpreterContext("note", "id", null, "title", "text",
         new AuthenticationInfo(),
         new HashMap<String, Object>(),
         new GUI(),

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/spark/src/test/java/org/apache/zeppelin/spark/SparkSqlInterpreterTest.java
----------------------------------------------------------------------
diff --git a/spark/src/test/java/org/apache/zeppelin/spark/SparkSqlInterpreterTest.java b/spark/src/test/java/org/apache/zeppelin/spark/SparkSqlInterpreterTest.java
index badd040..303a54d 100644
--- a/spark/src/test/java/org/apache/zeppelin/spark/SparkSqlInterpreterTest.java
+++ b/spark/src/test/java/org/apache/zeppelin/spark/SparkSqlInterpreterTest.java
@@ -75,7 +75,7 @@ public class SparkSqlInterpreterTest {
       sql.setInterpreterGroup(intpGroup);
       sql.open();
     }
-    context = new InterpreterContext("note", "id", "title", "text", new AuthenticationInfo(),
+    context = new InterpreterContext("note", "id", null, "title", "text", new AuthenticationInfo(),
         new HashMap<String, Object>(), new GUI(),
         new AngularObjectRegistry(intpGroup.getId(), null),
         new LocalResourcePool("id"),

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularElemTest.scala
----------------------------------------------------------------------
diff --git a/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularElemTest.scala b/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularElemTest.scala
index 9b5cd62..196e5cc 100644
--- a/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularElemTest.scala
+++ b/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularElemTest.scala
@@ -33,7 +33,7 @@ trait AbstractAngularElemTest
 
   override def beforeEach() {
     val intpGroup = new InterpreterGroup()
-    val context = new InterpreterContext("note", "paragraph", "title", "text",
+    val context = new InterpreterContext("note", "paragraph", null, "title", "text",
       new AuthenticationInfo(), new util.HashMap[String, Object](), new GUI(),
       new AngularObjectRegistry(intpGroup.getId(), null),
       null,

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularModelTest.scala
----------------------------------------------------------------------
diff --git a/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularModelTest.scala b/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularModelTest.scala
index 33ca508..dba2a33 100644
--- a/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularModelTest.scala
+++ b/zeppelin-display/src/test/scala/org/apache/zeppelin/display/angular/AbstractAngularModelTest.scala
@@ -29,7 +29,7 @@ trait AbstractAngularModelTest extends FlatSpec
 with BeforeAndAfter with BeforeAndAfterEach with Eventually with Matchers {
   override def beforeEach() {
     val intpGroup = new InterpreterGroup()
-    val context = new InterpreterContext("note", "id", "title", "text", new AuthenticationInfo(),
+    val context = new InterpreterContext("note", "id", null, "title", "text", new AuthenticationInfo(),
       new java.util.HashMap[String, Object](), new GUI(), new AngularObjectRegistry(
         intpGroup.getId(), null),
       null,

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/InterpreterContext.java
----------------------------------------------------------------------
diff --git a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/InterpreterContext.java b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/InterpreterContext.java
index f8c9032..db540aa 100644
--- a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/InterpreterContext.java
+++ b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/InterpreterContext.java
@@ -49,6 +49,7 @@ public class InterpreterContext {
   }
 
   private final String noteId;
+  private final String replName;
   private final String paragraphTitle;
   private final String paragraphId;
   private final String paragraphText;
@@ -63,6 +64,7 @@ public class InterpreterContext {
 
   public InterpreterContext(String noteId,
                             String paragraphId,
+                            String replName,
                             String paragraphTitle,
                             String paragraphText,
                             AuthenticationInfo authenticationInfo,
@@ -75,6 +77,7 @@ public class InterpreterContext {
                             ) {
     this.noteId = noteId;
     this.paragraphId = paragraphId;
+    this.replName = replName;
     this.paragraphTitle = paragraphTitle;
     this.paragraphText = paragraphText;
     this.authenticationInfo = authenticationInfo;
@@ -88,6 +91,7 @@ public class InterpreterContext {
 
   public InterpreterContext(String noteId,
                             String paragraphId,
+                            String replName,
                             String paragraphTitle,
                             String paragraphText,
                             AuthenticationInfo authenticationInfo,
@@ -98,8 +102,8 @@ public class InterpreterContext {
                             List<InterpreterContextRunner> contextRunners,
                             InterpreterOutput output,
                             RemoteInterpreterEventClient eventClient) {
-    this(noteId, paragraphId, paragraphTitle, paragraphText, authenticationInfo, config, gui,
-        angularObjectRegistry, resourcePool, contextRunners, output);
+    this(noteId, paragraphId, replName, paragraphTitle, paragraphText, authenticationInfo,
+        config, gui, angularObjectRegistry, resourcePool, contextRunners, output);
     this.client = new RemoteEventClient(eventClient);
   }
 
@@ -107,6 +111,10 @@ public class InterpreterContext {
     return noteId;
   }
 
+  public String getReplName() {
+    return replName;
+  }
+
   public String getParagraphId() {
     return paragraphId;
   }

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreter.java
----------------------------------------------------------------------
diff --git a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreter.java b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreter.java
index b2a4fbe..f50fac1 100644
--- a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreter.java
+++ b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreter.java
@@ -494,6 +494,7 @@ public class RemoteInterpreter extends Interpreter {
     return new RemoteInterpreterContext(
         ic.getNoteId(),
         ic.getParagraphId(),
+        ic.getReplName(),
         ic.getParagraphTitle(),
         ic.getParagraphText(),
         gson.toJson(ic.getAuthenticationInfo()),

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreterServer.java
----------------------------------------------------------------------
diff --git a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreterServer.java b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreterServer.java
index 4d6f3ba..b9b3868 100644
--- a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreterServer.java
+++ b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/remote/RemoteInterpreterServer.java
@@ -544,6 +544,7 @@ public class RemoteInterpreterServer
     return new InterpreterContext(
         ric.getNoteId(),
         ric.getParagraphId(),
+        ric.getReplName(),
         ric.getParagraphTitle(),
         ric.getParagraphText(),
         gson.fromJson(ric.getAuthenticationInfo(), AuthenticationInfo.class),

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/InterpreterCompletion.java
----------------------------------------------------------------------
diff --git a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/InterpreterCompletion.java b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/InterpreterCompletion.java
index cc7615e..04d345a 100644
--- a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/InterpreterCompletion.java
+++ b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/InterpreterCompletion.java
@@ -1,21 +1,4 @@
 /**
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements.  See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership.  The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License.  You may obtain a copy of the License at
- *
- *     http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-/**
  * Autogenerated by Thrift Compiler (0.9.2)
  *
  * DO NOT EDIT UNLESS YOU ARE SURE THAT YOU KNOW WHAT YOU ARE DOING
@@ -51,7 +34,7 @@ import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
 @SuppressWarnings({"cast", "rawtypes", "serial", "unchecked"})
-@Generated(value = "Autogenerated by Thrift Compiler (0.9.2)", date = "2016-11-17")
+@Generated(value = "Autogenerated by Thrift Compiler (0.9.2)", date = "2016-11-18")
 public class InterpreterCompletion implements org.apache.thrift.TBase<InterpreterCompletion, InterpreterCompletion._Fields>, java.io.Serializable, Cloneable, Comparable<InterpreterCompletion> {
   private static final org.apache.thrift.protocol.TStruct STRUCT_DESC = new org.apache.thrift.protocol.TStruct("InterpreterCompletion");
 

http://git-wip-us.apache.org/repos/asf/zeppelin/blob/b7307d49/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/RemoteApplicationResult.java
----------------------------------------------------------------------
diff --git a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/RemoteApplicationResult.java b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/RemoteApplicationResult.java
index 95c5cf6..fc0670c 100644
--- a/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/RemoteApplicationResult.java
+++ b/zeppelin-interpreter/src/main/java/org/apache/zeppelin/interpreter/thrift/RemoteApplicationResult.java
@@ -1,21 +1,4 @@
 /**
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements.  See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership.  The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License.  You may obtain a copy of the License at
- *
- *     http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-/**
  * Autogenerated by Thrift Compiler (0.9.2)
  *
  * DO NOT EDIT UNLESS YOU ARE SURE THAT YOU KNOW WHAT YOU ARE DOING
@@ -51,7 +34,7 @@ import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
 @SuppressWarnings({"cast", "rawtypes", "serial", "unchecked"})
-@Generated(value = "Autogenerated by Thrift Compiler (0.9.2)", date = "2016-11-17")
+@Generated(value = "Autogenerated by Thrift Compiler (0.9.2)", date = "2016-11-18")
 public class RemoteApplicationResult implements org.apache.thrift.TBase<RemoteApplicationResult, RemoteApplicationResult._Fields>, java.io.Serializable, Cloneable, Comparable<RemoteApplicationResult> {
   private static final org.apache.thrift.protocol.TStruct STRUCT_DESC = new org.apache.thrift.protocol.TStruct("RemoteApplicationResult");