YARN-2743. Fixed a bug in ResourceManager that was causing RMDelegationToken identifiers to be tampered and thus causing app submission failures in secure mode. Contributed by Jian He.

(cherry picked from commit 0186645505)
This commit is contained in:
Vinod Kumar Vavilapalli 2014-10-26 11:14:34 -07:00
parent 7bb5071a9e
commit 259db7a726
13 changed files with 199 additions and 333 deletions

View File

@ -159,7 +159,7 @@ extends TokenIdentifier {
return masterKeyId; return masterKeyId;
} }
static boolean isEqual(Object a, Object b) { protected static boolean isEqual(Object a, Object b) {
return a == null ? b == null : a.equals(b); return a == null ? b == null : a.equals(b);
} }

View File

@ -18,15 +18,8 @@
package org.apache.hadoop.security.token.delegation; package org.apache.hadoop.security.token.delegation;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.io.MD5Hash;
import org.apache.hadoop.io.Text;
import java.io.ByteArrayInputStream; import java.io.ByteArrayInputStream;
import java.io.ByteArrayOutputStream;
import java.io.DataInputStream; import java.io.DataInputStream;
import java.io.DataOutputStream;
import java.io.IOException; import java.io.IOException;
import java.util.Arrays; import java.util.Arrays;
import java.util.HashMap; import java.util.HashMap;
@ -39,10 +32,13 @@ import javax.crypto.SecretKey;
import org.apache.commons.logging.Log; import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory; import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.classification.InterfaceAudience;
import org.apache.hadoop.classification.InterfaceStability;
import org.apache.hadoop.io.Text;
import org.apache.hadoop.security.AccessControlException; import org.apache.hadoop.security.AccessControlException;
import org.apache.hadoop.security.HadoopKerberosName; import org.apache.hadoop.security.HadoopKerberosName;
import org.apache.hadoop.security.token.Token;
import org.apache.hadoop.security.token.SecretManager; import org.apache.hadoop.security.token.SecretManager;
import org.apache.hadoop.security.token.Token;
import org.apache.hadoop.util.Daemon; import org.apache.hadoop.util.Daemon;
import org.apache.hadoop.util.Time; import org.apache.hadoop.util.Time;
@ -386,7 +382,8 @@ extends AbstractDelegationTokenIdentifier>
identifier.setMaxDate(now + tokenMaxLifetime); identifier.setMaxDate(now + tokenMaxLifetime);
identifier.setMasterKeyId(currentKey.getKeyId()); identifier.setMasterKeyId(currentKey.getKeyId());
identifier.setSequenceNumber(sequenceNum); identifier.setSequenceNumber(sequenceNum);
LOG.info("Creating password for identifier: [" + MD5Hash.digest(identifier.getBytes()) + ", " + currentKey.getKeyId() + "]"); LOG.info("Creating password for identifier: " + identifier
+ ", currentKey: " + currentKey.getKeyId());
byte[] password = createPassword(identifier.getBytes(), currentKey.getKey()); byte[] password = createPassword(identifier.getBytes(), currentKey.getKey());
DelegationTokenInformation tokenInfo = new DelegationTokenInformation(now DelegationTokenInformation tokenInfo = new DelegationTokenInformation(now
+ tokenRenewInterval, password, getTrackingIdIfEnabled(identifier)); + tokenRenewInterval, password, getTrackingIdIfEnabled(identifier));

View File

@ -574,6 +574,10 @@ Release 2.6.0 - UNRELEASED
YARN-2715. Fixed ResourceManager to respect common configurations for proxy YARN-2715. Fixed ResourceManager to respect common configurations for proxy
users/groups beyond just the YARN level config. (Zhijie Shen via vinodkv) users/groups beyond just the YARN level config. (Zhijie Shen via vinodkv)
YARN-2743. Fixed a bug in ResourceManager that was causing RMDelegationToken
identifiers to be tampered and thus causing app submission failures in
secure mode. (Jian He via vinodkv)
BREAKDOWN OF YARN-1051 SUBTASKS AND RELATED JIRAS BREAKDOWN OF YARN-1051 SUBTASKS AND RELATED JIRAS
YARN-1707. Introduce APIs to add/remove/resize queues in the YARN-1707. Introduce APIs to add/remove/resize queues in the

View File

@ -22,193 +22,61 @@ import java.io.DataOutput;
import java.io.DataOutputStream; import java.io.DataOutputStream;
import java.io.IOException; import java.io.IOException;
import org.apache.hadoop.classification.InterfaceAudience.Private;
import org.apache.hadoop.io.Text; import org.apache.hadoop.io.Text;
import org.apache.hadoop.security.HadoopKerberosName;
import org.apache.hadoop.security.UserGroupInformation;
import org.apache.hadoop.security.UserGroupInformation.AuthenticationMethod;
import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenIdentifier; import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenIdentifier;
import org.apache.hadoop.yarn.proto.YarnSecurityTokenProtos.YARNDelegationTokenIdentifierProto; import org.apache.hadoop.yarn.proto.YarnSecurityTokenProtos.YARNDelegationTokenIdentifierProto;
@Private
public abstract class YARNDelegationTokenIdentifier extends public abstract class YARNDelegationTokenIdentifier extends
AbstractDelegationTokenIdentifier { AbstractDelegationTokenIdentifier {
YARNDelegationTokenIdentifierProto.Builder builder = YARNDelegationTokenIdentifierProto.Builder builder =
YARNDelegationTokenIdentifierProto.newBuilder(); YARNDelegationTokenIdentifierProto.newBuilder();
public YARNDelegationTokenIdentifier() {} public YARNDelegationTokenIdentifier() {
}
public YARNDelegationTokenIdentifier(Text owner, Text renewer, Text realUser) { public YARNDelegationTokenIdentifier(Text owner, Text renewer, Text realUser) {
setOwner(owner); super(owner, renewer, realUser);
setRenewer(renewer);
setRealUser(realUser);
}
/**
* Get the username encoded in the token identifier
*
* @return the username or owner
*/
@Override
public UserGroupInformation getUser() {
String owner = getOwner() == null ? null : getOwner().toString();
String realUser = getRealUser() == null ? null: getRealUser().toString();
if ( (owner == null) || (owner.toString().isEmpty())) {
return null;
}
final UserGroupInformation realUgi;
final UserGroupInformation ugi;
if ((realUser == null) || (realUser.toString().isEmpty())
|| realUser.equals(owner)) {
ugi = realUgi = UserGroupInformation.createRemoteUser(owner.toString());
} else {
realUgi = UserGroupInformation.createRemoteUser(realUser.toString());
ugi = UserGroupInformation.createProxyUser(owner.toString(), realUgi);
}
realUgi.setAuthenticationMethod(AuthenticationMethod.TOKEN);
return ugi;
} }
public Text getOwner() { public YARNDelegationTokenIdentifier(
String owner = builder.getOwner(); YARNDelegationTokenIdentifierProto.Builder builder) {
if (owner == null) { this.builder = builder;
return null;
} else {
return new Text(owner);
}
} }
@Override @Override
public void setOwner(Text owner) { public synchronized void readFields(DataInput in) throws IOException {
if (builder != null && owner != null) {
builder.setOwner(owner.toString());
}
}
public Text getRenewer() {
String renewer = builder.getRenewer();
if (renewer == null) {
return null;
} else {
return new Text(renewer);
}
}
@Override
public void setRenewer(Text renewer) {
if (builder != null && renewer != null) {
HadoopKerberosName renewerKrbName = new HadoopKerberosName(renewer.toString());
try {
builder.setRenewer(renewerKrbName.getShortName());
} catch (IOException e) {
throw new RuntimeException(e);
}
}
}
public Text getRealUser() {
String realUser = builder.getRealUser();
if (realUser == null) {
return null;
} else {
return new Text(realUser);
}
}
@Override
public void setRealUser(Text realUser) {
if (builder != null && realUser != null) {
builder.setRealUser(realUser.toString());
}
}
public void setIssueDate(long issueDate) {
builder.setIssueDate(issueDate);
}
public long getIssueDate() {
return builder.getIssueDate();
}
public void setRenewDate(long renewDate) {
builder.setRenewDate(renewDate);
}
public long getRenewDate() {
return builder.getRenewDate();
}
public void setMaxDate(long maxDate) {
builder.setMaxDate(maxDate);
}
public long getMaxDate() {
return builder.getMaxDate();
}
public void setSequenceNumber(int seqNum) {
builder.setSequenceNumber(seqNum);
}
public int getSequenceNumber() {
return builder.getSequenceNumber();
}
public void setMasterKeyId(int newId) {
builder.setMasterKeyId(newId);
}
public int getMasterKeyId() {
return builder.getMasterKeyId();
}
protected 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 YARNDelegationTokenIdentifier) {
YARNDelegationTokenIdentifier that = (YARNDelegationTokenIdentifier) obj;
return this.getSequenceNumber() == that.getSequenceNumber()
&& this.getIssueDate() == that.getIssueDate()
&& this.getMaxDate() == that.getMaxDate()
&& this.getMasterKeyId() == that.getMasterKeyId()
&& isEqual(this.getOwner(), that.getOwner())
&& isEqual(this.getRenewer(), that.getRenewer())
&& isEqual(this.getRealUser(), that.getRealUser());
}
return false;
}
@Override
public int hashCode() {
return this.getSequenceNumber();
}
@Override
public void readFields(DataInput in) throws IOException {
builder.mergeFrom((DataInputStream) in); builder.mergeFrom((DataInputStream) in);
} if (builder.getOwner() != null) {
setOwner(new Text(builder.getOwner()));
@Override }
public void write(DataOutput out) throws IOException { if (builder.getRenewer() != null) {
builder.build().writeTo((DataOutputStream)out); setRenewer(new Text(builder.getRenewer()));
} }
if (builder.getRealUser() != null) {
@Override setRealUser(new Text(builder.getRealUser()));
public String toString() { }
StringBuilder buffer = new StringBuilder(); setIssueDate(builder.getIssueDate());
buffer setMaxDate(builder.getMaxDate());
.append("owner=" + getOwner() + ", renewer=" + getRenewer() + ", realUser=" setSequenceNumber(builder.getSequenceNumber());
+ getRealUser() + ", issueDate=" + getIssueDate() setMasterKeyId(builder.getMasterKeyId());
+ ", maxDate=" + getMaxDate() + ", sequenceNumber="
+ getSequenceNumber() + ", masterKeyId="
+ getMasterKeyId());
return buffer.toString();
} }
@Override
public synchronized void write(DataOutput out) throws IOException {
builder.setOwner(getOwner().toString());
builder.setRenewer(getRenewer().toString());
builder.setRealUser(getRealUser().toString());
builder.setIssueDate(getIssueDate());
builder.setMaxDate(getMaxDate());
builder.setSequenceNumber(getSequenceNumber());
builder.setMasterKeyId(getMasterKeyId());
builder.build().writeTo((DataOutputStream) out);
}
public YARNDelegationTokenIdentifierProto getProto() {
return builder.build();
}
} }

View File

@ -63,7 +63,6 @@ message YARNDelegationTokenIdentifierProto {
optional int64 issueDate = 4; optional int64 issueDate = 4;
optional int64 maxDate = 5; optional int64 maxDate = 5;
optional int32 sequenceNumber = 6; optional int32 sequenceNumber = 6;
optional int32 masterKeyId = 7 [default = -1]; optional int32 masterKeyId = 7;
optional int64 renewDate = 8;
} }

View File

@ -257,7 +257,6 @@ public class TestYARNTokenIdentifier {
long maxDate = 2; long maxDate = 2;
int sequenceNumber = 3; int sequenceNumber = 3;
int masterKeyId = 4; int masterKeyId = 4;
long renewDate = 5;
TimelineDelegationTokenIdentifier token = TimelineDelegationTokenIdentifier token =
new TimelineDelegationTokenIdentifier(owner, renewer, realUser); new TimelineDelegationTokenIdentifier(owner, renewer, realUser);
@ -265,7 +264,6 @@ public class TestYARNTokenIdentifier {
token.setMaxDate(maxDate); token.setMaxDate(maxDate);
token.setSequenceNumber(sequenceNumber); token.setSequenceNumber(sequenceNumber);
token.setMasterKeyId(masterKeyId); token.setMasterKeyId(masterKeyId);
token.setRenewDate(renewDate);
TimelineDelegationTokenIdentifier anotherToken = TimelineDelegationTokenIdentifier anotherToken =
new TimelineDelegationTokenIdentifier(); new TimelineDelegationTokenIdentifier();
@ -299,9 +297,6 @@ public class TestYARNTokenIdentifier {
Assert.assertEquals("masterKeyId from proto is not the same with original token", Assert.assertEquals("masterKeyId from proto is not the same with original token",
anotherToken.getMasterKeyId(), masterKeyId); anotherToken.getMasterKeyId(), masterKeyId);
Assert.assertEquals("renewDate from proto is not the same with original token",
anotherToken.getRenewDate(), renewDate);
} }
} }

View File

@ -279,6 +279,7 @@
<imports> <imports>
<param>${basedir}/../../../../hadoop-common-project/hadoop-common/src/main/proto</param> <param>${basedir}/../../../../hadoop-common-project/hadoop-common/src/main/proto</param>
<param>${basedir}/../../hadoop-yarn-api/src/main/proto</param> <param>${basedir}/../../hadoop-yarn-api/src/main/proto</param>
<param>${basedir}/../../hadoop-yarn-common/src/main/proto/server/</param>
<param>${basedir}/../hadoop-yarn-server-common/src/main/proto</param> <param>${basedir}/../hadoop-yarn-server-common/src/main/proto</param>
<param>${basedir}/src/main/proto</param> <param>${basedir}/src/main/proto</param>
</imports> </imports>

View File

@ -47,9 +47,9 @@ import org.apache.hadoop.yarn.api.records.ApplicationId;
import org.apache.hadoop.yarn.conf.YarnConfiguration; import org.apache.hadoop.yarn.conf.YarnConfiguration;
import org.apache.hadoop.yarn.proto.YarnServerCommonProtos.VersionProto; import org.apache.hadoop.yarn.proto.YarnServerCommonProtos.VersionProto;
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.AMRMTokenSecretManagerStateProto; import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.AMRMTokenSecretManagerStateProto;
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.EpochProto;
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.ApplicationAttemptStateDataProto; import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.ApplicationAttemptStateDataProto;
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.ApplicationStateDataProto; import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.ApplicationStateDataProto;
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.EpochProto;
import org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier; import org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier;
import org.apache.hadoop.yarn.server.records.Version; import org.apache.hadoop.yarn.server.records.Version;
import org.apache.hadoop.yarn.server.records.impl.pb.VersionPBImpl; import org.apache.hadoop.yarn.server.records.impl.pb.VersionPBImpl;
@ -57,6 +57,7 @@ import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.AMRMTokenS
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationAttemptStateData; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationAttemptStateData;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationStateData; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationStateData;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.Epoch; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.Epoch;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.RMDelegationTokenIdentifierData;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.AMRMTokenSecretManagerStatePBImpl; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.AMRMTokenSecretManagerStatePBImpl;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationAttemptStateDataPBImpl; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationAttemptStateDataPBImpl;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationStateDataPBImpl; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationStateDataPBImpl;
@ -369,12 +370,24 @@ public class FileSystemRMStateStore extends RMStateStore {
DelegationKey key = new DelegationKey(); DelegationKey key = new DelegationKey();
key.readFields(fsIn); key.readFields(fsIn);
rmState.rmSecretManagerState.masterKeyState.add(key); rmState.rmSecretManagerState.masterKeyState.add(key);
if (LOG.isDebugEnabled()) {
LOG.debug("Loaded delegation key: keyId=" + key.getKeyId()
+ ", expirationDate=" + key.getExpiryDate());
}
} else if (childNodeName.startsWith(DELEGATION_TOKEN_PREFIX)) { } else if (childNodeName.startsWith(DELEGATION_TOKEN_PREFIX)) {
RMDelegationTokenIdentifier identifier = new RMDelegationTokenIdentifier(); RMDelegationTokenIdentifierData identifierData =
identifier.readFields(fsIn); new RMDelegationTokenIdentifierData();
long renewDate = identifier.getRenewDate(); identifierData.readFields(fsIn);
RMDelegationTokenIdentifier identifier =
identifierData.getTokenIdentifier();
long renewDate = identifierData.getRenewDate();
rmState.rmSecretManagerState.delegationTokenState.put(identifier, rmState.rmSecretManagerState.delegationTokenState.put(identifier,
renewDate); renewDate);
if (LOG.isDebugEnabled()) {
LOG.debug("Loaded RMDelegationTokenIdentifier: " + identifier
+ " renewDate=" + renewDate);
}
} else { } else {
LOG.warn("Unknown file for recovering RMDelegationTokenSecretManager"); LOG.warn("Unknown file for recovering RMDelegationTokenSecretManager");
} }
@ -503,18 +516,15 @@ public class FileSystemRMStateStore extends RMStateStore {
Path nodeCreatePath = Path nodeCreatePath =
getNodePath(rmDTSecretManagerRoot, getNodePath(rmDTSecretManagerRoot,
DELEGATION_TOKEN_PREFIX + identifier.getSequenceNumber()); DELEGATION_TOKEN_PREFIX + identifier.getSequenceNumber());
ByteArrayOutputStream os = new ByteArrayOutputStream(); RMDelegationTokenIdentifierData identifierData =
DataOutputStream fsOut = new DataOutputStream(os); new RMDelegationTokenIdentifierData(identifier, renewDate);
identifier.setRenewDate(renewDate);
identifier.write(fsOut);
if (isUpdate) { if (isUpdate) {
LOG.info("Updating RMDelegationToken_" + identifier.getSequenceNumber()); LOG.info("Updating RMDelegationToken_" + identifier.getSequenceNumber());
updateFile(nodeCreatePath, os.toByteArray()); updateFile(nodeCreatePath, identifierData.toByteArray());
} else { } else {
LOG.info("Storing RMDelegationToken_" + identifier.getSequenceNumber()); LOG.info("Storing RMDelegationToken_" + identifier.getSequenceNumber());
writeFile(nodeCreatePath, os.toByteArray()); writeFile(nodeCreatePath, identifierData.toByteArray());
} }
fsOut.close();
// store sequence number // store sequence number
Path latestSequenceNumberPath = getNodePath(rmDTSecretManagerRoot, Path latestSequenceNumberPath = getNodePath(rmDTSecretManagerRoot,

View File

@ -50,13 +50,14 @@ import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.Appl
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.ApplicationStateDataProto; import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.ApplicationStateDataProto;
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.EpochProto; import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.EpochProto;
import org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier; import org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier;
import org.apache.hadoop.yarn.server.records.impl.pb.VersionPBImpl;
import org.apache.hadoop.yarn.server.records.Version; import org.apache.hadoop.yarn.server.records.Version;
import org.apache.hadoop.yarn.server.records.impl.pb.VersionPBImpl;
import org.apache.hadoop.yarn.server.resourcemanager.RMZKUtils; import org.apache.hadoop.yarn.server.resourcemanager.RMZKUtils;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.AMRMTokenSecretManagerState; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.AMRMTokenSecretManagerState;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationAttemptStateData; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationAttemptStateData;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationStateData; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.ApplicationStateData;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.Epoch; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.Epoch;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.RMDelegationTokenIdentifierData;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.AMRMTokenSecretManagerStatePBImpl; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.AMRMTokenSecretManagerStatePBImpl;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationAttemptStateDataPBImpl; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationAttemptStateDataPBImpl;
import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationStateDataPBImpl; import org.apache.hadoop.yarn.server.resourcemanager.recovery.records.impl.pb.ApplicationStateDataPBImpl;
@ -488,6 +489,10 @@ public class ZKRMStateStore extends RMStateStore {
DelegationKey key = new DelegationKey(); DelegationKey key = new DelegationKey();
key.readFields(fsIn); key.readFields(fsIn);
rmState.rmSecretManagerState.masterKeyState.add(key); rmState.rmSecretManagerState.masterKeyState.add(key);
if (LOG.isDebugEnabled()) {
LOG.debug("Loaded delegation key: keyId=" + key.getKeyId()
+ ", expirationDate=" + key.getExpiryDate());
}
} }
} finally { } finally {
is.close(); is.close();
@ -527,12 +532,18 @@ public class ZKRMStateStore extends RMStateStore {
try { try {
if (childNodeName.startsWith(DELEGATION_TOKEN_PREFIX)) { if (childNodeName.startsWith(DELEGATION_TOKEN_PREFIX)) {
RMDelegationTokenIdentifierData identifierData =
new RMDelegationTokenIdentifierData();
identifierData.readFields(fsIn);
RMDelegationTokenIdentifier identifier = RMDelegationTokenIdentifier identifier =
new RMDelegationTokenIdentifier(); identifierData.getTokenIdentifier();
identifier.readFields(fsIn); long renewDate = identifierData.getRenewDate();
long renewDate = identifier.getRenewDate();
rmState.rmSecretManagerState.delegationTokenState.put(identifier, rmState.rmSecretManagerState.delegationTokenState.put(identifier,
renewDate); renewDate);
if (LOG.isDebugEnabled()) {
LOG.debug("Loaded RMDelegationTokenIdentifier: " + identifier
+ " renewDate=" + renewDate);
}
} }
} finally { } finally {
is.close(); is.close();
@ -770,23 +781,20 @@ public class ZKRMStateStore extends RMStateStore {
String nodeCreatePath = String nodeCreatePath =
getNodePath(delegationTokensRootPath, DELEGATION_TOKEN_PREFIX getNodePath(delegationTokensRootPath, DELEGATION_TOKEN_PREFIX
+ rmDTIdentifier.getSequenceNumber()); + rmDTIdentifier.getSequenceNumber());
ByteArrayOutputStream tokenOs = new ByteArrayOutputStream();
DataOutputStream tokenOut = new DataOutputStream(tokenOs);
ByteArrayOutputStream seqOs = new ByteArrayOutputStream(); ByteArrayOutputStream seqOs = new ByteArrayOutputStream();
DataOutputStream seqOut = new DataOutputStream(seqOs); DataOutputStream seqOut = new DataOutputStream(seqOs);
RMDelegationTokenIdentifierData identifierData =
new RMDelegationTokenIdentifierData(rmDTIdentifier, renewDate);
try { try {
rmDTIdentifier.setRenewDate(renewDate);
rmDTIdentifier.write(tokenOut);
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
LOG.debug((isUpdate ? "Storing " : "Updating ") + "RMDelegationToken_" + LOG.debug((isUpdate ? "Storing " : "Updating ") + "RMDelegationToken_" +
rmDTIdentifier.getSequenceNumber()); rmDTIdentifier.getSequenceNumber());
} }
if (isUpdate) { if (isUpdate) {
opList.add(Op.setData(nodeCreatePath, tokenOs.toByteArray(), -1)); opList.add(Op.setData(nodeCreatePath, identifierData.toByteArray(), -1));
} else { } else {
opList.add(Op.create(nodeCreatePath, tokenOs.toByteArray(), zkAcl, opList.add(Op.create(nodeCreatePath, identifierData.toByteArray(), zkAcl,
CreateMode.PERSISTENT)); CreateMode.PERSISTENT));
} }
@ -799,7 +807,6 @@ public class ZKRMStateStore extends RMStateStore {
opList.add(Op.setData(dtSequenceNumberPath, seqOs.toByteArray(), -1)); opList.add(Op.setData(dtSequenceNumberPath, seqOs.toByteArray(), -1));
} finally { } finally {
tokenOs.close();
seqOs.close(); seqOs.close();
} }
} }

View File

@ -0,0 +1,61 @@
/**
* 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.resourcemanager.recovery.records;
import java.io.ByteArrayInputStream;
import java.io.DataInput;
import java.io.DataInputStream;
import java.io.IOException;
import org.apache.hadoop.yarn.proto.YarnServerResourceManagerRecoveryProtos.RMDelegationTokenIdentifierDataProto;
import org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier;
import org.apache.hadoop.yarn.security.client.YARNDelegationTokenIdentifier;
public class RMDelegationTokenIdentifierData {
RMDelegationTokenIdentifierDataProto.Builder builder =
RMDelegationTokenIdentifierDataProto.newBuilder();
public RMDelegationTokenIdentifierData() {}
public RMDelegationTokenIdentifierData(
YARNDelegationTokenIdentifier identifier, long renewdate) {
builder.setTokenIdentifier(identifier.getProto());
builder.setRenewDate(renewdate);
}
public void readFields(DataInput in) throws IOException {
builder.mergeFrom((DataInputStream) in);
}
public byte[] toByteArray() throws IOException {
return builder.build().toByteArray();
}
public RMDelegationTokenIdentifier getTokenIdentifier() throws IOException {
ByteArrayInputStream in =
new ByteArrayInputStream(builder.getTokenIdentifier().toByteArray());
RMDelegationTokenIdentifier identifer = new RMDelegationTokenIdentifier();
identifer.readFields(new DataInputStream(in));
return identifer;
}
public long getRenewDate() {
return builder.getRenewDate();
}
}

View File

@ -24,6 +24,7 @@ package hadoop.yarn;
import "yarn_server_common_protos.proto"; import "yarn_server_common_protos.proto";
import "yarn_protos.proto"; import "yarn_protos.proto";
import "yarn_security_token.proto";
//////////////////////////////////////////////////////////////////////// ////////////////////////////////////////////////////////////////////////
////// RM recovery related records ///////////////////////////////////// ////// RM recovery related records /////////////////////////////////////
@ -91,3 +92,8 @@ message AMRMTokenSecretManagerStateProto {
optional MasterKeyProto current_master_key = 1; optional MasterKeyProto current_master_key = 1;
optional MasterKeyProto next_master_key = 2; optional MasterKeyProto next_master_key = 2;
} }
message RMDelegationTokenIdentifierDataProto {
optional YARNDelegationTokenIdentifierProto token_identifier = 1;
optional int64 renewDate = 2;
}

View File

@ -20,155 +20,73 @@ package org.apache.hadoop.yarn.server.resourcemanager;
import java.io.DataInput; import java.io.DataInput;
import java.io.DataInputStream; import java.io.DataInputStream;
import java.io.DataOutput; import java.io.DataOutput;
import java.io.DataOutputStream;
import java.io.IOException; import java.io.IOException;
import org.apache.commons.io.IOUtils;
import org.apache.hadoop.io.Text; import org.apache.hadoop.io.Text;
import org.apache.hadoop.security.UserGroupInformation;
import org.apache.hadoop.security.UserGroupInformation.AuthenticationMethod;
import org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier;
import org.apache.hadoop.yarn.proto.YarnSecurityTestClientAMTokenProtos.RMDelegationTokenIdentifierForTestProto; import org.apache.hadoop.yarn.proto.YarnSecurityTestClientAMTokenProtos.RMDelegationTokenIdentifierForTestProto;
import org.apache.hadoop.yarn.security.client.RMDelegationTokenIdentifier;
public class RMDelegationTokenIdentifierForTest extends public class RMDelegationTokenIdentifierForTest extends
RMDelegationTokenIdentifier { RMDelegationTokenIdentifier {
private RMDelegationTokenIdentifierForTestProto proto; private RMDelegationTokenIdentifierForTestProto.Builder builder =
private RMDelegationTokenIdentifierForTestProto.Builder builder; RMDelegationTokenIdentifierForTestProto.newBuilder();
public RMDelegationTokenIdentifierForTest() { public RMDelegationTokenIdentifierForTest() {
} }
public RMDelegationTokenIdentifierForTest( public RMDelegationTokenIdentifierForTest(RMDelegationTokenIdentifier token,
RMDelegationTokenIdentifier token, String message) { String message) {
builder = RMDelegationTokenIdentifierForTestProto.newBuilder();
if (token.getOwner() != null) { if (token.getOwner() != null) {
builder.setOwner(token.getOwner().toString()); setOwner(new Text(token.getOwner()));
} }
if (token.getRenewer() != null) { if (token.getRenewer() != null) {
builder.setRenewer(token.getRenewer().toString()); setRenewer(new Text(token.getRenewer()));
} }
if (token.getRealUser() != null) { if (token.getRealUser() != null) {
builder.setRealUser(token.getRealUser().toString()); setRealUser(new Text(token.getRealUser()));
} }
builder.setIssueDate(token.getIssueDate()); setIssueDate(token.getIssueDate());
builder.setMaxDate(token.getMaxDate()); setMaxDate(token.getMaxDate());
builder.setSequenceNumber(token.getSequenceNumber()); setSequenceNumber(token.getSequenceNumber());
builder.setMasterKeyId(token.getMasterKeyId()); setMasterKeyId(token.getMasterKeyId());
builder.setMessage(message); builder.setMessage(message);
proto = builder.build();
builder = null;
} }
@Override @Override
public void write(DataOutput out) throws IOException { public void write(DataOutput out) throws IOException {
out.write(proto.toByteArray()); builder.setOwner(getOwner().toString());
builder.setRenewer(getRenewer().toString());
builder.setRealUser(getRealUser().toString());
builder.setIssueDate(getIssueDate());
builder.setMaxDate(getMaxDate());
builder.setSequenceNumber(getSequenceNumber());
builder.setMasterKeyId(getMasterKeyId());
builder.setMessage(getMessage());
builder.build().writeTo((DataOutputStream) out);
} }
@Override @Override
public void readFields(DataInput in) throws IOException { public void readFields(DataInput in) throws IOException {
DataInputStream dis = (DataInputStream)in; builder.mergeFrom((DataInputStream) in);
byte[] buffer = IOUtils.toByteArray(dis); if (builder.getOwner() != null) {
proto = RMDelegationTokenIdentifierForTestProto.parseFrom(buffer); setOwner(new Text(builder.getOwner()));
}
/**
* Get the username encoded in the token identifier
*
* @return the username or owner
*/
@Override
public UserGroupInformation getUser() {
String owner = getOwner().toString();
String realUser = getRealUser().toString();
if ( (owner == null) || (owner.toString().isEmpty())) {
return null;
} }
final UserGroupInformation realUgi; if (builder.getRenewer() != null) {
final UserGroupInformation ugi; setRenewer(new Text(builder.getRenewer()));
if ((realUser == null) || (realUser.toString().isEmpty())
|| realUser.equals(owner)) {
ugi = realUgi = UserGroupInformation.createRemoteUser(owner.toString());
} else {
realUgi = UserGroupInformation.createRemoteUser(realUser.toString());
ugi = UserGroupInformation.createProxyUser(owner.toString(), realUgi);
} }
realUgi.setAuthenticationMethod(AuthenticationMethod.TOKEN); if (builder.getRealUser() != null) {
return ugi; setRealUser(new Text(builder.getRealUser()));
}
public Text getOwner() {
String owner = proto.getOwner();
if (owner == null) {
return null;
} else {
return new Text(owner);
} }
} setIssueDate(builder.getIssueDate());
setMaxDate(builder.getMaxDate());
public Text getRenewer() { setSequenceNumber(builder.getSequenceNumber());
String renewer = proto.getRenewer(); setMasterKeyId(builder.getMasterKeyId());
if (renewer == null) {
return null;
} else {
return new Text(renewer);
}
}
public Text getRealUser() {
String realUser = proto.getRealUser();
if (realUser == null) {
return null;
} else {
return new Text(realUser);
}
}
public void setIssueDate(long issueDate) {
RMDelegationTokenIdentifierForTestProto.Builder builder =
RMDelegationTokenIdentifierForTestProto.newBuilder(proto);
builder.setIssueDate(issueDate);
proto = builder.build();
}
public long getIssueDate() {
return proto.getIssueDate();
}
public void setMaxDate(long maxDate) {
RMDelegationTokenIdentifierForTestProto.Builder builder =
RMDelegationTokenIdentifierForTestProto.newBuilder(proto);
builder.setMaxDate(maxDate);
proto = builder.build();
}
public long getMaxDate() {
return proto.getMaxDate();
}
public void setSequenceNumber(int seqNum) {
RMDelegationTokenIdentifierForTestProto.Builder builder =
RMDelegationTokenIdentifierForTestProto.newBuilder(proto);
builder.setSequenceNumber(seqNum);
proto = builder.build();
}
public int getSequenceNumber() {
return proto.getSequenceNumber();
}
public void setMasterKeyId(int newId) {
RMDelegationTokenIdentifierForTestProto.Builder builder =
RMDelegationTokenIdentifierForTestProto.newBuilder(proto);
builder.setMasterKeyId(newId);
proto = builder.build();
}
public int getMasterKeyId() {
return proto.getMasterKeyId();
} }
public String getMessage() { public String getMessage() {
return proto.getMessage(); return builder.getMessage();
} }
@Override @Override
@ -189,10 +107,4 @@ public class RMDelegationTokenIdentifierForTest extends
} }
return false; return false;
} }
@Override
public int hashCode() {
return this.getSequenceNumber();
}
} }

View File

@ -28,6 +28,7 @@ import static org.mockito.Mockito.when;
import static org.mockito.Mockito.spy; import static org.mockito.Mockito.spy;
import java.util.ArrayList; import java.util.ArrayList;
import java.util.Arrays;
import java.util.HashMap; import java.util.HashMap;
import java.util.HashSet; import java.util.HashSet;
import java.util.Map; import java.util.Map;
@ -402,6 +403,7 @@ public class RMStateStoreTestBase extends ClientBaseWithFixes{
RMDelegationTokenIdentifier dtId1 = RMDelegationTokenIdentifier dtId1 =
new RMDelegationTokenIdentifier(new Text("owner1"), new RMDelegationTokenIdentifier(new Text("owner1"),
new Text("renewer1"), new Text("realuser1")); new Text("renewer1"), new Text("realuser1"));
byte[] tokenBeforeStore = dtId1.getBytes();
Long renewDate1 = new Long(System.currentTimeMillis()); Long renewDate1 = new Long(System.currentTimeMillis());
int sequenceNumber = 1111; int sequenceNumber = 1111;
store.storeRMDelegationTokenAndSequenceNumber(dtId1, renewDate1, store.storeRMDelegationTokenAndSequenceNumber(dtId1, renewDate1,
@ -423,6 +425,10 @@ public class RMStateStoreTestBase extends ClientBaseWithFixes{
Assert.assertEquals(keySet, secretManagerState.getMasterKeyState()); Assert.assertEquals(keySet, secretManagerState.getMasterKeyState());
Assert.assertEquals(sequenceNumber, Assert.assertEquals(sequenceNumber,
secretManagerState.getDTSequenceNumber()); secretManagerState.getDTSequenceNumber());
RMDelegationTokenIdentifier tokenAfterStore =
secretManagerState.getTokenState().keySet().iterator().next();
Assert.assertTrue(Arrays.equals(tokenBeforeStore,
tokenAfterStore.getBytes()));
// update RM delegation token; // update RM delegation token;
renewDate1 = new Long(System.currentTimeMillis()); renewDate1 = new Long(System.currentTimeMillis());