You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by vi...@apache.org on 2015/09/02 00:27:55 UTC

hadoop git commit: YARN-3287. Made TimelineClient put methods do as the correct login context. Contributed by Daryn Sharp and Jonathan Eagles.

Repository: hadoop
Updated Branches:
  refs/heads/branch-2.6.1 a57ada6c1 -> 9005b141a


YARN-3287. Made TimelineClient put methods do as the correct login context. Contributed by Daryn Sharp and Jonathan Eagles.

(cherry picked from commit d6e05c5ee26feefc17267b7c9db1e2a3dbdef117)
(cherry picked from commit a94d23762e2cf4211fe84661eb67504c7072db49)
(cherry picked from commit 68e07eb50b872ec8a78923df8f5f640f08a72aa2)


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

Branch: refs/heads/branch-2.6.1
Commit: 9005b141a5188b2509b115348dea50816766b7f1
Parents: a57ada6
Author: Zhijie Shen <zj...@apache.org>
Authored: Mon Mar 9 13:54:36 2015 -0700
Committer: Vinod Kumar Vavilapalli <vi...@apache.org>
Committed: Tue Sep 1 15:24:36 2015 -0700

----------------------------------------------------------------------
 hadoop-yarn-project/CHANGES.txt                 |   3 +
 .../client/api/impl/TimelineClientImpl.java     |  82 +++----
 .../TestTimelineAuthenticationFilter.java       | 221 +++++++++----------
 3 files changed, 135 insertions(+), 171 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hadoop/blob/9005b141/hadoop-yarn-project/CHANGES.txt
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/CHANGES.txt b/hadoop-yarn-project/CHANGES.txt
index 1b1d4c0..ef76d45 100644
--- a/hadoop-yarn-project/CHANGES.txt
+++ b/hadoop-yarn-project/CHANGES.txt
@@ -108,6 +108,9 @@ Release 2.6.1 - UNRELEASED
     YARN-3227. Timeline renew delegation token fails when RM user's TGT is expired
     (Zhijie Shen via xgong)
 
+    YARN-3287. Made TimelineClient put methods do as the correct login context.
+    (Daryn Sharp and Jonathan Eagles via zjshen)
+
 Release 2.6.0 - 2014-11-18
 
   INCOMPATIBLE CHANGES

http://git-wip-us.apache.org/repos/asf/hadoop/blob/9005b141/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/client/api/impl/TimelineClientImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/client/api/impl/TimelineClientImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/client/api/impl/TimelineClientImpl.java
index ad5345c..f5c85c1 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/client/api/impl/TimelineClientImpl.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/client/api/impl/TimelineClientImpl.java
@@ -45,6 +45,7 @@ import org.apache.hadoop.classification.InterfaceAudience.Private;
 import org.apache.hadoop.classification.InterfaceStability.Unstable;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.security.authentication.client.AuthenticationException;
 import org.apache.hadoop.security.authentication.client.ConnectionConfigurator;
 import org.apache.hadoop.security.ssl.SSLFactory;
 import org.apache.hadoop.security.token.Token;
@@ -105,6 +106,8 @@ public class TimelineClientImpl extends TimelineClient {
   private DelegationTokenAuthenticator authenticator;
   private DelegationTokenAuthenticatedURL.Token token;
   private URI resURI;
+  private UserGroupInformation authUgi;
+  private String doAsUser;
 
   @Private
   @VisibleForTesting
@@ -246,6 +249,15 @@ public class TimelineClientImpl extends TimelineClient {
   }
 
   protected void serviceInit(Configuration conf) throws Exception {
+    UserGroupInformation ugi = UserGroupInformation.getCurrentUser();
+    UserGroupInformation realUgi = ugi.getRealUser();
+    if (realUgi != null) {
+      authUgi = realUgi;
+      doAsUser = ugi.getShortUserName();
+    } else {
+      authUgi = ugi;
+      doAsUser = null;
+    }
     ClientConfig cc = new DefaultClientConfig();
     cc.getClasses().add(YarnJacksonJaxbJsonProvider.class);
     connConfigurator = newConnConfigurator(conf);
@@ -295,16 +307,20 @@ public class TimelineClientImpl extends TimelineClient {
     doPosting(domain, "domain");
   }
 
-  private ClientResponse doPosting(Object obj, String path) throws IOException, YarnException {
+  private ClientResponse doPosting(final Object obj, final String path)
+      throws IOException, YarnException {
     ClientResponse resp;
     try {
-      resp = doPostingObject(obj, path);
-    } catch (RuntimeException re) {
-      // runtime exception is expected if the client cannot connect the server
-      String msg =
-          "Failed to get the response from the timeline server.";
-      LOG.error(msg, re);
-      throw re;
+      resp = authUgi.doAs(new PrivilegedExceptionAction<ClientResponse>() {
+        @Override
+        public ClientResponse run() throws Exception {
+          return doPostingObject(obj, path);
+        }
+      });
+    } catch (UndeclaredThrowableException e) {
+        throw new IOException(e.getCause());
+    } catch (InterruptedException ie) {
+      throw new IOException(ie);
     }
     if (resp == null ||
         resp.getClientResponseStatus() != ClientResponse.Status.OK) {
@@ -325,11 +341,6 @@ public class TimelineClientImpl extends TimelineClient {
   @Override
   public Token<TimelineDelegationTokenIdentifier> getDelegationToken(
       final String renewer) throws IOException, YarnException {
-    boolean isProxyAccess =
-        UserGroupInformation.getCurrentUser().getAuthenticationMethod()
-        == UserGroupInformation.AuthenticationMethod.PROXY;
-    final String doAsUser = isProxyAccess ?
-        UserGroupInformation.getCurrentUser().getShortUserName() : null;
     PrivilegedExceptionAction<Token<TimelineDelegationTokenIdentifier>> getDTAction =
         new PrivilegedExceptionAction<Token<TimelineDelegationTokenIdentifier>>() {
 
@@ -351,11 +362,6 @@ public class TimelineClientImpl extends TimelineClient {
   public long renewDelegationToken(
       final Token<TimelineDelegationTokenIdentifier> timelineDT)
           throws IOException, YarnException {
-    boolean isProxyAccess =
-        UserGroupInformation.getCurrentUser().getAuthenticationMethod()
-        == UserGroupInformation.AuthenticationMethod.PROXY;
-    final String doAsUser = isProxyAccess ?
-        UserGroupInformation.getCurrentUser().getShortUserName() : null;
     PrivilegedExceptionAction<Long> renewDTAction =
         new PrivilegedExceptionAction<Long>() {
 
@@ -383,11 +389,6 @@ public class TimelineClientImpl extends TimelineClient {
   public void cancelDelegationToken(
       final Token<TimelineDelegationTokenIdentifier> timelineDT)
           throws IOException, YarnException {
-    boolean isProxyAccess =
-        UserGroupInformation.getCurrentUser().getAuthenticationMethod()
-        == UserGroupInformation.AuthenticationMethod.PROXY;
-    final String doAsUser = isProxyAccess ?
-        UserGroupInformation.getCurrentUser().getShortUserName() : null;
     PrivilegedExceptionAction<Void> cancelDTAction =
         new PrivilegedExceptionAction<Void>() {
 
@@ -419,15 +420,9 @@ public class TimelineClientImpl extends TimelineClient {
       @Override
       public Object run() throws IOException {
         // Try pass the request, if fail, keep retrying
-        boolean isProxyAccess =
-            UserGroupInformation.getCurrentUser().getAuthenticationMethod()
-            == UserGroupInformation.AuthenticationMethod.PROXY;
-        UserGroupInformation callerUGI = isProxyAccess ?
-            UserGroupInformation.getCurrentUser().getRealUser()
-            : UserGroupInformation.getCurrentUser();
-        callerUGI.checkTGTAndReloginFromKeytab();
+        authUgi.checkTGTAndReloginFromKeytab();
         try {
-          return callerUGI.doAs(action);
+          return authUgi.doAs(action);
         } catch (UndeclaredThrowableException e) {
           throw new IOException(e.getCause());
         } catch (InterruptedException e) {
@@ -467,28 +462,15 @@ public class TimelineClientImpl extends TimelineClient {
 
     @Override
     public HttpURLConnection getHttpURLConnection(final URL url) throws IOException {
-      boolean isProxyAccess =
-          UserGroupInformation.getCurrentUser().getAuthenticationMethod()
-          == UserGroupInformation.AuthenticationMethod.PROXY;
-      UserGroupInformation callerUGI = isProxyAccess ?
-          UserGroupInformation.getCurrentUser().getRealUser()
-          : UserGroupInformation.getCurrentUser();
-      final String doAsUser = isProxyAccess ?
-          UserGroupInformation.getCurrentUser().getShortUserName() : null;
-      callerUGI.checkTGTAndReloginFromKeytab();
+      authUgi.checkTGTAndReloginFromKeytab();
       try {
-        return callerUGI.doAs(new PrivilegedExceptionAction<HttpURLConnection>() {
-          @Override
-          public HttpURLConnection run() throws Exception {
-            return new DelegationTokenAuthenticatedURL(
-                authenticator, connConfigurator).openConnection(url, token,
-                doAsUser);
-          }
-        });
+        return new DelegationTokenAuthenticatedURL(
+            authenticator, connConfigurator).openConnection(url, token,
+              doAsUser);
       } catch (UndeclaredThrowableException e) {
         throw new IOException(e.getCause());
-      } catch (InterruptedException e) {
-        throw new IOException(e);
+      } catch (AuthenticationException ae) {
+        throw new IOException(ae);
       }
     }
 

http://git-wip-us.apache.org/repos/asf/hadoop/blob/9005b141/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/security/TestTimelineAuthenticationFilter.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/security/TestTimelineAuthenticationFilter.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/security/TestTimelineAuthenticationFilter.java
index 53d8c81..c93e8f2 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/security/TestTimelineAuthenticationFilter.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/security/TestTimelineAuthenticationFilter.java
@@ -34,6 +34,7 @@ import org.apache.hadoop.io.Text;
 import org.apache.hadoop.minikdc.MiniKdc;
 import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.security.authentication.KerberosTestUtils;
+import org.apache.hadoop.security.authentication.client.AuthenticationException;
 import org.apache.hadoop.security.authentication.server.KerberosAuthenticationHandler;
 import org.apache.hadoop.security.authorize.AuthorizationException;
 import org.apache.hadoop.security.ssl.KeyStoreTestUtil;
@@ -47,9 +48,9 @@ import org.apache.hadoop.yarn.security.client.TimelineDelegationTokenIdentifier;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.ApplicationHistoryServer;
 import org.apache.hadoop.yarn.server.timeline.MemoryTimelineStore;
 import org.apache.hadoop.yarn.server.timeline.TimelineStore;
-import org.junit.After;
+import org.junit.AfterClass;
 import org.junit.Assert;
-import org.junit.Before;
+import org.junit.BeforeClass;
 import org.junit.Test;
 import org.junit.runner.RunWith;
 import org.junit.runners.Parameterized;
@@ -77,20 +78,19 @@ public class TestTimelineAuthenticationFilter {
     return Arrays.asList(new Object[][] { { false }, { true } });
   }
 
-  private MiniKdc testMiniKDC;
-  private String keystoresDir;
-  private String sslConfDir;
-  private ApplicationHistoryServer testTimelineServer;
-  private Configuration conf;
-  private TimelineClient client;
-  private boolean withSsl;
+  private static MiniKdc testMiniKDC;
+  private static String keystoresDir;
+  private static String sslConfDir;
+  private static ApplicationHistoryServer testTimelineServer;
+  private static Configuration conf;
+  private static boolean withSsl;
 
   public TestTimelineAuthenticationFilter(boolean withSsl) {
-    this.withSsl = withSsl;
+    TestTimelineAuthenticationFilter.withSsl = withSsl;
   }
 
-  @Before
-  public void setup() {
+  @BeforeClass
+  public static void setup() {
     try {
       testMiniKDC = new MiniKdc(MiniKdc.createConf(), testRootDir);
       testMiniKDC.start();
@@ -127,6 +127,7 @@ public class TestTimelineAuthenticationFilter {
           "localhost:8190");
       conf.set("hadoop.proxyuser.HTTP.hosts", "*");
       conf.set("hadoop.proxyuser.HTTP.users", FOO_USER);
+      conf.setInt(YarnConfiguration.TIMELINE_SERVICE_CLIENT_MAX_RETRIES, 1);
 
       if (withSsl) {
         conf.set(YarnConfiguration.YARN_HTTP_POLICY_KEY,
@@ -146,14 +147,17 @@ public class TestTimelineAuthenticationFilter {
     } catch (Exception e) {
       assertTrue("Couldn't setup TimelineServer", false);
     }
+  }
 
-    client = TimelineClient.createTimelineClient();
+  private TimelineClient createTimelineClientForUGI() {
+    TimelineClient client = TimelineClient.createTimelineClient();
     client.init(conf);
     client.start();
+    return client;
   }
 
-  @After
-  public void tearDown() throws Exception {
+  @AfterClass
+  public static void tearDown() throws Exception {
     if (testMiniKDC != null) {
       testMiniKDC.stop();
     }
@@ -162,10 +166,6 @@ public class TestTimelineAuthenticationFilter {
       testTimelineServer.stop();
     }
 
-    if (client != null) {
-      client.stop();
-    }
-
     if (withSsl) {
       KeyStoreTestUtil.cleanupSSLConfig(keystoresDir, sslConfDir);
       File base = new File(BASEDIR);
@@ -178,6 +178,7 @@ public class TestTimelineAuthenticationFilter {
     KerberosTestUtils.doAs(HTTP_USER + "/localhost", new Callable<Void>() {
       @Override
       public Void call() throws Exception {
+        TimelineClient client = createTimelineClientForUGI();
         TimelineEntity entityToStore = new TimelineEntity();
         entityToStore.setEntityType(
             TestTimelineAuthenticationFilter.class.getName());
@@ -199,6 +200,7 @@ public class TestTimelineAuthenticationFilter {
     KerberosTestUtils.doAs(HTTP_USER + "/localhost", new Callable<Void>() {
       @Override
       public Void call() throws Exception {
+        TimelineClient client = createTimelineClientForUGI();
         TimelineDomain domainToStore = new TimelineDomain();
         domainToStore.setId(TestTimelineAuthenticationFilter.class.getName());
         domainToStore.setReaders("*");
@@ -215,119 +217,96 @@ public class TestTimelineAuthenticationFilter {
 
   @Test
   public void testDelegationTokenOperations() throws Exception {
-    KerberosTestUtils.doAs(HTTP_USER + "/localhost", new Callable<Void>() {
-      @Override
-      public Void call() throws Exception {
-        // Let HTTP user to get the delegation for itself
-        Token<TimelineDelegationTokenIdentifier> token =
-            client.getDelegationToken(
-                UserGroupInformation.getCurrentUser().getShortUserName());
-        Assert.assertNotNull(token);
-        TimelineDelegationTokenIdentifier tDT = token.decodeIdentifier();
-        Assert.assertNotNull(tDT);
-        Assert.assertEquals(new Text(HTTP_USER), tDT.getOwner());
+    TimelineClient httpUserClient =
+      KerberosTestUtils.doAs(HTTP_USER + "/localhost", new Callable<TimelineClient>() {
+        @Override
+        public TimelineClient call() throws Exception {
+          return createTimelineClientForUGI();
+        }
+      });
+    UserGroupInformation httpUser =
+      KerberosTestUtils.doAs(HTTP_USER + "/localhost", new Callable<UserGroupInformation>() {
+        @Override
+        public UserGroupInformation call() throws Exception {
+          return UserGroupInformation.getCurrentUser();
+        }
+      });
+    // Let HTTP user to get the delegation for itself
+    Token<TimelineDelegationTokenIdentifier> token =
+      httpUserClient.getDelegationToken(httpUser.getShortUserName());
+    Assert.assertNotNull(token);
+    TimelineDelegationTokenIdentifier tDT = token.decodeIdentifier();
+    Assert.assertNotNull(tDT);
+    Assert.assertEquals(new Text(HTTP_USER), tDT.getOwner());
 
-        // Renew token
-        long renewTime1 = client.renewDelegationToken(token);
-        Thread.sleep(100);
-        long renewTime2 = client.renewDelegationToken(token);
-        Assert.assertTrue(renewTime1 < renewTime2);
+    // Renew token
+    long renewTime1 = httpUserClient.renewDelegationToken(token);
+    Thread.sleep(100);
+    long renewTime2 = httpUserClient.renewDelegationToken(token);
+    Assert.assertTrue(renewTime1 < renewTime2);
 
-        // Cancel token
-        client.cancelDelegationToken(token);
-        // Renew should not be successful because the token is canceled
-        try {
-          client.renewDelegationToken(token);
-          Assert.fail();
-        } catch (Exception e) {
-          Assert.assertTrue(e.getMessage().contains(
-              "Renewal request for unknown token"));
-        }
+    // Cancel token
+    httpUserClient.cancelDelegationToken(token);
+    // Renew should not be successful because the token is canceled
+    try {
+      httpUserClient.renewDelegationToken(token);
+      Assert.fail();
+    } catch (Exception e) {
+      Assert.assertTrue(e.getMessage().contains(
+            "Renewal request for unknown token"));
+    }
 
-        // Let HTTP user to get the delegation token for FOO user
-        UserGroupInformation fooUgi = UserGroupInformation.createProxyUser(
-            FOO_USER, UserGroupInformation.getCurrentUser());
-        token = fooUgi.doAs(
-            new PrivilegedExceptionAction<Token<TimelineDelegationTokenIdentifier>>() {
+    // Let HTTP user to get the delegation token for FOO user
+    UserGroupInformation fooUgi = UserGroupInformation.createProxyUser(
+        FOO_USER, httpUser);
+    TimelineClient fooUserClient = fooUgi.doAs(
+        new PrivilegedExceptionAction<TimelineClient>() {
           @Override
-          public Token<TimelineDelegationTokenIdentifier> run()
-              throws Exception {
-            return client.getDelegationToken(
-                UserGroupInformation.getCurrentUser().getShortUserName());
+          public TimelineClient run() throws Exception {
+            return createTimelineClientForUGI();
           }
         });
-        Assert.assertNotNull(token);
-        tDT = token.decodeIdentifier();
-        Assert.assertNotNull(tDT);
-        Assert.assertEquals(new Text(FOO_USER), tDT.getOwner());
-        Assert.assertEquals(new Text(HTTP_USER), tDT.getRealUser());
+    token = fooUserClient.getDelegationToken(httpUser.getShortUserName());
+    Assert.assertNotNull(token);
+    tDT = token.decodeIdentifier();
+    Assert.assertNotNull(tDT);
+    Assert.assertEquals(new Text(FOO_USER), tDT.getOwner());
+    Assert.assertEquals(new Text(HTTP_USER), tDT.getRealUser());
 
-        // Renew token
-        final Token<TimelineDelegationTokenIdentifier> tokenToRenew = token;
-        renewTime1 = fooUgi.doAs(
-            new PrivilegedExceptionAction<Long>() {
-          @Override
-          public Long run() throws Exception {
-            return client.renewDelegationToken(tokenToRenew);
-          }
-        });
-        renewTime2 = fooUgi.doAs(
-            new PrivilegedExceptionAction<Long>() {
-          @Override
-          public Long run() throws Exception {
-            return client.renewDelegationToken(tokenToRenew);
-          }
-        });
-        Assert.assertTrue(renewTime1 < renewTime2);
+    // Renew token as the renewer
+    final Token<TimelineDelegationTokenIdentifier> tokenToRenew = token;
+    renewTime1 = httpUserClient.renewDelegationToken(tokenToRenew);
+    renewTime2 = httpUserClient.renewDelegationToken(tokenToRenew);
+    Assert.assertTrue(renewTime1 < renewTime2);
 
-        // Cancel token
-        fooUgi.doAs(
-            new PrivilegedExceptionAction<Void>() {
-          @Override
-          public Void run() throws Exception {
-            client.cancelDelegationToken(tokenToRenew);
-            return null;
-          }
-        });
-        // Renew should not be successful because the token is canceled
-        try {
-          fooUgi.doAs(
-              new PrivilegedExceptionAction<Void>() {
-            @Override
-            public Void run() throws Exception {
-              client.renewDelegationToken(tokenToRenew);
-              return null;
-            }
-          });
-          Assert.fail();
-        } catch (Exception e) {
-          Assert.assertTrue(e.getMessage().contains(
-              "Renewal request for unknown token"));
-        }
+    // Cancel token
+    fooUserClient.cancelDelegationToken(tokenToRenew);
+
+    // Renew should not be successful because the token is canceled
+    try {
+      httpUserClient.renewDelegationToken(tokenToRenew);
+      Assert.fail();
+    } catch (Exception e) {
+      Assert.assertTrue(
+          e.getMessage().contains("Renewal request for unknown token"));
+    }
 
-        // Let HTTP user to get the delegation token for BAR user
-        UserGroupInformation barUgi = UserGroupInformation.createProxyUser(
-            BAR_USER, UserGroupInformation.getCurrentUser());
-        token = barUgi.doAs(
-            new PrivilegedExceptionAction<Token<TimelineDelegationTokenIdentifier>>() {
+    // Let HTTP user to get the delegation token for BAR user
+    UserGroupInformation barUgi = UserGroupInformation.createProxyUser(
+        BAR_USER, httpUser);
+    TimelineClient barUserClient = barUgi.doAs(
+        new PrivilegedExceptionAction<TimelineClient>() {
           @Override
-          public Token<TimelineDelegationTokenIdentifier> run()
-              throws Exception {
-            try {
-              Token<TimelineDelegationTokenIdentifier> token =
-                  client.getDelegationToken(
-                      UserGroupInformation.getCurrentUser().getShortUserName());
-              Assert.fail();
-              return token;
-            } catch (Exception e) {
-              Assert.assertTrue(e instanceof AuthorizationException);
-              return null;
-            }
+          public TimelineClient run() {
+            return createTimelineClientForUGI();
           }
         });
-        return null;
-      }
-    });
-  }
 
+    try {
+      barUserClient.getDelegationToken(httpUser.getShortUserName());
+      Assert.fail();
+    } catch (Exception e) {
+      Assert.assertTrue(e.getCause() instanceof AuthorizationException || e.getCause() instanceof AuthenticationException);
+    }
+  }
 }