diff --git a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/CommonConfigurationKeys.java b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/CommonConfigurationKeys.java index 6949c67f27..db82498b3c 100644 --- a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/CommonConfigurationKeys.java +++ b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/CommonConfigurationKeys.java @@ -399,6 +399,8 @@ public class CommonConfigurationKeys extends CommonConfigurationKeysPublic { public static final String ZK_ACL_DEFAULT = "world:anyone:rwcda"; /** Authentication for the ZooKeeper ensemble. */ public static final String ZK_AUTH = ZK_PREFIX + "auth"; + /** Principal name for zookeeper servers. */ + public static final String ZK_SERVER_PRINCIPAL = ZK_PREFIX + "server.principal"; /** Address of the ZooKeeper ensemble. */ public static final String ZK_ADDRESS = ZK_PREFIX + "address"; diff --git a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/security/token/delegation/ZKDelegationTokenSecretManager.java b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/security/token/delegation/ZKDelegationTokenSecretManager.java index 5e5ea8cebc..452565676a 100644 --- a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/security/token/delegation/ZKDelegationTokenSecretManager.java +++ b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/security/token/delegation/ZKDelegationTokenSecretManager.java @@ -55,6 +55,7 @@ import org.apache.hadoop.security.token.Token; import org.apache.hadoop.security.token.delegation.web.DelegationTokenManager; import static org.apache.hadoop.util.Time.now; +import org.apache.hadoop.util.curator.ZKCuratorManager; import org.apache.zookeeper.CreateMode; import org.apache.zookeeper.KeeperException; import org.apache.zookeeper.KeeperException.NoNodeException; @@ -98,6 +99,8 @@ public abstract class ZKDelegationTokenSecretManager authInfos) throws IOException { CuratorFramework client = CuratorFrameworkFactory.builder() .connectString(zkHostPort) + .zookeeperFactory(new HadoopZookeeperFactory( + conf.get(CommonConfigurationKeys.ZK_SERVER_PRINCIPAL))) .sessionTimeoutMs(zkSessionTimeout) .retryPolicy(retryPolicy) .authorization(authInfos) @@ -428,4 +434,27 @@ public void setData(String path, byte[] data, int version) .forPath(path, data)); } } + + public static class HadoopZookeeperFactory implements ZookeeperFactory { + private final String zkPrincipal; + + public HadoopZookeeperFactory(String zkPrincipal) { + this.zkPrincipal = zkPrincipal; + } + + @Override + public ZooKeeper newZooKeeper(String connectString, int sessionTimeout, + Watcher watcher, boolean canBeReadOnly + ) throws Exception { + ZKClientConfig zkClientConfig = new ZKClientConfig(); + if (zkPrincipal != null) { + LOG.info("Configuring zookeeper to use {} as the server principal", + zkPrincipal); + zkClientConfig.setProperty(ZKClientConfig.ZK_SASL_CLIENT_USERNAME, + zkPrincipal); + } + return new ZooKeeper(connectString, sessionTimeout, watcher, + canBeReadOnly, zkClientConfig); + } + } } \ No newline at end of file