Переглянути джерело

YARN-2049. Added delegation-token support for the Timeline Server. Contributed by Zhijie Shen.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/trunk@1597130 13f79535-47bb-0310-9956-ffa450edef68
Vinod Kumar Vavilapalli 11 роки тому
батько
коміт
b867b69556
24 змінених файлів з 1335 додано та 35 видалено
  1. 3 0
      hadoop-yarn-project/CHANGES.txt
  2. 63 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timeline/TimelineDelegationTokenResponse.java
  3. 0 8
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/conf/YarnConfiguration.java
  4. 0 2
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/TimelineClient.java
  5. 252 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/impl/TimelineAuthenticator.java
  6. 83 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/ClientTimelineSecurityInfo.java
  7. 44 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineAuthenticationConsts.java
  8. 63 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineDelegationTokenIdentifier.java
  9. 55 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineDelegationTokenOperation.java
  10. 62 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineDelegationTokenSelector.java
  11. 2 6
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/util/timeline/TimelineUtils.java
  12. 6 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/webapp/YarnJacksonJaxbJsonProvider.java
  13. 1 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/resources/META-INF/services/org.apache.hadoop.security.SecurityInfo
  14. 1 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenIdentifier
  15. 1 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenRenewer
  16. 41 13
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/ApplicationHistoryServer.java
  17. 48 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineAuthenticationFilter.java
  18. 127 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineAuthenticationFilterInitializer.java
  19. 236 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineClientAuthenticationService.java
  20. 189 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineDelegationTokenSecretManagerService.java
  21. 51 4
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/webapp/AHSWebApp.java
  22. 2 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryClientService.java
  23. 3 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryServer.java
  24. 2 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-tests/src/test/java/org/apache/hadoop/yarn/server/MiniYARNCluster.java

+ 3 - 0
hadoop-yarn-project/CHANGES.txt

@@ -90,6 +90,9 @@ Release 2.5.0 - UNRELEASED
 
     YARN-2017. Merged some of the common scheduler code. (Jian He via vinodkv)
 
+    YARN-2049. Added delegation-token support for the Timeline Server. (Zhijie
+    Shen via vinodkv)
+
   OPTIMIZATIONS
 
   BUG FIXES 

+ 63 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timeline/TimelineDelegationTokenResponse.java

@@ -0,0 +1,63 @@
+/**
+ * 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.hadoop.yarn.api.records.timeline;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlElement;
+import javax.xml.bind.annotation.XmlRootElement;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+
+/**
+ * The response of delegation token related request
+ */
+@XmlRootElement(name = "delegationtoken")
+@XmlAccessorType(XmlAccessType.NONE)
+@Public
+@Unstable
+public class TimelineDelegationTokenResponse {
+
+  private String type;
+  private Object content;
+
+  public TimelineDelegationTokenResponse() {
+
+  }
+
+  @XmlElement(name = "type")
+  public String getType() {
+    return type;
+  }
+
+  public void setType(String type) {
+    this.type = type;
+  }
+
+  @XmlElement(name = "content")
+  public Object getContent() {
+    return content;
+  }
+
+  public void setContent(Object content) {
+    this.content = content;
+  }
+
+}

+ 0 - 8
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/conf/YarnConfiguration.java

@@ -1136,14 +1136,6 @@ public class YarnConfiguration extends Configuration {
   public static final String DEFAULT_TIMELINE_SERVICE_WEBAPP_HTTPS_ADDRESS =
       "0.0.0.0:" + DEFAULT_TIMELINE_SERVICE_WEBAPP_HTTPS_PORT;
 
-  /**The kerberos principal to be used for spnego filter for timeline service.*/
-  public static final String TIMELINE_SERVICE_WEBAPP_SPNEGO_USER_NAME_KEY =
-      TIMELINE_SERVICE_PREFIX + "webapp.spnego-principal";
-
-  /**The kerberos keytab to be used for spnego filter for timeline service.*/
-  public static final String TIMELINE_SERVICE_WEBAPP_SPNEGO_KEYTAB_FILE_KEY =
-      TIMELINE_SERVICE_PREFIX + "webapp.spnego-keytab-file";
-
   /** Timeline service store class */
   public static final String TIMELINE_SERVICE_STORE =
       TIMELINE_SERVICE_PREFIX + "store-class";

+ 0 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/TimelineClient.java

@@ -32,8 +32,6 @@ import org.apache.hadoop.yarn.exceptions.YarnException;
 /**
  * A client library that can be used to post some information in terms of a
  * number of conceptual entities.
- * 
- * @See Entity
  */
 @Public
 @Unstable

+ 252 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/impl/TimelineAuthenticator.java

@@ -0,0 +1,252 @@
+/**
+ * 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.hadoop.yarn.client.api.impl;
+
+import java.io.IOException;
+import java.lang.reflect.Constructor;
+import java.net.HttpURLConnection;
+import java.net.URL;
+import java.net.URLEncoder;
+import java.text.MessageFormat;
+import java.util.HashMap;
+import java.util.Map;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.security.authentication.client.AuthenticatedURL;
+import org.apache.hadoop.security.authentication.client.AuthenticationException;
+import org.apache.hadoop.security.authentication.client.Authenticator;
+import org.apache.hadoop.security.authentication.client.KerberosAuthenticator;
+import org.apache.hadoop.security.token.Token;
+import org.apache.hadoop.yarn.api.records.timeline.TimelineDelegationTokenResponse;
+import org.apache.hadoop.yarn.security.client.TimelineDelegationTokenIdentifier;
+import org.apache.hadoop.yarn.security.client.TimelineDelegationTokenOperation;
+import org.apache.hadoop.yarn.security.client.TimelineAuthenticationConsts;
+import org.apache.hadoop.yarn.webapp.YarnJacksonJaxbJsonProvider;
+import org.codehaus.jackson.JsonNode;
+import org.codehaus.jackson.map.ObjectMapper;
+
+/**
+ * A <code>KerberosAuthenticator</code> subclass that fallback to
+ * {@link TimelineAuthenticationConsts}.
+ */
+@Private
+@Unstable
+public class TimelineAuthenticator extends KerberosAuthenticator {
+
+  private static ObjectMapper mapper;
+
+  static {
+    mapper = new ObjectMapper();
+    YarnJacksonJaxbJsonProvider.configObjectMapper(mapper);
+  }
+
+  /**
+   * Returns the fallback authenticator if the server does not use Kerberos
+   * SPNEGO HTTP authentication.
+   * 
+   * @return a {@link TimelineAuthenticationConsts} instance.
+   */
+  @Override
+  protected Authenticator getFallBackAuthenticator() {
+    return new TimelineAuthenticator();
+  }
+
+  public static void injectDelegationToken(Map<String, String> params,
+      Token<?> dtToken)
+      throws IOException {
+    if (dtToken != null) {
+      params.put(TimelineAuthenticationConsts.DELEGATION_PARAM,
+          dtToken.encodeToUrlString());
+    }
+  }
+
+  private boolean hasDelegationToken(URL url) {
+    return url.getQuery().contains(
+        TimelineAuthenticationConsts.DELEGATION_PARAM + "=");
+  }
+
+  @Override
+  public void authenticate(URL url, AuthenticatedURL.Token token)
+      throws IOException, AuthenticationException {
+    if (!hasDelegationToken(url)) {
+      super.authenticate(url, token);
+    }
+  }
+
+  public static Token<TimelineDelegationTokenIdentifier> getDelegationToken(
+      URL url, AuthenticatedURL.Token token, String renewer) throws IOException {
+    TimelineDelegationTokenOperation op =
+        TimelineDelegationTokenOperation.GETDELEGATIONTOKEN;
+    Map<String, String> params = new HashMap<String, String>();
+    params.put(TimelineAuthenticationConsts.OP_PARAM, op.toString());
+    params.put(TimelineAuthenticationConsts.RENEWER_PARAM, renewer);
+    url = appendParams(url, params);
+    AuthenticatedURL aUrl =
+        new AuthenticatedURL(new TimelineAuthenticator());
+    try {
+      HttpURLConnection conn = aUrl.openConnection(url, token);
+      conn.setRequestMethod(op.getHttpMethod());
+      TimelineDelegationTokenResponse dtRes = validateAndParseResponse(conn);
+      if (!dtRes.getType().equals(
+          TimelineAuthenticationConsts.DELEGATION_TOKEN_URL)) {
+        throw new IOException("The response content is not expected: "
+            + dtRes.getContent());
+      }
+      String tokenStr = dtRes.getContent().toString();
+      Token<TimelineDelegationTokenIdentifier> dToken =
+          new Token<TimelineDelegationTokenIdentifier>();
+      dToken.decodeFromUrlString(tokenStr);
+      return dToken;
+    } catch (AuthenticationException ex) {
+      throw new IOException(ex.toString(), ex);
+    }
+  }
+
+  public static long renewDelegationToken(URL url,
+      AuthenticatedURL.Token token,
+      Token<TimelineDelegationTokenIdentifier> dToken) throws IOException {
+    Map<String, String> params = new HashMap<String, String>();
+    params.put(TimelineAuthenticationConsts.OP_PARAM,
+        TimelineDelegationTokenOperation.RENEWDELEGATIONTOKEN.toString());
+    params.put(TimelineAuthenticationConsts.TOKEN_PARAM,
+        dToken.encodeToUrlString());
+    url = appendParams(url, params);
+    AuthenticatedURL aUrl =
+        new AuthenticatedURL(new TimelineAuthenticator());
+    try {
+      HttpURLConnection conn = aUrl.openConnection(url, token);
+      conn.setRequestMethod(
+          TimelineDelegationTokenOperation.RENEWDELEGATIONTOKEN.getHttpMethod());
+      TimelineDelegationTokenResponse dtRes = validateAndParseResponse(conn);
+      if (!dtRes.getType().equals(
+          TimelineAuthenticationConsts.DELEGATION_TOKEN_EXPIRATION_TIME)) {
+        throw new IOException("The response content is not expected: "
+            + dtRes.getContent());
+      }
+      return Long.valueOf(dtRes.getContent().toString());
+    } catch (AuthenticationException ex) {
+      throw new IOException(ex.toString(), ex);
+    }
+  }
+
+  public static void cancelDelegationToken(URL url,
+      AuthenticatedURL.Token token,
+      Token<TimelineDelegationTokenIdentifier> dToken) throws IOException {
+    Map<String, String> params = new HashMap<String, String>();
+    params.put(TimelineAuthenticationConsts.OP_PARAM,
+        TimelineDelegationTokenOperation.CANCELDELEGATIONTOKEN.toString());
+    params.put(TimelineAuthenticationConsts.TOKEN_PARAM,
+        dToken.encodeToUrlString());
+    url = appendParams(url, params);
+    AuthenticatedURL aUrl =
+        new AuthenticatedURL(new TimelineAuthenticator());
+    try {
+      HttpURLConnection conn = aUrl.openConnection(url, token);
+      conn.setRequestMethod(TimelineDelegationTokenOperation.CANCELDELEGATIONTOKEN
+          .getHttpMethod());
+      validateAndParseResponse(conn);
+    } catch (AuthenticationException ex) {
+      throw new IOException(ex.toString(), ex);
+    }
+  }
+
+  /**
+   * Convenience method that appends parameters an HTTP <code>URL</code>.
+   * 
+   * @param url
+   *          the url.
+   * @param params
+   *          the query string parameters.
+   * 
+   * @return a <code>URL</code>
+   * 
+   * @throws IOException
+   *           thrown if an IO error occurs.
+   */
+  public static URL appendParams(URL url, Map<String, String> params)
+      throws IOException {
+    StringBuilder sb = new StringBuilder();
+    sb.append(url);
+    String separator = url.toString().contains("?") ? "&" : "?";
+    for (Map.Entry<String, String> entry : params.entrySet()) {
+      sb.append(separator).append(entry.getKey()).append("=").
+          append(URLEncoder.encode(entry.getValue(), "UTF8"));
+      separator = "&";
+    }
+    return new URL(sb.toString());
+  }
+
+  /**
+   * Validates the response of an <code>HttpURLConnection</code>. If the current
+   * status code is not 200, it will throw an exception with a detail message
+   * using Server side error messages if available. Otherwise,
+   * {@link TimelineDelegationTokenResponse} will be parsed and returned.
+   * 
+   * @param conn
+   *          the <code>HttpURLConnection</code>.
+   * @return
+   * @throws IOException
+   *           thrown if the current status code is not 200 or the JSON response
+   *           cannot be parsed correctly
+   */
+  private static TimelineDelegationTokenResponse validateAndParseResponse(
+      HttpURLConnection conn) throws IOException {
+    int status = conn.getResponseCode();
+    JsonNode json = mapper.readTree(conn.getInputStream());
+    if (status == HttpURLConnection.HTTP_OK) {
+      return mapper.readValue(json, TimelineDelegationTokenResponse.class);
+    } else {
+      // If the status code is not 200, some thing wrong should happen at the
+      // server side, the JSON content is going to contain exception details.
+      // We can use the JSON content to reconstruct the exception object.
+      try {
+        String message =
+            json.get(TimelineAuthenticationConsts.ERROR_MESSAGE_JSON)
+                .getTextValue();
+        String exception =
+            json.get(TimelineAuthenticationConsts.ERROR_EXCEPTION_JSON)
+                .getTextValue();
+        String className =
+            json.get(TimelineAuthenticationConsts.ERROR_CLASSNAME_JSON)
+                .getTextValue();
+
+        try {
+          ClassLoader cl = TimelineAuthenticator.class.getClassLoader();
+          Class<?> klass = cl.loadClass(className);
+          Constructor<?> constr = klass.getConstructor(String.class);
+          throw (IOException) constr.newInstance(message);
+        } catch (IOException ex) {
+          throw ex;
+        } catch (Exception ex) {
+          throw new IOException(MessageFormat.format("{0} - {1}", exception,
+              message));
+        }
+      } catch (IOException ex) {
+        if (ex.getCause() instanceof IOException) {
+          throw (IOException) ex.getCause();
+        }
+        throw new IOException(
+            MessageFormat.format("HTTP status [{0}], {1}",
+                status, conn.getResponseMessage()));
+      }
+    }
+  }
+
+}

+ 83 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/ClientTimelineSecurityInfo.java

@@ -0,0 +1,83 @@
+/**
+ * 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.hadoop.yarn.security.client;
+
+import java.lang.annotation.Annotation;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.security.KerberosInfo;
+import org.apache.hadoop.security.SecurityInfo;
+import org.apache.hadoop.security.token.TokenIdentifier;
+import org.apache.hadoop.security.token.TokenInfo;
+import org.apache.hadoop.security.token.TokenSelector;
+import org.apache.hadoop.yarn.api.ApplicationHistoryProtocolPB;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+
+
+@Public
+@Unstable
+public class ClientTimelineSecurityInfo extends SecurityInfo {
+  @Override
+  public KerberosInfo getKerberosInfo(Class<?> protocol, Configuration conf) {
+    if (!protocol
+        .equals(ApplicationHistoryProtocolPB.class)) {
+      return null;
+    }
+    return new KerberosInfo() {
+
+      @Override
+      public Class<? extends Annotation> annotationType() {
+        return null;
+      }
+
+      @Override
+      public String serverPrincipal() {
+        return YarnConfiguration.TIMELINE_SERVICE_PRINCIPAL;
+      }
+
+      @Override
+      public String clientPrincipal() {
+        return null;
+      }
+    };
+  }
+
+  @Override
+  public TokenInfo getTokenInfo(Class<?> protocol, Configuration conf) {
+    if (!protocol
+        .equals(ApplicationHistoryProtocolPB.class)) {
+      return null;
+    }
+    return new TokenInfo() {
+
+      @Override
+      public Class<? extends Annotation> annotationType() {
+        return null;
+      }
+
+      @Override
+      public Class<? extends TokenSelector<? extends TokenIdentifier>>
+          value() {
+        return TimelineDelegationTokenSelector.class;
+      }
+    };
+  }
+}

+ 44 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineAuthenticationConsts.java

@@ -0,0 +1,44 @@
+/**
+ * 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.hadoop.yarn.security.client;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+
+/**
+ * The constants that are going to be used by the timeline Kerberos + delegation
+ * token authentication.
+ */
+
+@Private
+@Unstable
+public class TimelineAuthenticationConsts {
+
+  public static final String ERROR_EXCEPTION_JSON = "exception";
+  public static final String ERROR_CLASSNAME_JSON = "javaClassName";
+  public static final String ERROR_MESSAGE_JSON = "message";
+
+  public static final String OP_PARAM = "op";
+  public static final String DELEGATION_PARAM = "delegation";
+  public static final String TOKEN_PARAM = "token";
+  public static final String RENEWER_PARAM = "renewer";
+  public static final String DELEGATION_TOKEN_URL = "url";
+  public static final String DELEGATION_TOKEN_EXPIRATION_TIME =
+      "expirationTime";
+}

+ 63 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineDelegationTokenIdentifier.java

@@ -0,0 +1,63 @@
+/**
+ * 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.hadoop.yarn.security.client;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.security.token.Token;
+import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenIdentifier;
+
+@Public
+@Unstable
+public class TimelineDelegationTokenIdentifier extends AbstractDelegationTokenIdentifier {
+
+  public static final Text KIND_NAME = new Text("TIMELINE_DELEGATION_TOKEN");
+
+  public TimelineDelegationTokenIdentifier() {
+
+  }
+
+  /**
+   * Create a new timeline delegation token identifier
+   *
+   * @param owner the effective username of the token owner
+   * @param renewer the username of the renewer
+   * @param realUser the real username of the token owner
+   */
+  public TimelineDelegationTokenIdentifier(Text owner, Text renewer,
+      Text realUser) {
+    super(owner, renewer, realUser);
+  }
+
+  @Override
+  public Text getKind() {
+    return KIND_NAME;
+  }
+
+  @InterfaceAudience.Private
+  public static class Renewer extends Token.TrivialRenewer {
+    @Override
+    protected Text getKind() {
+      return KIND_NAME;
+    }
+  }
+
+}

+ 55 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineDelegationTokenOperation.java

@@ -0,0 +1,55 @@
+/**
+ * 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.hadoop.yarn.security.client;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.http.client.methods.HttpGet;
+import org.apache.http.client.methods.HttpPut;
+
+/**
+ * DelegationToken operations.
+ */
+@Unstable
+@Private
+public enum TimelineDelegationTokenOperation {
+  // TODO: need think about which ops can be done without kerberos
+  // credentials, for safety, we enforces all need kerberos credentials now.
+  GETDELEGATIONTOKEN(HttpGet.METHOD_NAME, true),
+  RENEWDELEGATIONTOKEN(HttpPut.METHOD_NAME, true),
+  CANCELDELEGATIONTOKEN(HttpPut.METHOD_NAME, true);
+
+  private String httpMethod;
+  private boolean requiresKerberosCredentials;
+
+  private TimelineDelegationTokenOperation(String httpMethod,
+      boolean requiresKerberosCredentials) {
+    this.httpMethod = httpMethod;
+    this.requiresKerberosCredentials = requiresKerberosCredentials;
+  }
+
+  public String getHttpMethod() {
+    return httpMethod;
+  }
+
+  public boolean requiresKerberosCredentials() {
+    return requiresKerberosCredentials;
+  }
+
+}

+ 62 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/security/client/TimelineDelegationTokenSelector.java

@@ -0,0 +1,62 @@
+/**
+ * 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.hadoop.yarn.security.client;
+
+import java.util.Collection;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.security.token.Token;
+import org.apache.hadoop.security.token.TokenIdentifier;
+import org.apache.hadoop.security.token.TokenSelector;
+
+@Public
+@Unstable
+public class TimelineDelegationTokenSelector
+    implements TokenSelector<TimelineDelegationTokenIdentifier> {
+
+  private static final Log LOG = LogFactory
+      .getLog(TimelineDelegationTokenSelector.class);
+
+  @SuppressWarnings("unchecked")
+  public Token<TimelineDelegationTokenIdentifier> selectToken(Text service,
+      Collection<Token<? extends TokenIdentifier>> tokens) {
+    if (service == null) {
+      return null;
+    }
+    if (LOG.isDebugEnabled()) {
+      LOG.debug("Looking for a token with service " + service.toString());
+    }
+    for (Token<? extends TokenIdentifier> token : tokens) {
+      if (LOG.isDebugEnabled()) {
+        LOG.debug("Token kind is " + token.getKind().toString()
+            + " and the token's service name is " + token.getService());
+      }
+      if (TimelineDelegationTokenIdentifier.KIND_NAME.equals(token.getKind())
+          && service.equals(token.getService())) {
+        return (Token<TimelineDelegationTokenIdentifier>) token;
+      }
+    }
+    return null;
+  }
+
+}

+ 2 - 6
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/util/timeline/TimelineUtils.java

@@ -22,12 +22,10 @@ import java.io.IOException;
 
 import org.apache.hadoop.classification.InterfaceAudience.Public;
 import org.apache.hadoop.classification.InterfaceStability.Evolving;
+import org.apache.hadoop.yarn.webapp.YarnJacksonJaxbJsonProvider;
 import org.codehaus.jackson.JsonGenerationException;
-import org.codehaus.jackson.map.AnnotationIntrospector;
 import org.codehaus.jackson.map.JsonMappingException;
 import org.codehaus.jackson.map.ObjectMapper;
-import org.codehaus.jackson.map.annotate.JsonSerialize.Inclusion;
-import org.codehaus.jackson.xc.JaxbAnnotationIntrospector;
 
 /**
  * The helper class for the timeline module.
@@ -41,9 +39,7 @@ public class TimelineUtils {
 
   static {
     mapper = new ObjectMapper();
-    AnnotationIntrospector introspector = new JaxbAnnotationIntrospector();
-    mapper.setAnnotationIntrospector(introspector);
-    mapper.setSerializationInclusion(Inclusion.NON_NULL);
+    YarnJacksonJaxbJsonProvider.configObjectMapper(mapper);
   }
 
   /**

+ 6 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/webapp/YarnJacksonJaxbJsonProvider.java

@@ -49,9 +49,14 @@ public class YarnJacksonJaxbJsonProvider extends JacksonJaxbJsonProvider {
   @Override
   public ObjectMapper locateMapper(Class<?> type, MediaType mediaType) {
     ObjectMapper mapper = super.locateMapper(type, mediaType);
+    configObjectMapper(mapper);
+    return mapper;
+  }
+
+  public static void configObjectMapper(ObjectMapper mapper) {
     AnnotationIntrospector introspector = new JaxbAnnotationIntrospector();
     mapper.setAnnotationIntrospector(introspector);
     mapper.setSerializationInclusion(Inclusion.NON_NULL);
-    return mapper;
   }
+
 }

+ 1 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/resources/META-INF/services/org.apache.hadoop.security.SecurityInfo

@@ -12,6 +12,7 @@
 #   limitations under the License.
 #
 org.apache.hadoop.yarn.security.client.ClientRMSecurityInfo
+org.apache.hadoop.yarn.security.client.ClientTimelineSecurityInfo
 org.apache.hadoop.yarn.security.ContainerManagerSecurityInfo
 org.apache.hadoop.yarn.security.SchedulerSecurityInfo
 org.apache.hadoop.yarn.security.admin.AdminSecurityInfo

+ 1 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenIdentifier

@@ -15,4 +15,5 @@ org.apache.hadoop.yarn.security.ContainerTokenIdentifier
 org.apache.hadoop.yarn.security.AMRMTokenIdentifier
 org.apache.hadoop.yarn.security.client.ClientToAMTokenIdentifier
 org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier
+org.apache.hadoop.yarn.security.client.TimelineDelegationTokenIdentifier
 org.apache.hadoop.yarn.security.NMTokenIdentifier

+ 1 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/resources/META-INF/services/org.apache.hadoop.security.token.TokenRenewer

@@ -14,3 +14,4 @@
 org.apache.hadoop.yarn.security.AMRMTokenIdentifier$Renewer
 org.apache.hadoop.yarn.security.ContainerTokenIdentifier$Renewer
 org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier$Renewer
+org.apache.hadoop.yarn.security.client.TimelineDelegationTokenIdentifier$Renewer

+ 41 - 13
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/ApplicationHistoryServer.java

@@ -28,6 +28,7 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
 import org.apache.hadoop.metrics2.source.JvmMetrics;
 import org.apache.hadoop.security.SecurityUtil;
+import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.service.CompositeService;
 import org.apache.hadoop.service.Service;
 import org.apache.hadoop.util.ExitUtil;
@@ -39,6 +40,8 @@ import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.LeveldbTimelineStore;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.TimelineStore;
+import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.security.TimelineAuthenticationFilterInitializer;
+import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.security.TimelineDelegationTokenSecretManagerService;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.webapp.AHSWebApp;
 import org.apache.hadoop.yarn.webapp.WebApp;
 import org.apache.hadoop.yarn.webapp.WebApps;
@@ -56,10 +59,11 @@ public class ApplicationHistoryServer extends CompositeService {
   private static final Log LOG = LogFactory
     .getLog(ApplicationHistoryServer.class);
 
-  ApplicationHistoryClientService ahsClientService;
-  ApplicationHistoryManager historyManager;
-  TimelineStore timelineStore;
-  private WebApp webApp;
+  protected ApplicationHistoryClientService ahsClientService;
+  protected ApplicationHistoryManager historyManager;
+  protected TimelineStore timelineStore;
+  protected TimelineDelegationTokenSecretManagerService secretManagerService;
+  protected WebApp webApp;
 
   public ApplicationHistoryServer() {
     super(ApplicationHistoryServer.class.getName());
@@ -73,6 +77,8 @@ public class ApplicationHistoryServer extends CompositeService {
     addService((Service) historyManager);
     timelineStore = createTimelineStore(conf);
     addIfService(timelineStore);
+    secretManagerService = createTimelineDelegationTokenSecretManagerService(conf);
+    addService(secretManagerService);
 
     DefaultMetricsSystem.initialize("ApplicationHistoryServer");
     JvmMetrics.initSingleton("ApplicationHistoryServer", null);
@@ -158,21 +164,43 @@ public class ApplicationHistoryServer extends CompositeService {
         TimelineStore.class), conf);
   }
 
+  protected TimelineDelegationTokenSecretManagerService
+      createTimelineDelegationTokenSecretManagerService(Configuration conf) {
+    return new TimelineDelegationTokenSecretManagerService();
+  }
+
   protected void startWebApp() {
-    String bindAddress = WebAppUtils.getAHSWebAppURLWithoutScheme(getConfig());
+    Configuration conf = getConfig();
+    // Play trick to make the customized filter will only be loaded by the
+    // timeline server when security is enabled and Kerberos authentication
+    // is used.
+    if (UserGroupInformation.isSecurityEnabled()
+        && conf
+            .get(TimelineAuthenticationFilterInitializer.PREFIX + "type", "")
+            .equals("kerberos")) {
+      String initializers = conf.get("hadoop.http.filter.initializers");
+      initializers =
+          initializers == null || initializers.length() == 0 ? "" : ","
+              + initializers;
+      if (!initializers.contains(
+          TimelineAuthenticationFilterInitializer.class.getName())) {
+        conf.set("hadoop.http.filter.initializers",
+            TimelineAuthenticationFilterInitializer.class.getName()
+            + initializers);
+      }
+    }
+    String bindAddress = WebAppUtils.getAHSWebAppURLWithoutScheme(conf);
     LOG.info("Instantiating AHSWebApp at " + bindAddress);
     try {
+      AHSWebApp ahsWebApp = AHSWebApp.getInstance();
+      ahsWebApp.setApplicationHistoryManager(historyManager);
+      ahsWebApp.setTimelineStore(timelineStore);
+      ahsWebApp.setTimelineDelegationTokenSecretManagerService(secretManagerService);
       webApp =
           WebApps
             .$for("applicationhistory", ApplicationHistoryClientService.class,
-              ahsClientService, "ws")
-            .with(getConfig())
-            .withHttpSpnegoPrincipalKey(
-              YarnConfiguration.TIMELINE_SERVICE_WEBAPP_SPNEGO_USER_NAME_KEY)
-            .withHttpSpnegoKeytabKey(
-              YarnConfiguration.TIMELINE_SERVICE_WEBAPP_SPNEGO_KEYTAB_FILE_KEY)
-            .at(bindAddress)
-            .start(new AHSWebApp(historyManager, timelineStore));
+                ahsClientService, "ws")
+            .with(conf).at(bindAddress).start(ahsWebApp);
     } catch (Exception e) {
       String msg = "AHSWebApp failed to start.";
       LOG.error(msg, e);

+ 48 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineAuthenticationFilter.java

@@ -0,0 +1,48 @@
+/**
+ * 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.hadoop.yarn.server.applicationhistoryservice.timeline.security;
+
+import java.util.Properties;
+
+import javax.servlet.FilterConfig;
+import javax.servlet.ServletException;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.security.authentication.server.AuthenticationFilter;
+
+@Private
+@Unstable
+public class TimelineAuthenticationFilter extends AuthenticationFilter {
+
+  @Override
+  protected Properties getConfiguration(String configPrefix,
+      FilterConfig filterConfig) throws ServletException {
+    // In yarn-site.xml, we can simply set type to "kerberos". However, we need
+    // to replace the name here to use the customized Kerberos + DT service
+    // instead of the standard Kerberos handler.
+    Properties properties = super.getConfiguration(configPrefix, filterConfig);
+    if (properties.getProperty(AUTH_TYPE).equals("kerberos")) {
+      properties.setProperty(
+          AUTH_TYPE, TimelineClientAuthenticationService.class.getName());
+    }
+    return properties;
+  }
+
+}

+ 127 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineAuthenticationFilterInitializer.java

@@ -0,0 +1,127 @@
+/**
+ * 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.hadoop.yarn.server.applicationhistoryservice.timeline.security;
+
+import java.io.FileReader;
+import java.io.IOException;
+import java.io.Reader;
+import java.util.HashMap;
+import java.util.Map;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.http.FilterContainer;
+import org.apache.hadoop.http.FilterInitializer;
+import org.apache.hadoop.http.HttpServer2;
+import org.apache.hadoop.security.SecurityUtil;
+
+/**
+ * <p>
+ * Initializes {@link TimelineAuthenticationFilter} which provides support for
+ * Kerberos HTTP SPNEGO authentication.
+ * <p/>
+ * <p>
+ * It enables Kerberos HTTP SPNEGO plus delegation token authentication for the
+ * timeline server.
+ * <p/>
+ * Refer to the <code>core-default.xml</code> file, after the comment 'HTTP
+ * Authentication' for details on the configuration options. All related
+ * configuration properties have 'hadoop.http.authentication.' as prefix.
+ */
+public class TimelineAuthenticationFilterInitializer extends FilterInitializer {
+
+  /**
+   * The configuration prefix of timeline Kerberos + DT authentication
+   */
+  public static final String PREFIX = "yarn.timeline-service.http.authentication.";
+
+  private static final String SIGNATURE_SECRET_FILE =
+      TimelineAuthenticationFilter.SIGNATURE_SECRET + ".file";
+
+  /**
+   * <p>
+   * Initializes {@link TimelineAuthenticationFilter}
+   * <p/>
+   * <p>
+   * Propagates to {@link TimelineAuthenticationFilter} configuration all YARN
+   * configuration properties prefixed with
+   * "yarn.timeline-service.authentication."
+   * </p>
+   * 
+   * @param container
+   *          The filter container
+   * @param conf
+   *          Configuration for run-time parameters
+   */
+  @Override
+  public void initFilter(FilterContainer container, Configuration conf) {
+    Map<String, String> filterConfig = new HashMap<String, String>();
+
+    // setting the cookie path to root '/' so it is used for all resources.
+    filterConfig.put(TimelineAuthenticationFilter.COOKIE_PATH, "/");
+
+    for (Map.Entry<String, String> entry : conf) {
+      String name = entry.getKey();
+      if (name.startsWith(PREFIX)) {
+        String value = conf.get(name);
+        name = name.substring(PREFIX.length());
+        filterConfig.put(name, value);
+      }
+    }
+
+    String signatureSecretFile = filterConfig.get(SIGNATURE_SECRET_FILE);
+    if (signatureSecretFile != null) {
+      try {
+        StringBuilder secret = new StringBuilder();
+        Reader reader = new FileReader(signatureSecretFile);
+        int c = reader.read();
+        while (c > -1) {
+          secret.append((char) c);
+          c = reader.read();
+        }
+        reader.close();
+        filterConfig
+            .put(TimelineAuthenticationFilter.SIGNATURE_SECRET,
+                secret.toString());
+      } catch (IOException ex) {
+        throw new RuntimeException(
+            "Could not read HTTP signature secret file: "
+                + signatureSecretFile);
+      }
+    }
+
+    // Resolve _HOST into bind address
+    String bindAddress = conf.get(HttpServer2.BIND_ADDRESS);
+    String principal =
+        filterConfig.get(TimelineClientAuthenticationService.PRINCIPAL);
+    if (principal != null) {
+      try {
+        principal = SecurityUtil.getServerPrincipal(principal, bindAddress);
+      } catch (IOException ex) {
+        throw new RuntimeException(
+            "Could not resolve Kerberos principal name: " + ex.toString(), ex);
+      }
+      filterConfig.put(TimelineClientAuthenticationService.PRINCIPAL,
+          principal);
+    }
+
+    container.addGlobalFilter("Timeline Authentication Filter",
+        TimelineAuthenticationFilter.class.getName(),
+        filterConfig);
+  }
+}

+ 236 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineClientAuthenticationService.java

@@ -0,0 +1,236 @@
+/**
+ * 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.hadoop.yarn.server.applicationhistoryservice.timeline.security;
+
+import java.io.IOException;
+import java.io.Writer;
+import java.text.MessageFormat;
+import java.util.HashSet;
+import java.util.Set;
+
+import javax.servlet.http.HttpServletRequest;
+import javax.servlet.http.HttpServletResponse;
+import javax.ws.rs.core.MediaType;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.security.authentication.client.AuthenticationException;
+import org.apache.hadoop.security.authentication.server.AuthenticationToken;
+import org.apache.hadoop.security.authentication.server.KerberosAuthenticationHandler;
+import org.apache.hadoop.security.token.Token;
+import org.apache.hadoop.yarn.api.records.timeline.TimelineDelegationTokenResponse;
+import org.apache.hadoop.yarn.security.client.TimelineDelegationTokenIdentifier;
+import org.apache.hadoop.yarn.security.client.TimelineDelegationTokenOperation;
+import org.apache.hadoop.yarn.security.client.TimelineAuthenticationConsts;
+import org.apache.hadoop.yarn.server.applicationhistoryservice.webapp.AHSWebApp;
+import org.apache.hadoop.yarn.webapp.YarnJacksonJaxbJsonProvider;
+import org.codehaus.jackson.map.ObjectMapper;
+
+/**
+ * Server side <code>AuthenticationHandler</code> that authenticates requests
+ * using the incoming delegation token as a 'delegation' query string parameter.
+ * <p/>
+ * If not delegation token is present in the request it delegates to the
+ * {@link KerberosAuthenticationHandler}
+ */
+@Private
+@Unstable
+public class TimelineClientAuthenticationService
+    extends KerberosAuthenticationHandler {
+
+  public static final String TYPE = "kerberos-dt";
+  private static final Set<String> DELEGATION_TOKEN_OPS = new HashSet<String>();
+  private static final String OP_PARAM = "op";
+  private static final String ENTER = System.getProperty("line.separator");
+
+  private ObjectMapper mapper;
+
+  static {
+    DELEGATION_TOKEN_OPS.add(
+        TimelineDelegationTokenOperation.GETDELEGATIONTOKEN.toString());
+    DELEGATION_TOKEN_OPS.add(
+        TimelineDelegationTokenOperation.RENEWDELEGATIONTOKEN.toString());
+    DELEGATION_TOKEN_OPS.add(
+        TimelineDelegationTokenOperation.CANCELDELEGATIONTOKEN.toString());
+  }
+
+  public TimelineClientAuthenticationService() {
+    super();
+    mapper = new ObjectMapper();
+    YarnJacksonJaxbJsonProvider.configObjectMapper(mapper);
+  }
+
+  /**
+   * Returns authentication type of the handler.
+   * 
+   * @return <code>delegationtoken-kerberos</code>
+   */
+  @Override
+  public String getType() {
+    return TYPE;
+  }
+
+  @Override
+  public boolean managementOperation(AuthenticationToken token,
+      HttpServletRequest request, HttpServletResponse response)
+      throws IOException, AuthenticationException {
+    boolean requestContinues = true;
+    String op = request.getParameter(OP_PARAM);
+    op = (op != null) ? op.toUpperCase() : null;
+    if (DELEGATION_TOKEN_OPS.contains(op) &&
+        !request.getMethod().equals("OPTIONS")) {
+      TimelineDelegationTokenOperation dtOp =
+          TimelineDelegationTokenOperation.valueOf(op);
+      if (dtOp.getHttpMethod().equals(request.getMethod())) {
+        if (dtOp.requiresKerberosCredentials() && token == null) {
+          response.sendError(HttpServletResponse.SC_UNAUTHORIZED,
+              MessageFormat.format(
+                  "Operation [{0}] requires SPNEGO authentication established",
+                  dtOp));
+          requestContinues = false;
+        } else {
+          TimelineDelegationTokenSecretManagerService secretManager =
+              AHSWebApp.getInstance()
+                  .getTimelineDelegationTokenSecretManagerService();
+          try {
+            TimelineDelegationTokenResponse res = null;
+            switch (dtOp) {
+              case GETDELEGATIONTOKEN:
+                UserGroupInformation ownerUGI =
+                    UserGroupInformation.createRemoteUser(token.getUserName());
+                String renewerParam =
+                    request
+                        .getParameter(TimelineAuthenticationConsts.RENEWER_PARAM);
+                if (renewerParam == null) {
+                  renewerParam = token.getUserName();
+                }
+                Token<?> dToken =
+                    secretManager.createToken(ownerUGI, renewerParam);
+                res = new TimelineDelegationTokenResponse();
+                res.setType(TimelineAuthenticationConsts.DELEGATION_TOKEN_URL);
+                res.setContent(dToken.encodeToUrlString());
+                break;
+              case RENEWDELEGATIONTOKEN:
+              case CANCELDELEGATIONTOKEN:
+                String tokenParam =
+                    request
+                        .getParameter(TimelineAuthenticationConsts.TOKEN_PARAM);
+                if (tokenParam == null) {
+                  response.sendError(HttpServletResponse.SC_BAD_REQUEST,
+                      MessageFormat
+                          .format(
+                              "Operation [{0}] requires the parameter [{1}]",
+                              dtOp,
+                              TimelineAuthenticationConsts.TOKEN_PARAM));
+                  requestContinues = false;
+                } else {
+                  if (dtOp == TimelineDelegationTokenOperation.CANCELDELEGATIONTOKEN) {
+                    Token<TimelineDelegationTokenIdentifier> dt =
+                        new Token<TimelineDelegationTokenIdentifier>();
+                    dt.decodeFromUrlString(tokenParam);
+                    secretManager.cancelToken(dt, token.getUserName());
+                  } else {
+                    Token<TimelineDelegationTokenIdentifier> dt =
+                        new Token<TimelineDelegationTokenIdentifier>();
+                    dt.decodeFromUrlString(tokenParam);
+                    long expirationTime =
+                        secretManager.renewToken(dt, token.getUserName());
+                    res = new TimelineDelegationTokenResponse();
+                    res.setType(TimelineAuthenticationConsts.DELEGATION_TOKEN_EXPIRATION_TIME);
+                    res.setContent(expirationTime);
+                  }
+                }
+                break;
+            }
+            if (requestContinues) {
+              response.setStatus(HttpServletResponse.SC_OK);
+              if (res != null) {
+                response.setContentType(MediaType.APPLICATION_JSON);
+                Writer writer = response.getWriter();
+                mapper.writeValue(writer, res);
+                writer.write(ENTER);
+                writer.flush();
+
+              }
+              requestContinues = false;
+            }
+          } catch (IOException e) {
+            throw new AuthenticationException(e.toString(), e);
+          }
+        }
+      } else {
+        response
+            .sendError(
+                HttpServletResponse.SC_BAD_REQUEST,
+                MessageFormat
+                    .format(
+                        "Wrong HTTP method [{0}] for operation [{1}], it should be [{2}]",
+                        request.getMethod(), dtOp, dtOp.getHttpMethod()));
+        requestContinues = false;
+      }
+    }
+    return requestContinues;
+  }
+
+  /**
+   * Authenticates a request looking for the <code>delegation</code>
+   * query-string parameter and verifying it is a valid token. If there is not
+   * <code>delegation</code> query-string parameter, it delegates the
+   * authentication to the {@link KerberosAuthenticationHandler} unless it is
+   * disabled.
+   * 
+   * @param request
+   *          the HTTP client request.
+   * @param response
+   *          the HTTP client response.
+   * 
+   * @return the authentication token for the authenticated request.
+   * @throws IOException
+   *           thrown if an IO error occurred.
+   * @throws AuthenticationException
+   *           thrown if the authentication failed.
+   */
+  @Override
+  public AuthenticationToken authenticate(HttpServletRequest request,
+      HttpServletResponse response)
+      throws IOException, AuthenticationException {
+    AuthenticationToken token;
+    String delegationParam =
+        request
+            .getParameter(TimelineAuthenticationConsts.DELEGATION_PARAM);
+    if (delegationParam != null) {
+      Token<TimelineDelegationTokenIdentifier> dt =
+          new Token<TimelineDelegationTokenIdentifier>();
+      dt.decodeFromUrlString(delegationParam);
+      TimelineDelegationTokenSecretManagerService secretManager =
+          AHSWebApp.getInstance()
+              .getTimelineDelegationTokenSecretManagerService();
+      UserGroupInformation ugi = secretManager.verifyToken(dt);
+      final String shortName = ugi.getShortUserName();
+      // creating a ephemeral token
+      token = new AuthenticationToken(shortName, ugi.getUserName(), getType());
+      token.setExpires(0);
+    } else {
+      token = super.authenticate(request, response);
+    }
+    return token;
+  }
+
+}

+ 189 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/timeline/security/TimelineDelegationTokenSecretManagerService.java

@@ -0,0 +1,189 @@
+/**
+ * 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.hadoop.yarn.server.applicationhistoryservice.timeline.security;
+
+import java.io.ByteArrayInputStream;
+import java.io.DataInputStream;
+import java.io.IOException;
+import java.net.InetSocketAddress;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.security.SecurityUtil;
+import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.security.token.Token;
+import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenSecretManager;
+import org.apache.hadoop.service.AbstractService;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.apache.hadoop.yarn.security.client.TimelineDelegationTokenIdentifier;
+
+/**
+ * The service wrapper of {@link TimelineDelegationTokenSecretManager}
+ */
+@Private
+@Unstable
+public class TimelineDelegationTokenSecretManagerService extends AbstractService {
+
+  private TimelineDelegationTokenSecretManager secretManager = null;
+  private InetSocketAddress serviceAddr = null;
+
+  public TimelineDelegationTokenSecretManagerService() {
+    super(TimelineDelegationTokenSecretManagerService.class.getName());
+  }
+
+  @Override
+  protected void serviceInit(Configuration conf) throws Exception {
+    long secretKeyInterval =
+        conf.getLong(YarnConfiguration.DELEGATION_KEY_UPDATE_INTERVAL_KEY,
+            YarnConfiguration.DELEGATION_KEY_UPDATE_INTERVAL_DEFAULT);
+    long tokenMaxLifetime =
+        conf.getLong(YarnConfiguration.DELEGATION_TOKEN_MAX_LIFETIME_KEY,
+            YarnConfiguration.DELEGATION_TOKEN_MAX_LIFETIME_DEFAULT);
+    long tokenRenewInterval =
+        conf.getLong(YarnConfiguration.DELEGATION_TOKEN_RENEW_INTERVAL_KEY,
+            YarnConfiguration.DELEGATION_TOKEN_RENEW_INTERVAL_DEFAULT);
+    secretManager = new TimelineDelegationTokenSecretManager(secretKeyInterval,
+        tokenMaxLifetime, tokenRenewInterval,
+        3600000);
+    secretManager.startThreads();
+
+    if (YarnConfiguration.useHttps(getConfig())) {
+      serviceAddr = getConfig().getSocketAddr(
+          YarnConfiguration.TIMELINE_SERVICE_WEBAPP_HTTPS_ADDRESS,
+          YarnConfiguration.DEFAULT_TIMELINE_SERVICE_WEBAPP_HTTPS_ADDRESS,
+          YarnConfiguration.DEFAULT_TIMELINE_SERVICE_WEBAPP_HTTPS_PORT);
+    } else {
+      serviceAddr = getConfig().getSocketAddr(
+          YarnConfiguration.TIMELINE_SERVICE_WEBAPP_ADDRESS,
+          YarnConfiguration.DEFAULT_TIMELINE_SERVICE_WEBAPP_ADDRESS,
+          YarnConfiguration.DEFAULT_TIMELINE_SERVICE_WEBAPP_PORT);
+    }
+    super.init(conf);
+  }
+
+  @Override
+  protected void serviceStop() throws Exception {
+    secretManager.stopThreads();
+    super.stop();
+  }
+
+  /**
+   * Creates a delegation token.
+   *
+   * @param ugi UGI creating the token.
+   * @param renewer token renewer.
+   * @return new delegation token.
+   * @throws IOException thrown if the token could not be created.
+   */
+  public Token<TimelineDelegationTokenIdentifier> createToken(
+      UserGroupInformation ugi, String renewer) throws IOException {
+    renewer = (renewer == null) ? ugi.getShortUserName() : renewer;
+    String user = ugi.getUserName();
+    Text owner = new Text(user);
+    Text realUser = null;
+    if (ugi.getRealUser() != null) {
+      realUser = new Text(ugi.getRealUser().getUserName());
+    }
+    TimelineDelegationTokenIdentifier tokenIdentifier =
+        new TimelineDelegationTokenIdentifier(owner, new Text(renewer), realUser);
+    Token<TimelineDelegationTokenIdentifier> token =
+        new Token<TimelineDelegationTokenIdentifier>(tokenIdentifier, secretManager);
+    SecurityUtil.setTokenService(token, serviceAddr);
+    return token;
+  }
+
+  /**
+   * Renews a delegation token.
+   *
+   * @param token delegation token to renew.
+   * @param renewer token renewer.
+   * @throws IOException thrown if the token could not be renewed.
+   */
+  public long renewToken(Token<TimelineDelegationTokenIdentifier> token,
+      String renewer) throws IOException {
+      return secretManager.renewToken(token, renewer);
+  }
+
+  /**
+   * Cancels a delegation token.
+   *
+   * @param token delegation token to cancel.
+   * @param canceler token canceler.
+   * @throws IOException thrown if the token could not be canceled.
+   */
+  public void cancelToken(Token<TimelineDelegationTokenIdentifier> token,
+      String canceler) throws IOException {
+    secretManager.cancelToken(token, canceler);
+  }
+
+  /**
+   * Verifies a delegation token.
+   *
+   * @param token delegation token to verify.
+   * @return the UGI for the token.
+   * @throws IOException thrown if the token could not be verified.
+   */
+  public UserGroupInformation verifyToken(Token<TimelineDelegationTokenIdentifier> token)
+    throws IOException {
+    ByteArrayInputStream buf = new ByteArrayInputStream(token.getIdentifier());
+    DataInputStream dis = new DataInputStream(buf);
+    TimelineDelegationTokenIdentifier id = new TimelineDelegationTokenIdentifier();
+    try {
+      id.readFields(dis);
+      secretManager.verifyToken(id, token.getPassword());
+    } finally {
+      dis.close();
+    }
+    return id.getUser();
+  }
+
+  /**
+   * Create a timeline secret manager
+   * 
+   * @param delegationKeyUpdateInterval
+   *          the number of seconds for rolling new secret keys.
+   * @param delegationTokenMaxLifetime
+   *          the maximum lifetime of the delegation tokens
+   * @param delegationTokenRenewInterval
+   *          how often the tokens must be renewed
+   * @param delegationTokenRemoverScanInterval
+   *          how often the tokens are scanned for expired tokens
+   */
+  @Private
+  @Unstable
+  public static class TimelineDelegationTokenSecretManager extends
+      AbstractDelegationTokenSecretManager<TimelineDelegationTokenIdentifier> {
+
+    public TimelineDelegationTokenSecretManager(long delegationKeyUpdateInterval,
+        long delegationTokenMaxLifetime, long delegationTokenRenewInterval,
+        long delegationTokenRemoverScanInterval) {
+      super(delegationKeyUpdateInterval, delegationTokenMaxLifetime,
+          delegationTokenRenewInterval, delegationTokenRemoverScanInterval);
+    }
+
+    @Override
+    public TimelineDelegationTokenIdentifier createIdentifier() {
+      return new TimelineDelegationTokenIdentifier();
+    }
+
+  }
+
+}

+ 51 - 4
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/webapp/AHSWebApp.java

@@ -19,25 +19,70 @@ package org.apache.hadoop.yarn.server.applicationhistoryservice.webapp;
 
 import static org.apache.hadoop.yarn.util.StringHelper.pajoin;
 
+import org.apache.hadoop.classification.InterfaceAudience.Private;
 import org.apache.hadoop.yarn.server.api.ApplicationContext;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.ApplicationHistoryManager;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.TimelineStore;
+import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.security.TimelineDelegationTokenSecretManagerService;
 import org.apache.hadoop.yarn.webapp.GenericExceptionHandler;
 import org.apache.hadoop.yarn.webapp.WebApp;
 import org.apache.hadoop.yarn.webapp.YarnJacksonJaxbJsonProvider;
 import org.apache.hadoop.yarn.webapp.YarnWebParams;
 
+import com.google.common.annotations.VisibleForTesting;
+
 public class AHSWebApp extends WebApp implements YarnWebParams {
 
-  private final ApplicationHistoryManager applicationHistoryManager;
-  private final TimelineStore timelineStore;
+  private ApplicationHistoryManager applicationHistoryManager;
+  private TimelineStore timelineStore;
+  private TimelineDelegationTokenSecretManagerService secretManagerService;
+
+  private static AHSWebApp instance = null;
+
+  public static AHSWebApp getInstance() {
+    if (instance == null) {
+      instance = new AHSWebApp();
+    }
+    return instance;
+  }
+
+  @Private
+  @VisibleForTesting
+  public static void resetInstance() {
+    instance = null;
+  }
+
+  private AHSWebApp() {
 
-  public AHSWebApp(ApplicationHistoryManager applicationHistoryManager,
-      TimelineStore timelineStore) {
+  }
+
+  public ApplicationHistoryManager getApplicationHistoryManager() {
+    return applicationHistoryManager;
+  }
+
+  public void setApplicationHistoryManager(
+      ApplicationHistoryManager applicationHistoryManager) {
     this.applicationHistoryManager = applicationHistoryManager;
+  }
+
+  public TimelineStore getTimelineStore() {
+    return timelineStore;
+  }
+
+  public void setTimelineStore(TimelineStore timelineStore) {
     this.timelineStore = timelineStore;
   }
 
+  public TimelineDelegationTokenSecretManagerService
+      getTimelineDelegationTokenSecretManagerService() {
+    return secretManagerService;
+  }
+
+  public void setTimelineDelegationTokenSecretManagerService(
+      TimelineDelegationTokenSecretManagerService secretManagerService) {
+    this.secretManagerService = secretManagerService;
+  }
+
   @Override
   public void setup() {
     bind(YarnJacksonJaxbJsonProvider.class);
@@ -46,6 +91,8 @@ public class AHSWebApp extends WebApp implements YarnWebParams {
     bind(GenericExceptionHandler.class);
     bind(ApplicationContext.class).toInstance(applicationHistoryManager);
     bind(TimelineStore.class).toInstance(timelineStore);
+    bind(TimelineDelegationTokenSecretManagerService.class).toInstance(
+        secretManagerService);
     route("/", AHSController.class);
     route(pajoin("/apps", APP_STATE), AHSController.class);
     route(pajoin("/app", APPLICATION_ID), AHSController.class, "app");

+ 2 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryClientService.java

@@ -44,6 +44,7 @@ import org.apache.hadoop.yarn.api.records.ContainerId;
 import org.apache.hadoop.yarn.api.records.ContainerReport;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.exceptions.YarnException;
+import org.apache.hadoop.yarn.server.applicationhistoryservice.webapp.AHSWebApp;
 import org.apache.hadoop.yarn.webapp.util.WebAppUtils;
 import org.junit.After;
 import org.junit.Before;
@@ -74,6 +75,7 @@ public class TestApplicationHistoryClientService extends
 
   @After
   public void tearDown() throws Exception {
+    AHSWebApp.resetInstance();
     historyServer.stop();
   }
 

+ 3 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryServer.java

@@ -26,6 +26,7 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.service.Service.STATE;
 import org.apache.hadoop.util.ExitUtil;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.apache.hadoop.yarn.server.applicationhistoryservice.webapp.AHSWebApp;
 import org.junit.After;
 import org.junit.Test;
 
@@ -40,7 +41,7 @@ public class TestApplicationHistoryServer {
     Configuration config = new YarnConfiguration();
     historyServer.init(config);
     assertEquals(STATE.INITED, historyServer.getServiceState());
-    assertEquals(3, historyServer.getServices().size());
+    assertEquals(4, historyServer.getServices().size());
     ApplicationHistoryClientService historyService =
         historyServer.getClientService();
     assertNotNull(historyServer.getClientService());
@@ -73,5 +74,6 @@ public class TestApplicationHistoryServer {
     if (historyServer != null) {
       historyServer.stop();
     }
+    AHSWebApp.resetInstance();
   }
 }

+ 2 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-tests/src/test/java/org/apache/hadoop/yarn/server/MiniYARNCluster.java

@@ -58,6 +58,7 @@ import org.apache.hadoop.yarn.server.applicationhistoryservice.ApplicationHistor
 import org.apache.hadoop.yarn.server.applicationhistoryservice.MemoryApplicationHistoryStore;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.MemoryTimelineStore;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.timeline.TimelineStore;
+import org.apache.hadoop.yarn.server.applicationhistoryservice.webapp.AHSWebApp;
 import org.apache.hadoop.yarn.server.nodemanager.Context;
 import org.apache.hadoop.yarn.server.nodemanager.NodeHealthCheckerService;
 import org.apache.hadoop.yarn.server.nodemanager.NodeManager;
@@ -719,6 +720,7 @@ public class MiniYARNCluster extends CompositeService {
       if (appHistoryServer != null) {
         appHistoryServer.stop();
       }
+      AHSWebApp.resetInstance();
       super.serviceStop();
     }
   }