Browse Source

HADOOP-14987. Improve KMSClientProvider log around delegation token checking. Contributed by Xiaoyu Yao and Xiao Chen.

(cherry picked from commit 59d78a5088700350a5122c3a3ba5e76cd26d6a80)
(cherry picked from commit b8d8c25a18df467263b25677d45c9c60b43d4e68)

 Conflicts:
	hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/crypto/key/kms/KMSClientProvider.java
Xiao Chen 7 years ago
parent
commit
04fbda4bae

+ 27 - 19
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/crypto/key/kms/KMSClientProvider.java

@@ -123,6 +123,13 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
 
   private final ValueQueue<EncryptedKeyVersion> encKeyVersionQueue;
 
+  private final Text dtService;
+
+  // Allow fallback to default kms server port 9600 for certain tests that do
+  // not specify the port explicitly in the kms provider url.
+  @VisibleForTesting
+  public static volatile boolean fallbackDefaultPortForTesting = false;
+
   private class EncryptedQueueRefiller implements
     ValueQueue.QueueRefiller<EncryptedKeyVersion> {
 
@@ -369,7 +376,7 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
     return s;
   }
 
-  private String kmsUrl;
+  private URL kmsUrl;
   private SSLFactory sslFactory;
   private ConnectionConfigurator configurator;
   private DelegationTokenAuthenticatedURL.Token authToken;
@@ -421,7 +428,15 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
   public KMSClientProvider(URI uri, Configuration conf) throws IOException {
     super(conf);
     kmsUrl = createServiceURL(extractKMSPath(uri));
-    if ("https".equalsIgnoreCase(new URL(kmsUrl).getProtocol())) {
+    int kmsPort = kmsUrl.getPort();
+    if ((kmsPort == -1) && fallbackDefaultPortForTesting) {
+      kmsPort = 9600;
+    }
+
+    InetSocketAddress addr = new InetSocketAddress(kmsUrl.getHost(), kmsPort);
+    dtService = SecurityUtil.buildTokenService(addr);
+
+    if ("https".equalsIgnoreCase(kmsUrl.getProtocol())) {
       sslFactory = new SSLFactory(SSLFactory.Mode.CLIENT, conf);
       try {
         sslFactory.init();
@@ -457,19 +472,20 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
                     KMS_CLIENT_ENC_KEY_CACHE_NUM_REFILL_THREADS_DEFAULT),
             new EncryptedQueueRefiller());
     authToken = new DelegationTokenAuthenticatedURL.Token();
+    LOG.info("KMSClientProvider for KMS url: {} delegation token service: {}" +
+        " created.", kmsUrl, dtService);
   }
 
   private static Path extractKMSPath(URI uri) throws MalformedURLException, IOException {
     return ProviderUtils.unnestUri(uri);
   }
 
-  private static String createServiceURL(Path path) throws IOException {
+  private static URL createServiceURL(Path path) throws IOException {
     String str = new URL(path.toString()).toExternalForm();
     if (str.endsWith("/")) {
       str = str.substring(0, str.length() - 1);
     }
-    return new URL(str + KMSRESTConstants.SERVICE_VERSION + "/").
-        toExternalForm();
+    return new URL(str + KMSRESTConstants.SERVICE_VERSION + "/");
   }
 
   private URL createURL(String collection, String resource, String subResource,
@@ -982,7 +998,6 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
   public Token<?>[] addDelegationTokens(final String renewer,
       Credentials credentials) throws IOException {
     Token<?>[] tokens = null;
-    Text dtService = getDelegationTokenService();
     Token<?> token = credentials.getToken(dtService);
     if (token == null) {
       final URL url = createURL(null, null, null, null);
@@ -1017,22 +1032,15 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
     }
     return tokens;
   }
-  
-  private Text getDelegationTokenService() throws IOException {
-    URL url = new URL(kmsUrl);
-    InetSocketAddress addr = new InetSocketAddress(url.getHost(),
-        url.getPort());
-    Text dtService = SecurityUtil.buildTokenService(addr);
-    return dtService;
-  }
 
   private boolean currentUgiContainsKmsDt() throws IOException {
     // Add existing credentials from current UGI, since provider is cached.
     Credentials creds = UserGroupInformation.getCurrentUser().
         getCredentials();
     if (!creds.getAllTokens().isEmpty()) {
+      LOG.debug("Searching for token that matches service: {}", dtService);
       org.apache.hadoop.security.token.Token<? extends TokenIdentifier>
-          dToken = creds.getToken(getDelegationTokenService());
+          dToken = creds.getToken(dtService);
       if (dToken != null) {
         return true;
       }
@@ -1043,9 +1051,9 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
   private UserGroupInformation getActualUgi() throws IOException {
     final UserGroupInformation currentUgi = UserGroupInformation
         .getCurrentUser();
-    if (LOG.isDebugEnabled()) {
-      UserGroupInformation.logAllUserInfo(currentUgi);
-    }
+
+    UserGroupInformation.logAllUserInfo(LOG, currentUgi);
+
     // Use current user by default
     UserGroupInformation actualUgi = currentUgi;
     if (currentUgi.getRealUser() != null) {
@@ -1079,6 +1087,6 @@ public class KMSClientProvider extends KeyProvider implements CryptoExtension,
 
   @VisibleForTesting
   String getKMSUrl() {
-    return kmsUrl;
+    return kmsUrl.toString();
   }
 }

+ 39 - 8
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/security/UserGroupInformation.java

@@ -1868,20 +1868,51 @@ public class UserGroupInformation {
     }
   }
 
-  public static void logAllUserInfo(UserGroupInformation ugi) throws
+  /**
+   * Log current UGI and token information into specified log.
+   * @param ugi - UGI
+   * @throws IOException
+   */
+  @InterfaceAudience.LimitedPrivate({"HDFS", "KMS"})
+  @InterfaceStability.Unstable
+  public static void logUserInfo(Logger log, String caption,
+      UserGroupInformation ugi) throws IOException {
+    if (log.isDebugEnabled()) {
+      log.debug(caption + " UGI: " + ugi);
+      for (Token<?> token : ugi.getTokens()) {
+        log.debug("+token:" + token);
+      }
+    }
+  }
+
+  /**
+   * Log all (current, real, login) UGI and token info into specified log.
+   * @param ugi - UGI
+   * @throws IOException
+   */
+  @InterfaceAudience.LimitedPrivate({"HDFS", "KMS"})
+  @InterfaceStability.Unstable
+  public static void logAllUserInfo(Logger log, UserGroupInformation ugi) throws
       IOException {
-    if (LOG.isDebugEnabled()) {
-      LOG.debug("UGI: " + ugi);
+    if (log.isDebugEnabled()) {
+      logUserInfo(log, "Current", ugi.getCurrentUser());
       if (ugi.getRealUser() != null) {
-        LOG.debug("+RealUGI: " + ugi.getRealUser());
-      }
-      LOG.debug("+LoginUGI: " + ugi.getLoginUser());
-      for (Token<?> token : ugi.getTokens()) {
-        LOG.debug("+UGI token:" + token);
+        logUserInfo(log, "Real", ugi.getRealUser());
       }
+      logUserInfo(log, "Login", ugi.getLoginUser());
     }
   }
 
+  /**
+   * Log all (current, real, login) UGI and token info into UGI debug log.
+   * @param ugi - UGI
+   * @throws IOException
+   */
+  public static void logAllUserInfo(UserGroupInformation ugi) throws
+      IOException {
+    logAllUserInfo(LOG, ugi);
+  }
+
   private void print() throws IOException {
     System.out.println("User: " + getUserName());
     System.out.print("Group Ids: ");

+ 15 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/crypto/key/kms/TestLoadBalancingKMSClientProvider.java

@@ -39,8 +39,11 @@ import org.apache.hadoop.crypto.key.KeyProviderCryptoExtension;
 import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
 import org.apache.hadoop.net.ConnectTimeoutException;
 import org.apache.hadoop.security.AccessControlException;
+import org.apache.hadoop.security.SecurityUtil;
 import org.apache.hadoop.security.authentication.client.AuthenticationException;
 import org.apache.hadoop.security.authorize.AuthorizationException;
+import org.junit.After;
+import org.junit.BeforeClass;
 import org.junit.Test;
 import org.mockito.Mockito;
 
@@ -48,9 +51,20 @@ import com.google.common.collect.Sets;
 
 public class TestLoadBalancingKMSClientProvider {
 
+  @BeforeClass
+  public static void setup() throws IOException {
+    SecurityUtil.setTokenServiceUseIp(false);
+  }
+
+  @After
+  public void teardown() throws IOException {
+    KMSClientProvider.fallbackDefaultPortForTesting = false;
+  }
+
   @Test
   public void testCreation() throws Exception {
     Configuration conf = new Configuration();
+    KMSClientProvider.fallbackDefaultPortForTesting = true;
     KeyProvider kp = new KMSClientProvider.Factory().createProvider(new URI(
         "kms://http@host1/kms/foo"), conf);
     assertTrue(kp instanceof LoadBalancingKMSClientProvider);
@@ -231,6 +245,7 @@ public class TestLoadBalancingKMSClientProvider {
   @Test
   public void testClassCastException() throws Exception {
     Configuration conf = new Configuration();
+    KMSClientProvider.fallbackDefaultPortForTesting = true;
     KMSClientProvider p1 = new MyKMSClientProvider(
         new URI("kms://http@host1/kms/foo"), conf);
     LoadBalancingKMSClientProvider kp = new LoadBalancingKMSClientProvider(