Procházet zdrojové kódy

HDDS-836. Add TokenIdentifier Ozone for delegation token and block token. Contributed by Ajay Kumar.

Xiaoyu Yao před 6 roky
rodič
revize
6d6b1a00c2

+ 59 - 0
hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/security/x509/keys/SecurityUtil.java

@@ -18,6 +18,15 @@
  */
 package org.apache.hadoop.hdds.security.x509.keys;
 
+import java.security.KeyFactory;
+import java.security.NoSuchAlgorithmException;
+import java.security.NoSuchProviderException;
+import java.security.PrivateKey;
+import java.security.PublicKey;
+import java.security.spec.InvalidKeySpecException;
+import java.security.spec.PKCS8EncodedKeySpec;
+import java.security.spec.X509EncodedKeySpec;
+import org.apache.hadoop.hdds.security.x509.SecurityConfig;
 import org.apache.hadoop.hdds.security.x509.exceptions.CertificateException;
 import org.bouncycastle.asn1.ASN1ObjectIdentifier;
 import org.bouncycastle.asn1.ASN1Sequence;
@@ -76,4 +85,54 @@ public final class SecurityUtil {
     }
     throw new CertificateException("No PKCS#9 extension found in CSR");
   }
+
+  /*
+   * Returns private key created from encoded key.
+   * @return private key if successful else returns null.
+   */
+  public static PrivateKey getPrivateKey(byte[] encodedKey,
+      SecurityConfig secureConfig) {
+    PrivateKey pvtKey = null;
+    if (encodedKey == null || encodedKey.length == 0) {
+      return null;
+    }
+
+    try {
+      KeyFactory kf = null;
+
+      kf = KeyFactory.getInstance(secureConfig.getKeyAlgo(),
+          secureConfig.getProvider());
+      pvtKey = kf.generatePrivate(new PKCS8EncodedKeySpec(encodedKey));
+
+    } catch (NoSuchAlgorithmException | InvalidKeySpecException |
+        NoSuchProviderException e) {
+      return null;
+    }
+    return pvtKey;
+  }
+
+  /*
+   * Returns public key created from encoded key.
+   * @return public key if successful else returns null.
+   */
+  public static PublicKey getPublicKey(byte[] encodedKey,
+      SecurityConfig secureConfig) {
+    PublicKey key = null;
+    if (encodedKey == null || encodedKey.length == 0) {
+      return null;
+    }
+
+    try {
+      KeyFactory kf = null;
+      kf = KeyFactory.getInstance(secureConfig.getKeyAlgo(),
+          secureConfig.getProvider());
+      key = kf.generatePublic(new X509EncodedKeySpec(encodedKey));
+
+    } catch (NoSuchAlgorithmException | InvalidKeySpecException |
+        NoSuchProviderException e) {
+      return null;
+    }
+    return key;
+  }
+
 }

+ 24 - 0
hadoop-hdds/common/src/main/proto/hdds.proto

@@ -195,6 +195,30 @@ message ContainerBlockID {
     required int64 localID = 2;
 }
 
+
+/**
+ * Information for the Hdds block token.
+ * When adding further fields, make sure they are optional as they would
+ * otherwise not be backwards compatible.
+ */
+message BlockTokenSecretProto {
+    /**
+ * File access permissions mode.
+ */
+    enum AccessModeProto {
+        READ = 1;
+        WRITE = 2;
+        COPY = 3;
+        DELETE = 4;
+    }
+    required string ownerId = 1;
+    required string blockId = 2;
+    required uint64 expiryDate = 3;
+    required string omCertSerialId = 4;
+    repeated AccessModeProto modes = 5;
+
+}
+
 message BlockID {
     required ContainerBlockID containerBlockID = 1;
     optional uint64 blockCommitSequenceId = 2 [default = 0];

+ 178 - 0
hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/security/OzoneBlockTokenIdentifier.java

@@ -0,0 +1,178 @@
+/**
+ * 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.ozone.security;
+
+import com.google.common.annotations.VisibleForTesting;
+import org.apache.commons.lang3.builder.EqualsBuilder;
+import org.apache.commons.lang3.builder.HashCodeBuilder;
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.BlockTokenSecretProto;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.BlockTokenSecretProto.AccessModeProto;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos.BlockTokenSecretProto.Builder;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.security.token.TokenIdentifier;
+
+import java.io.DataInput;
+import java.io.DataInputStream;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.EnumSet;
+
+/**
+ * Block token identifier for Ozone/HDDS. Ozone block access token is similar
+ * to HDFS block access token, which is meant to be lightweight and
+ * short-lived. No need to renew or revoke a block access token. when a
+ * cached block access token expires, the client simply get a new one.
+ * Block access token should be cached only in memory and never write to disk.
+ */
+@InterfaceAudience.Private
+public class OzoneBlockTokenIdentifier extends TokenIdentifier {
+
+  static final Text KIND_NAME = new Text("HDDS_BLOCK_TOKEN");
+  private long expiryDate;
+  private String ownerId;
+  private String blockId;
+  private final EnumSet<AccessModeProto> modes;
+  private final String omCertSerialId;
+
+  public OzoneBlockTokenIdentifier(String ownerId, String blockId,
+      EnumSet<AccessModeProto> modes, long expiryDate, String omCertSerialId) {
+    this.ownerId = ownerId;
+    this.blockId = blockId;
+    this.expiryDate = expiryDate;
+    this.modes = modes == null ? EnumSet.noneOf(AccessModeProto.class) : modes;
+    this.omCertSerialId = omCertSerialId;
+  }
+
+  @Override
+  public UserGroupInformation getUser() {
+    if (this.getOwnerId() == null || "".equals(this.getOwnerId())) {
+      return UserGroupInformation.createRemoteUser(blockId);
+    }
+    return UserGroupInformation.createRemoteUser(ownerId);
+  }
+
+  public long getExpiryDate() {
+    return expiryDate;
+  }
+
+  public String getOwnerId() {
+    return ownerId;
+  }
+
+  public String getBlockId() {
+    return blockId;
+  }
+
+  public EnumSet<AccessModeProto> getAccessModes() {
+    return modes;
+  }
+
+  public String getOmCertSerialId(){
+    return omCertSerialId;
+  }
+
+  @Override
+  public Text getKind() {
+    return KIND_NAME;
+  }
+
+  @Override
+  public String toString() {
+    return "block_token_identifier (expiryDate=" + this.getExpiryDate()
+        + ", ownerId=" + this.getOwnerId()
+        + ", omCertSerialId=" + this.getOmCertSerialId()
+        + ", blockId=" + this.getBlockId() + ", access modes="
+        + this.getAccessModes() + ")";
+  }
+
+  static boolean isEqual(Object a, Object b) {
+    return a == null ? b == null : a.equals(b);
+  }
+
+  @Override
+  public boolean equals(Object obj) {
+    if (obj == this) {
+      return true;
+    }
+
+    if (obj instanceof OzoneBlockTokenIdentifier) {
+      OzoneBlockTokenIdentifier that = (OzoneBlockTokenIdentifier) obj;
+      return new EqualsBuilder()
+          .append(this.expiryDate, that.expiryDate)
+          .append(this.ownerId, that.ownerId)
+          .append(this.blockId, that.blockId)
+          .append(this.modes, that.modes)
+          .append(this.omCertSerialId, that.omCertSerialId)
+          .build();
+    }
+    return false;
+  }
+
+  @Override
+  public int hashCode() {
+    return new HashCodeBuilder(133, 567)
+        .append(this.expiryDate)
+        .append(this.blockId)
+        .append(this.ownerId)
+        .append(this.modes)
+        .append(this.omCertSerialId)
+        .build();
+  }
+
+  @Override
+  public void readFields(DataInput in) throws IOException {
+    final DataInputStream dis = (DataInputStream) in;
+    if (!dis.markSupported()) {
+      throw new IOException("Could not peek first byte.");
+    }
+    readFieldsProtobuf(dis);
+  }
+
+  @VisibleForTesting
+  public static OzoneBlockTokenIdentifier readFieldsProtobuf(DataInput in)
+      throws IOException {
+    BlockTokenSecretProto tokenPtoto =
+        BlockTokenSecretProto.parseFrom((DataInputStream) in);
+    return new OzoneBlockTokenIdentifier(tokenPtoto.getOwnerId(),
+        tokenPtoto.getBlockId(), EnumSet.copyOf(tokenPtoto.getModesList()),
+        tokenPtoto.getExpiryDate(), tokenPtoto.getOmCertSerialId());
+  }
+
+  @Override
+  public void write(DataOutput out) throws IOException {
+    writeProtobuf(out);
+  }
+
+  @VisibleForTesting
+  void writeProtobuf(DataOutput out) throws IOException {
+    Builder builder = BlockTokenSecretProto.newBuilder()
+        .setBlockId(this.getBlockId())
+        .setOwnerId(this.getOwnerId())
+        .setOmCertSerialId(this.getOmCertSerialId())
+        .setExpiryDate(this.getExpiryDate());
+    // Add access mode allowed
+    for (AccessModeProto mode : this.getAccessModes()) {
+      builder.addModes(AccessModeProto.valueOf(mode.name()));
+    }
+    out.write(builder.build().toByteArray());
+  }
+}
+

+ 55 - 0
hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/security/OzoneBlockTokenSelector.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.ozone.security;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+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;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.Collection;
+
+/**
+ * A block token selector for Ozone.
+ */
+@InterfaceAudience.Private
+public class OzoneBlockTokenSelector implements
+    TokenSelector<OzoneBlockTokenIdentifier> {
+
+  private static final Logger LOG = LoggerFactory
+      .getLogger(OzoneBlockTokenSelector.class);
+
+  @Override
+  @SuppressWarnings("unchecked")
+  public Token<OzoneBlockTokenIdentifier> selectToken(Text service,
+      Collection<Token<? extends TokenIdentifier>> tokens) {
+    if (service == null) {
+      return null;
+    }
+    for (Token<? extends TokenIdentifier> token : tokens) {
+      if (OzoneBlockTokenIdentifier.KIND_NAME.equals(token.getKind())) {
+        LOG.trace("Getting token for service:{}", service);
+        return (Token<OzoneBlockTokenIdentifier>) token;
+      }
+    }
+    return null;
+  }
+}

+ 52 - 0
hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/security/OzoneDelegationTokenSelector.java

@@ -0,0 +1,52 @@
+/**
+ * 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.ozone.security;
+
+import java.util.Collection;
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
+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.delegation.AbstractDelegationTokenSelector;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * A delegation token selector that is specialized for Ozone.
+ */
+@InterfaceAudience.Private
+public class OzoneDelegationTokenSelector
+    extends AbstractDelegationTokenSelector<DelegationTokenIdentifier> {
+
+  public OzoneDelegationTokenSelector() {
+    super(OzoneTokenIdentifier.KIND_NAME);
+  }
+
+  private static final Logger LOG = LoggerFactory
+      .getLogger(OzoneDelegationTokenSelector.class);
+
+  @Override
+  public Token selectToken(Text service,
+      Collection<Token<? extends TokenIdentifier>> tokens) {
+    LOG.trace("Getting token for service {}", service);
+    return super.selectToken(service, tokens);
+  }
+
+}
+

+ 195 - 0
hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/security/OzoneSecretKey.java

@@ -0,0 +1,195 @@
+/**
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.ozone.security;
+
+import com.google.common.base.Preconditions;
+import com.google.protobuf.ByteString;
+import java.io.ByteArrayInputStream;
+import java.io.DataInput;
+import java.io.DataInputStream;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.security.KeyPair;
+import java.security.PrivateKey;
+import java.security.PublicKey;
+import org.apache.commons.lang3.builder.EqualsBuilder;
+import org.apache.commons.lang3.builder.HashCodeBuilder;
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.hdds.security.x509.SecurityConfig;
+import org.apache.hadoop.hdds.security.x509.keys.SecurityUtil;
+import org.apache.hadoop.io.Writable;
+import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.SecretKeyProto;
+
+/**
+ * Wrapper class for Ozone/Hdds secret keys. Used in delegation tokens and block
+ * tokens.
+ */
+@InterfaceAudience.Private
+@InterfaceStability.Unstable
+public class OzoneSecretKey implements Writable {
+
+  private int keyId;
+  private long expiryDate;
+  private PrivateKey privateKey;
+  private PublicKey publicKey;
+  private int maxKeyLen;
+  private SecurityConfig securityConfig;
+
+  public OzoneSecretKey(int keyId, long expiryDate, KeyPair keyPair,
+      int maxKeyLen) {
+    Preconditions.checkNotNull(keyId);
+    this.keyId = keyId;
+    this.expiryDate = expiryDate;
+    byte[] encodedKey = keyPair.getPrivate().getEncoded();
+    this.maxKeyLen = maxKeyLen;
+    if (encodedKey.length > maxKeyLen) {
+      throw new RuntimeException("can't create " + encodedKey.length +
+          " byte long DelegationKey.");
+    }
+    this.privateKey = keyPair.getPrivate();
+    this.publicKey = keyPair.getPublic();
+  }
+
+  /*
+   * Create new instance using default signature algorithm and provider.
+   * */
+  public OzoneSecretKey(int keyId, long expiryDate, byte[] pvtKey,
+      byte[] publicKey, int maxKeyLen) {
+    Preconditions.checkNotNull(pvtKey);
+    Preconditions.checkNotNull(publicKey);
+
+    this.securityConfig = new SecurityConfig(new OzoneConfiguration());
+    this.keyId = keyId;
+    this.expiryDate = expiryDate;
+    this.maxKeyLen = maxKeyLen;
+    if (pvtKey.length > maxKeyLen) {
+      throw new RuntimeException("can't create " + pvtKey.length +
+          " byte long DelegationKey. Max allowed length is " + maxKeyLen);
+    }
+    this.privateKey = SecurityUtil.getPrivateKey(pvtKey, securityConfig);
+    this.publicKey = SecurityUtil.getPublicKey(publicKey, securityConfig);
+  }
+
+  public int getKeyId() {
+    return keyId;
+  }
+
+  public long getExpiryDate() {
+    return expiryDate;
+  }
+
+  public PrivateKey getPrivateKey() {
+    return privateKey;
+  }
+
+  public PublicKey getPublicKey() {
+    return publicKey;
+  }
+
+  public int getMaxKeyLen() {
+    return maxKeyLen;
+  }
+
+  public byte[] getEncodedPrivateKey() {
+    return privateKey.getEncoded();
+  }
+
+  public byte[] getEncodedPubliceKey() {
+    return publicKey.getEncoded();
+  }
+
+  public void setExpiryDate(long expiryDate) {
+    this.expiryDate = expiryDate;
+  }
+
+  @Override
+  public void write(DataOutput out) throws IOException {
+    SecretKeyProto token = SecretKeyProto.newBuilder()
+        .setKeyId(getKeyId())
+        .setExpiryDate(getExpiryDate())
+        .setPrivateKeyBytes(ByteString.copyFrom(getEncodedPrivateKey()))
+        .setPublicKeyBytes(ByteString.copyFrom(getEncodedPubliceKey()))
+        .setMaxKeyLen(getMaxKeyLen())
+        .build();
+    out.write(token.toByteArray());
+  }
+
+  @Override
+  public void readFields(DataInput in) throws IOException {
+    SecretKeyProto secretKey = SecretKeyProto.parseFrom((DataInputStream) in);
+    expiryDate = secretKey.getExpiryDate();
+    keyId = secretKey.getKeyId();
+    privateKey = SecurityUtil.getPrivateKey(secretKey.getPrivateKeyBytes()
+        .toByteArray(), securityConfig);
+    publicKey = SecurityUtil.getPublicKey(secretKey.getPublicKeyBytes()
+        .toByteArray(), securityConfig);
+    maxKeyLen = secretKey.getMaxKeyLen();
+  }
+
+  @Override
+  public int hashCode() {
+    HashCodeBuilder hashCodeBuilder = new HashCodeBuilder(537, 963);
+    hashCodeBuilder.append(getExpiryDate())
+        .append(getKeyId())
+        .append(getEncodedPrivateKey())
+        .append(getEncodedPubliceKey());
+
+    return hashCodeBuilder.build();
+  }
+
+  @Override
+  public boolean equals(Object obj) {
+    if (obj == this) {
+      return true;
+    }
+
+    if (obj instanceof OzoneSecretKey) {
+      OzoneSecretKey that = (OzoneSecretKey) obj;
+      return new EqualsBuilder()
+          .append(this.keyId, that.keyId)
+          .append(this.expiryDate, that.expiryDate)
+          .append(this.privateKey, that.privateKey)
+          .append(this.publicKey, that.publicKey)
+          .build();
+    }
+    return false;
+  }
+
+  /**
+   * Reads protobuf encoded input stream to construct {@link OzoneSecretKey}.
+   */
+  static OzoneSecretKey readProtoBuf(DataInput in) throws IOException {
+    Preconditions.checkNotNull(in);
+    SecretKeyProto key = SecretKeyProto.parseFrom((DataInputStream) in);
+    return new OzoneSecretKey(key.getKeyId(), key.getExpiryDate(),
+        key.getPrivateKeyBytes().toByteArray(),
+        key.getPublicKeyBytes().toByteArray(), key.getMaxKeyLen());
+  }
+
+  /**
+   * Reads protobuf encoded input stream to construct {@link OzoneSecretKey}.
+   */
+  static OzoneSecretKey readProtoBuf(byte[] identifier) throws IOException {
+    Preconditions.checkNotNull(identifier);
+    DataInputStream in = new DataInputStream(new ByteArrayInputStream(
+        identifier));
+    return readProtoBuf(in);
+  }
+
+}

+ 217 - 0
hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/security/OzoneTokenIdentifier.java

@@ -0,0 +1,217 @@
+/**
+ * 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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * 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.ozone.security;
+
+import java.io.ByteArrayInputStream;
+import java.io.DataInput;
+import java.io.DataInputStream;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.Arrays;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.OMTokenProto;
+import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenIdentifier;
+import org.apache.hadoop.security.token.Token;
+
+/**
+ * The token identifier for Ozone Master.
+ */
+@InterfaceAudience.Private
+@InterfaceStability.Unstable
+public class OzoneTokenIdentifier extends
+    AbstractDelegationTokenIdentifier {
+
+  public final static Text KIND_NAME = new Text("OzoneToken");
+
+  /**
+   * Create an empty delegation token identifier.
+   */
+  public OzoneTokenIdentifier() {
+    super();
+  }
+
+  /**
+   * Create a new ozone master 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 OzoneTokenIdentifier(Text owner, Text renewer, Text realUser) {
+    super(owner, renewer, realUser);
+  }
+
+  /**
+   * {@inheritDoc}
+   */
+  @Override
+  public Text getKind() {
+    return KIND_NAME;
+  }
+
+  /**
+   * Default TrivialRenewer.
+   */
+  @InterfaceAudience.Private
+  public static class Renewer extends Token.TrivialRenewer {
+
+    @Override
+    protected Text getKind() {
+      return KIND_NAME;
+    }
+  }
+
+  /**
+   * Overrides default implementation to write using Protobuf.
+   *
+   * @param out output stream
+   * @throws IOException
+   */
+  @Override
+  public void write(DataOutput out) throws IOException {
+    OMTokenProto token = OMTokenProto.newBuilder()
+        .setOwner(getOwner().toString())
+        .setRealUser(getRealUser().toString())
+        .setRenewer(getRenewer().toString())
+        .setIssueDate(getIssueDate())
+        .setMaxDate(getMaxDate())
+        .setSequenceNumber(getSequenceNumber())
+        .setMasterKeyId(getMasterKeyId()).build();
+    out.write(token.toByteArray());
+  }
+
+  /**
+   * Overrides default implementation to read using Protobuf.
+   *
+   * @param in input stream
+   * @throws IOException
+   */
+  @Override
+  public void readFields(DataInput in) throws IOException {
+    OMTokenProto token = OMTokenProto.parseFrom((DataInputStream) in);
+    setOwner(new Text(token.getOwner()));
+    setRealUser(new Text(token.getRealUser()));
+    setRenewer(new Text(token.getRenewer()));
+    setIssueDate(token.getIssueDate());
+    setMaxDate(token.getMaxDate());
+    setSequenceNumber(token.getSequenceNumber());
+    setMasterKeyId(token.getMasterKeyId());
+  }
+
+  /**
+   * Reads protobuf encoded input stream to construct {@link
+   * OzoneTokenIdentifier}.
+   */
+  public static OzoneTokenIdentifier readProtoBuf(DataInput in)
+      throws IOException {
+    OMTokenProto token = OMTokenProto.parseFrom((DataInputStream) in);
+    OzoneTokenIdentifier identifier = new OzoneTokenIdentifier();
+    identifier.setRenewer(new Text(token.getRenewer()));
+    identifier.setOwner(new Text(token.getOwner()));
+    identifier.setRealUser(new Text(token.getRealUser()));
+    identifier.setMaxDate(token.getMaxDate());
+    identifier.setIssueDate(token.getIssueDate());
+    identifier.setSequenceNumber(token.getSequenceNumber());
+    identifier.setMasterKeyId(token.getMasterKeyId());
+    return identifier;
+  }
+
+  /**
+   * Reads protobuf encoded input stream to construct {@link
+   * OzoneTokenIdentifier}.
+   */
+  public static OzoneTokenIdentifier readProtoBuf(byte[] identifier)
+      throws IOException {
+    DataInputStream in = new DataInputStream(new ByteArrayInputStream(
+        identifier));
+    return readProtoBuf(in);
+  }
+
+  /**
+   * Creates new instance.
+   */
+  public static OzoneTokenIdentifier newInstance() {
+    return new OzoneTokenIdentifier();
+  }
+
+  /**
+   * Creates new instance.
+   */
+  public static OzoneTokenIdentifier newInstance(Text owner, Text renewer,
+      Text realUser) {
+    return new OzoneTokenIdentifier(owner, renewer, realUser);
+  }
+
+  @Override
+  public int hashCode() {
+    return super.hashCode();
+  }
+
+  @Override
+  public boolean equals(Object obj) {
+    if (!(obj instanceof OzoneTokenIdentifier)) {
+      return false;
+    }
+    return super.equals(obj);
+  }
+
+  /**
+   * Class to encapsulate a token's renew date and password.
+   */
+  @InterfaceStability.Evolving
+  public static class TokenInfo {
+
+    private long renewDate;
+    private byte[] password;
+    private String trackingId;
+
+    public TokenInfo(long renewDate, byte[] password) {
+      this(renewDate, password, null);
+    }
+
+    public TokenInfo(long renewDate, byte[] password,
+        String trackingId) {
+      this.renewDate = renewDate;
+      this.password = Arrays.copyOf(password, password.length);
+      this.trackingId = trackingId;
+    }
+
+    /**
+     * returns renew date.
+     */
+    public long getRenewDate() {
+      return renewDate;
+    }
+
+    /**
+     * returns password.
+     */
+    byte[] getPassword() {
+      return password;
+    }
+
+    /**
+     * returns tracking id.
+     */
+    public String getTrackingId() {
+      return trackingId;
+    }
+  }
+}

+ 21 - 0
hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/security/package-info.java

@@ -0,0 +1,21 @@
+/**
+ * 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.ozone.security;
+/**
+ * Ozone security related classes.
+ */

+ 20 - 0
hadoop-ozone/common/src/main/proto/OzoneManagerProtocol.proto

@@ -466,6 +466,26 @@ message DeleteKeyResponse {
     optional uint64 openVersion = 4;
 }
 
+message OMTokenProto {
+    optional uint32 version        = 1;
+    optional string owner          = 2;
+    optional string renewer        = 3;
+    optional string realUser       = 4;
+    optional uint64 issueDate      = 5;
+    optional uint64 maxDate        = 6;
+    optional uint32 sequenceNumber = 7;
+    optional uint32 masterKeyId    = 8;
+    optional uint64 expiryDate     = 9;
+}
+
+message SecretKeyProto {
+    required uint32 keyId = 1;
+    required uint64 expiryDate = 2;
+    required bytes privateKeyBytes = 3;
+    required bytes publicKeyBytes = 4;
+    required uint32 maxKeyLen = 5;
+}
+
 message ListKeysRequest {
     required string volumeName = 1;
     required string bucketName = 2;

+ 255 - 0
hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/security/TestOzoneBlockTokenIdentifier.java

@@ -0,0 +1,255 @@
+/**
+ * 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.ozone.security;
+
+import java.io.File;
+import java.io.IOException;
+import java.security.GeneralSecurityException;
+import java.security.InvalidKeyException;
+import java.security.KeyPair;
+import java.security.NoSuchAlgorithmException;
+import java.security.NoSuchProviderException;
+import java.security.PrivateKey;
+import java.security.Signature;
+import java.security.SignatureException;
+import java.security.cert.Certificate;
+import java.security.cert.CertificateEncodingException;
+import java.security.cert.X509Certificate;
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.EnumSet;
+import java.util.List;
+import java.util.Map;
+import javax.crypto.KeyGenerator;
+import javax.crypto.Mac;
+import javax.crypto.SecretKey;
+import org.apache.commons.lang3.RandomStringUtils;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.fs.FileUtil;
+import org.apache.hadoop.hdds.protocol.proto.HddsProtos;
+import org.apache.hadoop.security.ssl.KeyStoreTestUtil;
+import org.apache.hadoop.test.GenericTestUtils;
+import org.apache.hadoop.util.Time;
+import org.junit.After;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Test class for OzoneManagerDelegationToken.
+ */
+public class TestOzoneBlockTokenIdentifier {
+
+  private static final Logger LOG = LoggerFactory
+      .getLogger(TestOzoneBlockTokenIdentifier.class);
+  private static final String BASEDIR = GenericTestUtils
+      .getTempPath(TestOzoneBlockTokenIdentifier.class.getSimpleName());
+  private static final String KEYSTORES_DIR =
+      new File(BASEDIR).getAbsolutePath();
+  private static long expiryTime;
+  private static KeyPair keyPair;
+  private static X509Certificate cert;
+
+  @BeforeClass
+  public static void setUp() throws Exception {
+    File base = new File(BASEDIR);
+    FileUtil.fullyDelete(base);
+    base.mkdirs();
+    expiryTime = Time.monotonicNow() + 60 * 60 * 24;
+
+    // Create Ozone Master key pair.
+    keyPair = KeyStoreTestUtil.generateKeyPair("RSA");
+    // Create Ozone Master certificate (SCM CA issued cert) and key store.
+    cert = KeyStoreTestUtil
+        .generateCertificate("CN=OzoneMaster", keyPair, 30, "SHA256withRSA");
+  }
+
+  @After
+  public void cleanUp() throws Exception {
+    // KeyStoreTestUtil.cleanupSSLConfig(KEYSTORES_DIR, sslConfsDir);
+  }
+
+  @Test
+  public void testSignToken() throws GeneralSecurityException, IOException {
+    String keystore = new File(KEYSTORES_DIR, "keystore.jks")
+        .getAbsolutePath();
+    String truststore = new File(KEYSTORES_DIR, "truststore.jks")
+        .getAbsolutePath();
+    String trustPassword = "trustPass";
+    String keyStorePassword = "keyStorePass";
+    String keyPassword = "keyPass";
+
+
+    KeyStoreTestUtil.createKeyStore(keystore, keyStorePassword, keyPassword,
+        "OzoneMaster", keyPair.getPrivate(), cert);
+
+    // Create trust store and put the certificate in the trust store
+    Map<String, X509Certificate> certs = Collections.singletonMap("server",
+        cert);
+    KeyStoreTestUtil.createTrustStore(truststore, trustPassword, certs);
+
+    // Sign the OzoneMaster Token with Ozone Master private key
+    PrivateKey privateKey = keyPair.getPrivate();
+    OzoneBlockTokenIdentifier tokenId = new OzoneBlockTokenIdentifier(
+        "testUser", "84940",
+        EnumSet.allOf(HddsProtos.BlockTokenSecretProto.AccessModeProto.class),
+        expiryTime, cert.getSerialNumber().toString());
+    byte[] signedToken = signTokenAsymmetric(tokenId, privateKey);
+
+    // Verify a valid signed OzoneMaster Token with Ozone Master
+    // public key(certificate)
+    boolean isValidToken = verifyTokenAsymmetric(tokenId, signedToken, cert);
+    LOG.info("{} is {}", tokenId, isValidToken ? "valid." : "invalid.");
+
+    // Verify an invalid signed OzoneMaster Token with Ozone Master
+    // public key(certificate)
+    tokenId = new OzoneBlockTokenIdentifier("", "",
+        EnumSet.allOf(HddsProtos.BlockTokenSecretProto.AccessModeProto.class),
+        expiryTime, cert.getSerialNumber().toString());
+    LOG.info("Unsigned token {} is {}", tokenId,
+        verifyTokenAsymmetric(tokenId, RandomUtils.nextBytes(128), cert));
+
+  }
+
+  public byte[] signTokenAsymmetric(OzoneBlockTokenIdentifier tokenId,
+      PrivateKey privateKey) throws NoSuchAlgorithmException,
+      InvalidKeyException, SignatureException {
+    Signature rsaSignature = Signature.getInstance("SHA256withRSA");
+    rsaSignature.initSign(privateKey);
+    rsaSignature.update(tokenId.getBytes());
+    byte[] signature = rsaSignature.sign();
+    return signature;
+  }
+
+  public boolean verifyTokenAsymmetric(OzoneBlockTokenIdentifier tokenId,
+      byte[] signature, Certificate certificate) throws InvalidKeyException,
+      NoSuchAlgorithmException, SignatureException {
+    Signature rsaSignature = Signature.getInstance("SHA256withRSA");
+    rsaSignature.initVerify(certificate);
+    rsaSignature.update(tokenId.getBytes());
+    boolean isValid = rsaSignature.verify(signature);
+    return isValid;
+  }
+
+  private byte[] signTokenSymmetric(OzoneBlockTokenIdentifier identifier,
+      Mac mac, SecretKey key) {
+    try {
+      mac.init(key);
+    } catch (InvalidKeyException ike) {
+      throw new IllegalArgumentException("Invalid key to HMAC computation",
+          ike);
+    }
+    return mac.doFinal(identifier.getBytes());
+  }
+
+  OzoneBlockTokenIdentifier generateTestToken() {
+    return new OzoneBlockTokenIdentifier(RandomStringUtils.randomAlphabetic(6),
+        RandomStringUtils.randomAlphabetic(5),
+        EnumSet.allOf(HddsProtos.BlockTokenSecretProto.AccessModeProto.class),
+        expiryTime, cert.getSerialNumber().toString());
+  }
+
+  @Test
+  public void testAsymmetricTokenPerf() throws NoSuchAlgorithmException,
+      CertificateEncodingException, NoSuchProviderException,
+      InvalidKeyException, SignatureException {
+    final int testTokenCount = 1000;
+    List<OzoneBlockTokenIdentifier> tokenIds = new ArrayList<>();
+    List<byte[]> tokenPasswordAsym = new ArrayList<>();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenIds.add(generateTestToken());
+    }
+
+    KeyPair kp = KeyStoreTestUtil.generateKeyPair("RSA");
+
+    // Create Ozone Master certificate (SCM CA issued cert) and key store
+    X509Certificate certificate;
+    certificate = KeyStoreTestUtil.generateCertificate("CN=OzoneMaster",
+        kp, 30, "SHA256withRSA");
+
+    long startTime = Time.monotonicNowNanos();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenPasswordAsym.add(
+          signTokenAsymmetric(tokenIds.get(i), kp.getPrivate()));
+    }
+    long duration = Time.monotonicNowNanos() - startTime;
+    LOG.info("Average token sign time with HmacSha256(RSA/1024 key) is {} ns",
+        duration / testTokenCount);
+
+    startTime = Time.monotonicNowNanos();
+    for (int i = 0; i < testTokenCount; i++) {
+      verifyTokenAsymmetric(tokenIds.get(i), tokenPasswordAsym.get(i),
+          certificate);
+    }
+    duration = Time.monotonicNowNanos() - startTime;
+    LOG.info("Average token verify time with HmacSha256(RSA/1024 key) "
+        + "is {} ns", duration / testTokenCount);
+  }
+
+  @Test
+  public void testSymmetricTokenPerf() {
+    String hmacSHA1 = "HmacSHA1";
+    String hmacSHA256 = "HmacSHA256";
+
+    testSymmetricTokenPerfHelper(hmacSHA1, 64);
+    testSymmetricTokenPerfHelper(hmacSHA256, 1024);
+  }
+
+  public void testSymmetricTokenPerfHelper(String hmacAlgorithm, int keyLen) {
+    final int testTokenCount = 1000;
+    List<OzoneBlockTokenIdentifier> tokenIds = new ArrayList<>();
+    List<byte[]> tokenPasswordSym = new ArrayList<>();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenIds.add(generateTestToken());
+    }
+
+    KeyGenerator keyGen;
+    try {
+      keyGen = KeyGenerator.getInstance(hmacAlgorithm);
+      keyGen.init(keyLen);
+    } catch (NoSuchAlgorithmException nsa) {
+      throw new IllegalArgumentException("Can't find " + hmacAlgorithm +
+          " algorithm.");
+    }
+
+    Mac mac;
+    try {
+      mac = Mac.getInstance(hmacAlgorithm);
+    } catch (NoSuchAlgorithmException nsa) {
+      throw new IllegalArgumentException("Can't find " + hmacAlgorithm +
+          " algorithm.");
+    }
+
+    SecretKey secretKey = keyGen.generateKey();
+
+    long startTime = Time.monotonicNowNanos();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenPasswordSym.add(
+          signTokenSymmetric(tokenIds.get(i), mac, secretKey));
+    }
+    long duration = Time.monotonicNowNanos() - startTime;
+    LOG.info("Average token sign time with {}({} symmetric key) is {} ns",
+        hmacAlgorithm, keyLen, duration / testTokenCount);
+  }
+
+  // TODO: verify certificate with a trust store
+  public boolean verifyCert(Certificate certificate) {
+    return true;
+  }
+}

+ 300 - 0
hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/security/TestOzoneTokenIdentifier.java

@@ -0,0 +1,300 @@
+/**
+ * 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.ozone.security;
+
+import java.io.DataInputStream;
+import java.io.DataOutputStream;
+import java.io.File;
+import java.io.FileInputStream;
+import java.io.FileOutputStream;
+import java.io.IOException;
+import java.security.GeneralSecurityException;
+import java.security.InvalidKeyException;
+import java.security.KeyPair;
+import java.security.NoSuchAlgorithmException;
+import java.security.NoSuchProviderException;
+import java.security.PrivateKey;
+import java.security.Signature;
+import java.security.SignatureException;
+import java.security.cert.Certificate;
+import java.security.cert.CertificateEncodingException;
+import java.security.cert.X509Certificate;
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.List;
+import java.util.Map;
+import javax.crypto.KeyGenerator;
+import javax.crypto.Mac;
+import javax.crypto.SecretKey;
+import org.apache.commons.lang3.RandomStringUtils;
+import org.apache.commons.lang3.RandomUtils;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.FileUtil;
+import org.apache.hadoop.io.Text;
+import org.apache.hadoop.security.ssl.KeyStoreTestUtil;
+import org.apache.hadoop.security.ssl.TestSSLFactory;
+import org.apache.hadoop.test.GenericTestUtils;
+import org.apache.hadoop.util.Time;
+import org.junit.AfterClass;
+import org.junit.Assert;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Test class for {@link OzoneTokenIdentifier}.
+ */
+public class TestOzoneTokenIdentifier {
+
+  private static final Logger LOG = LoggerFactory
+      .getLogger(TestOzoneTokenIdentifier.class);
+  private static final String BASEDIR = GenericTestUtils
+      .getTempPath(TestOzoneTokenIdentifier.class.getSimpleName());
+  private static final String KEYSTORES_DIR =
+      new File(BASEDIR).getAbsolutePath();
+  private static File base;
+  private static String sslConfsDir;
+  private static final String EXCLUDE_CIPHERS =
+      "TLS_ECDHE_RSA_WITH_RC4_128_SHA,"
+      + "SSL_DHE_RSA_EXPORT_WITH_DES40_CBC_SHA,  \n"
+      + "SSL_RSA_WITH_DES_CBC_SHA,"
+      + "SSL_DHE_RSA_WITH_DES_CBC_SHA,  "
+      + "SSL_RSA_EXPORT_WITH_RC4_40_MD5,\t \n"
+      + "SSL_RSA_EXPORT_WITH_DES40_CBC_SHA,"
+      + "SSL_RSA_WITH_RC4_128_MD5";
+
+  @BeforeClass
+  public static void setUp() throws Exception {
+    base = new File(BASEDIR);
+    FileUtil.fullyDelete(base);
+    base.mkdirs();
+  }
+
+  private Configuration createConfiguration(boolean clientCert,
+                                            boolean trustStore)
+      throws Exception {
+    Configuration conf = new Configuration();
+    KeyStoreTestUtil.setupSSLConfig(KEYSTORES_DIR, sslConfsDir, conf,
+        clientCert, trustStore, EXCLUDE_CIPHERS);
+    sslConfsDir = KeyStoreTestUtil.getClasspathDir(TestSSLFactory.class);
+    return conf;
+  }
+
+  @AfterClass
+  static public void cleanUp() throws Exception {
+    FileUtil.fullyDelete(base);
+    KeyStoreTestUtil.cleanupSSLConfig(KEYSTORES_DIR, sslConfsDir);
+  }
+
+  @Test
+  public void testSignToken() throws GeneralSecurityException, IOException {
+    String keystore = new File(KEYSTORES_DIR, "keystore.jks")
+        .getAbsolutePath();
+    String truststore = new File(KEYSTORES_DIR, "truststore.jks")
+        .getAbsolutePath();
+    String trustPassword = "trustPass";
+    String keyStorePassword = "keyStorePass";
+    String keyPassword = "keyPass";
+
+    // Create Ozone Master key pair
+    KeyPair keyPair = KeyStoreTestUtil.generateKeyPair("RSA");
+
+    // Create Ozone Master certificate (SCM CA issued cert) and key store
+    X509Certificate cert = KeyStoreTestUtil
+        .generateCertificate("CN=OzoneMaster", keyPair, 30, "SHA256withRSA");
+    KeyStoreTestUtil.createKeyStore(keystore, keyStorePassword, keyPassword,
+        "OzoneMaster", keyPair.getPrivate(), cert);
+
+    // Create trust store and put the certificate in the trust store
+    Map<String, X509Certificate> certs = Collections.singletonMap("server",
+        cert);
+    KeyStoreTestUtil.createTrustStore(truststore, trustPassword, certs);
+
+    // Sign the OzoneMaster Token with Ozone Master private key
+    PrivateKey privateKey = keyPair.getPrivate();
+    OzoneTokenIdentifier tokenId = new OzoneTokenIdentifier();
+    byte[] signedToken = signTokenAsymmetric(tokenId, privateKey);
+
+    // Verify a valid signed OzoneMaster Token with Ozone Master
+    // public key(certificate)
+    boolean isValidToken = verifyTokenAsymmetric(tokenId, signedToken, cert);
+    LOG.info("{} is {}", tokenId, isValidToken ? "valid." : "invalid.");
+
+    // Verify an invalid signed OzoneMaster Token with Ozone Master
+    // public key(certificate)
+    tokenId = new OzoneTokenIdentifier(new Text("oozie"),
+        new Text("rm"), new Text("client"));
+    LOG.info("Unsigned token {} is {}", tokenId,
+        verifyTokenAsymmetric(tokenId, RandomUtils.nextBytes(128), cert));
+
+  }
+
+  public byte[] signTokenAsymmetric(OzoneTokenIdentifier tokenId,
+      PrivateKey privateKey) throws NoSuchAlgorithmException,
+      InvalidKeyException, SignatureException {
+    Signature rsaSignature = Signature.getInstance("SHA256withRSA");
+    rsaSignature.initSign(privateKey);
+    rsaSignature.update(tokenId.getBytes());
+    byte[] signature = rsaSignature.sign();
+    return signature;
+  }
+
+  public boolean verifyTokenAsymmetric(OzoneTokenIdentifier tokenId,
+      byte[] signature, Certificate certificate) throws InvalidKeyException,
+      NoSuchAlgorithmException, SignatureException {
+    Signature rsaSignature = Signature.getInstance("SHA256withRSA");
+    rsaSignature.initVerify(certificate);
+    rsaSignature.update(tokenId.getBytes());
+    boolean isValide = rsaSignature.verify(signature);
+    return isValide;
+  }
+
+  private byte[] signTokenSymmetric(OzoneTokenIdentifier identifier,
+      Mac mac, SecretKey key) {
+    try {
+      mac.init(key);
+    } catch (InvalidKeyException ike) {
+      throw new IllegalArgumentException("Invalid key to HMAC computation",
+          ike);
+    }
+    return mac.doFinal(identifier.getBytes());
+  }
+
+  OzoneTokenIdentifier generateTestToken() {
+    return new OzoneTokenIdentifier(
+        new Text(RandomStringUtils.randomAlphabetic(6)),
+        new Text(RandomStringUtils.randomAlphabetic(5)),
+        new Text(RandomStringUtils.randomAlphabetic(4)));
+  }
+
+  @Test
+  public void testAsymmetricTokenPerf() throws NoSuchAlgorithmException,
+      CertificateEncodingException, NoSuchProviderException,
+      InvalidKeyException, SignatureException {
+    final int testTokenCount = 1000;
+    List<OzoneTokenIdentifier> tokenIds = new ArrayList<>();
+    List<byte[]> tokenPasswordAsym = new ArrayList<>();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenIds.add(generateTestToken());
+    }
+
+    KeyPair keyPair = KeyStoreTestUtil.generateKeyPair("RSA");
+
+    // Create Ozone Master certificate (SCM CA issued cert) and key store
+    X509Certificate cert;
+    cert = KeyStoreTestUtil.generateCertificate("CN=OzoneMaster",
+        keyPair, 30, "SHA256withRSA");
+
+    long startTime = Time.monotonicNowNanos();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenPasswordAsym.add(
+          signTokenAsymmetric(tokenIds.get(i), keyPair.getPrivate()));
+    }
+    long duration = Time.monotonicNowNanos() - startTime;
+    LOG.info("Average token sign time with HmacSha256(RSA/1024 key) is {} ns",
+        duration/testTokenCount);
+
+    startTime = Time.monotonicNowNanos();
+    for (int i = 0; i < testTokenCount; i++) {
+      verifyTokenAsymmetric(tokenIds.get(i), tokenPasswordAsym.get(i), cert);
+    }
+    duration = Time.monotonicNowNanos() - startTime;
+    LOG.info("Average token verify time with HmacSha256(RSA/1024 key) "
+            + "is {} ns", duration/testTokenCount);
+  }
+
+  @Test
+  public void testSymmetricTokenPerf() {
+    String hmacSHA1 = "HmacSHA1";
+    String hmacSHA256 = "HmacSHA256";
+
+    testSymmetricTokenPerfHelper(hmacSHA1, 64);
+    testSymmetricTokenPerfHelper(hmacSHA256, 1024);
+  }
+
+
+  public void testSymmetricTokenPerfHelper(String hmacAlgorithm, int keyLen) {
+    final int testTokenCount = 1000;
+    List<OzoneTokenIdentifier> tokenIds = new ArrayList<>();
+    List<byte[]> tokenPasswordSym = new ArrayList<>();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenIds.add(generateTestToken());
+    }
+
+    KeyGenerator keyGen;
+    try {
+      keyGen = KeyGenerator.getInstance(hmacAlgorithm);
+      keyGen.init(keyLen);
+    } catch (NoSuchAlgorithmException nsa) {
+      throw new IllegalArgumentException("Can't find " + hmacAlgorithm +
+          " algorithm.");
+    }
+
+    Mac mac;
+    try {
+      mac =  Mac.getInstance(hmacAlgorithm);
+    } catch (NoSuchAlgorithmException nsa) {
+      throw new IllegalArgumentException("Can't find " + hmacAlgorithm +
+          " algorithm.");
+    }
+
+    SecretKey secretKey = keyGen.generateKey();
+
+    long startTime = Time.monotonicNowNanos();
+    for (int i = 0; i < testTokenCount; i++) {
+      tokenPasswordSym.add(
+          signTokenSymmetric(tokenIds.get(i), mac, secretKey));
+    }
+    long duration = Time.monotonicNowNanos() - startTime;
+    LOG.info("Average token sign time with {}({} symmetric key) is {} ns",
+        hmacAlgorithm, keyLen, duration/testTokenCount);
+  }
+
+  /*
+   * Test serialization/deserialization of OzoneTokenIdentifier.
+   */
+  @Test
+  public void testReadWriteInProtobuf() throws IOException {
+    OzoneTokenIdentifier id = getIdentifierInst();
+    File idFile = new File(BASEDIR + "/tokenFile");
+
+    FileOutputStream fop = new FileOutputStream(idFile);
+    DataOutputStream dataOutputStream = new DataOutputStream(fop);
+    id.write(dataOutputStream);
+    fop.close();
+
+    FileInputStream fis = new FileInputStream(idFile);
+    DataInputStream dis = new DataInputStream(fis);
+    OzoneTokenIdentifier id2 = new OzoneTokenIdentifier();
+
+    id2.readFields(dis);
+    Assert.assertEquals(id, id2);
+  }
+
+
+  public OzoneTokenIdentifier getIdentifierInst() {
+    OzoneTokenIdentifier id = new OzoneTokenIdentifier();
+    id.setOwner(new Text("User1"));
+    id.setRenewer(new Text("yarn"));
+    id.setIssueDate(Time.now());
+    id.setMaxDate(Time.now() + 5000);
+    id.setSequenceNumber(1);
+    return id;
+  }
+}

+ 21 - 0
hadoop-ozone/ozone-manager/src/test/java/org/apache/hadoop/ozone/security/package-info.java

@@ -0,0 +1,21 @@
+/**
+ * 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.ozone.security;
+/**
+ * Ozone security tests.
+ */