|
@@ -34,6 +34,7 @@ import org.apache.hadoop.yarn.api.records.ExecutionType;
|
|
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
import org.apache.hadoop.yarn.api.records.Priority;
|
|
import org.apache.hadoop.yarn.api.records.Priority;
|
|
import org.apache.hadoop.yarn.api.records.Resource;
|
|
import org.apache.hadoop.yarn.api.records.Resource;
|
|
|
|
+import org.apache.hadoop.yarn.api.records.impl.pb.LogAggregationContextPBImpl;
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
import org.apache.hadoop.yarn.nodelabels.CommonNodeLabelsManager;
|
|
import org.apache.hadoop.yarn.nodelabels.CommonNodeLabelsManager;
|
|
import org.apache.hadoop.yarn.proto.YarnSecurityTokenProtos.ContainerTokenIdentifierProto;
|
|
import org.apache.hadoop.yarn.proto.YarnSecurityTokenProtos.ContainerTokenIdentifierProto;
|
|
@@ -49,6 +50,15 @@ public class TestYARNTokenIdentifier {
|
|
|
|
|
|
@Test
|
|
@Test
|
|
public void testNMTokenIdentifier() throws IOException {
|
|
public void testNMTokenIdentifier() throws IOException {
|
|
|
|
+ testNMTokenIdentifier(false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Test
|
|
|
|
+ public void testNMTokenIdentifierOldFormat() throws IOException {
|
|
|
|
+ testNMTokenIdentifier(true);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ public void testNMTokenIdentifier(boolean oldFormat) throws IOException {
|
|
ApplicationAttemptId appAttemptId = ApplicationAttemptId.newInstance(
|
|
ApplicationAttemptId appAttemptId = ApplicationAttemptId.newInstance(
|
|
ApplicationId.newInstance(1, 1), 1);
|
|
ApplicationId.newInstance(1, 1), 1);
|
|
NodeId nodeId = NodeId.newInstance("host0", 0);
|
|
NodeId nodeId = NodeId.newInstance("host0", 0);
|
|
@@ -59,8 +69,13 @@ public class TestYARNTokenIdentifier {
|
|
appAttemptId, nodeId, applicationSubmitter, masterKeyId);
|
|
appAttemptId, nodeId, applicationSubmitter, masterKeyId);
|
|
|
|
|
|
NMTokenIdentifier anotherToken = new NMTokenIdentifier();
|
|
NMTokenIdentifier anotherToken = new NMTokenIdentifier();
|
|
-
|
|
|
|
- byte[] tokenContent = token.getBytes();
|
|
|
|
|
|
+
|
|
|
|
+ byte[] tokenContent;
|
|
|
|
+ if (oldFormat) {
|
|
|
|
+ tokenContent = writeInOldFormat(token);
|
|
|
|
+ } else {
|
|
|
|
+ tokenContent = token.getBytes();
|
|
|
|
+ }
|
|
DataInputBuffer dib = new DataInputBuffer();
|
|
DataInputBuffer dib = new DataInputBuffer();
|
|
dib.reset(tokenContent, tokenContent.length);
|
|
dib.reset(tokenContent, tokenContent.length);
|
|
anotherToken.readFields(dib);
|
|
anotherToken.readFields(dib);
|
|
@@ -89,6 +104,15 @@ public class TestYARNTokenIdentifier {
|
|
|
|
|
|
@Test
|
|
@Test
|
|
public void testAMRMTokenIdentifier() throws IOException {
|
|
public void testAMRMTokenIdentifier() throws IOException {
|
|
|
|
+ testAMRMTokenIdentifier(false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Test
|
|
|
|
+ public void testAMRMTokenIdentifierOldFormat() throws IOException {
|
|
|
|
+ testAMRMTokenIdentifier(true);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ public void testAMRMTokenIdentifier(boolean oldFormat) throws IOException {
|
|
ApplicationAttemptId appAttemptId = ApplicationAttemptId.newInstance(
|
|
ApplicationAttemptId appAttemptId = ApplicationAttemptId.newInstance(
|
|
ApplicationId.newInstance(1, 1), 1);
|
|
ApplicationId.newInstance(1, 1), 1);
|
|
int masterKeyId = 1;
|
|
int masterKeyId = 1;
|
|
@@ -96,7 +120,13 @@ public class TestYARNTokenIdentifier {
|
|
AMRMTokenIdentifier token = new AMRMTokenIdentifier(appAttemptId, masterKeyId);
|
|
AMRMTokenIdentifier token = new AMRMTokenIdentifier(appAttemptId, masterKeyId);
|
|
|
|
|
|
AMRMTokenIdentifier anotherToken = new AMRMTokenIdentifier();
|
|
AMRMTokenIdentifier anotherToken = new AMRMTokenIdentifier();
|
|
- byte[] tokenContent = token.getBytes();
|
|
|
|
|
|
+
|
|
|
|
+ byte[] tokenContent;
|
|
|
|
+ if (oldFormat) {
|
|
|
|
+ tokenContent = writeInOldFormat(token);
|
|
|
|
+ } else {
|
|
|
|
+ tokenContent = token.getBytes();
|
|
|
|
+ }
|
|
DataInputBuffer dib = new DataInputBuffer();
|
|
DataInputBuffer dib = new DataInputBuffer();
|
|
dib.reset(tokenContent, tokenContent.length);
|
|
dib.reset(tokenContent, tokenContent.length);
|
|
anotherToken.readFields(dib);
|
|
anotherToken.readFields(dib);
|
|
@@ -139,7 +169,7 @@ public class TestYARNTokenIdentifier {
|
|
Assert.assertEquals("clientName from proto is not the same with original token",
|
|
Assert.assertEquals("clientName from proto is not the same with original token",
|
|
anotherToken.getClientName(), clientName);
|
|
anotherToken.getClientName(), clientName);
|
|
}
|
|
}
|
|
-
|
|
|
|
|
|
+
|
|
@Test
|
|
@Test
|
|
public void testContainerTokenIdentifierProtoMissingFields()
|
|
public void testContainerTokenIdentifierProtoMissingFields()
|
|
throws IOException {
|
|
throws IOException {
|
|
@@ -166,6 +196,17 @@ public class TestYARNTokenIdentifier {
|
|
|
|
|
|
@Test
|
|
@Test
|
|
public void testContainerTokenIdentifier() throws IOException {
|
|
public void testContainerTokenIdentifier() throws IOException {
|
|
|
|
+ testContainerTokenIdentifier(false, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Test
|
|
|
|
+ public void testContainerTokenIdentifierOldFormat() throws IOException {
|
|
|
|
+ testContainerTokenIdentifier(true, true);
|
|
|
|
+ testContainerTokenIdentifier(true, false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ public void testContainerTokenIdentifier(boolean oldFormat,
|
|
|
|
+ boolean withLogAggregation) throws IOException {
|
|
ContainerId containerID = ContainerId.newContainerId(
|
|
ContainerId containerID = ContainerId.newContainerId(
|
|
ApplicationAttemptId.newInstance(ApplicationId.newInstance(
|
|
ApplicationAttemptId.newInstance(ApplicationId.newInstance(
|
|
1, 1), 1), 1);
|
|
1, 1), 1), 1);
|
|
@@ -183,8 +224,13 @@ public class TestYARNTokenIdentifier {
|
|
masterKeyId, rmIdentifier, priority, creationTime);
|
|
masterKeyId, rmIdentifier, priority, creationTime);
|
|
|
|
|
|
ContainerTokenIdentifier anotherToken = new ContainerTokenIdentifier();
|
|
ContainerTokenIdentifier anotherToken = new ContainerTokenIdentifier();
|
|
-
|
|
|
|
- byte[] tokenContent = token.getBytes();
|
|
|
|
|
|
+
|
|
|
|
+ byte[] tokenContent;
|
|
|
|
+ if (oldFormat) {
|
|
|
|
+ tokenContent = writeInOldFormat(token, withLogAggregation);
|
|
|
|
+ } else {
|
|
|
|
+ tokenContent = token.getBytes();
|
|
|
|
+ }
|
|
DataInputBuffer dib = new DataInputBuffer();
|
|
DataInputBuffer dib = new DataInputBuffer();
|
|
dib.reset(tokenContent, tokenContent.length);
|
|
dib.reset(tokenContent, tokenContent.length);
|
|
anotherToken.readFields(dib);
|
|
anotherToken.readFields(dib);
|
|
@@ -451,4 +497,67 @@ public class TestYARNTokenIdentifier {
|
|
anotherToken.getExecutionType());
|
|
anotherToken.getExecutionType());
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ @SuppressWarnings("deprecation")
|
|
|
|
+ private byte[] writeInOldFormat(ContainerTokenIdentifier token,
|
|
|
|
+ boolean withLogAggregation) throws IOException {
|
|
|
|
+ ByteArrayOutputStream baos = new ByteArrayOutputStream();
|
|
|
|
+ DataOutputStream out = new DataOutputStream(baos);
|
|
|
|
+ ApplicationAttemptId applicationAttemptId = token.getContainerID()
|
|
|
|
+ .getApplicationAttemptId();
|
|
|
|
+ ApplicationId applicationId = applicationAttemptId.getApplicationId();
|
|
|
|
+ out.writeLong(applicationId.getClusterTimestamp());
|
|
|
|
+ out.writeInt(applicationId.getId());
|
|
|
|
+ out.writeInt(applicationAttemptId.getAttemptId());
|
|
|
|
+ out.writeLong(token.getContainerID().getContainerId());
|
|
|
|
+ out.writeUTF(token.getNmHostAddress());
|
|
|
|
+ out.writeUTF(token.getApplicationSubmitter());
|
|
|
|
+ out.writeInt(token.getResource().getMemory());
|
|
|
|
+ out.writeInt(token.getResource().getVirtualCores());
|
|
|
|
+ out.writeLong(token.getExpiryTimeStamp());
|
|
|
|
+ out.writeInt(token.getMasterKeyId());
|
|
|
|
+ out.writeLong(token.getRMIdentifier());
|
|
|
|
+ out.writeInt(token.getPriority().getPriority());
|
|
|
|
+ out.writeLong(token.getCreationTime());
|
|
|
|
+ if (withLogAggregation) {
|
|
|
|
+ if (token.getLogAggregationContext() == null) {
|
|
|
|
+ out.writeInt(-1);
|
|
|
|
+ } else {
|
|
|
|
+ byte[] logAggregationContext = ((LogAggregationContextPBImpl)
|
|
|
|
+ token.getLogAggregationContext()).getProto().toByteArray();
|
|
|
|
+ out.writeInt(logAggregationContext.length);
|
|
|
|
+ out.write(logAggregationContext);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ out.close();
|
|
|
|
+ return baos.toByteArray();
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private byte[] writeInOldFormat(NMTokenIdentifier token) throws IOException {
|
|
|
|
+ ByteArrayOutputStream baos = new ByteArrayOutputStream();
|
|
|
|
+ DataOutputStream out = new DataOutputStream(baos);
|
|
|
|
+ ApplicationId applicationId = token.getApplicationAttemptId()
|
|
|
|
+ .getApplicationId();
|
|
|
|
+ out.writeLong(applicationId.getClusterTimestamp());
|
|
|
|
+ out.writeInt(applicationId.getId());
|
|
|
|
+ out.writeInt(token.getApplicationAttemptId().getAttemptId());
|
|
|
|
+ out.writeUTF(token.getNodeId().toString());
|
|
|
|
+ out.writeUTF(token.getApplicationSubmitter());
|
|
|
|
+ out.writeInt(token.getKeyId());
|
|
|
|
+ out.close();
|
|
|
|
+ return baos.toByteArray();
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private byte[] writeInOldFormat(AMRMTokenIdentifier token)
|
|
|
|
+ throws IOException {
|
|
|
|
+ ByteArrayOutputStream baos = new ByteArrayOutputStream();
|
|
|
|
+ DataOutputStream out = new DataOutputStream(baos);
|
|
|
|
+ ApplicationId applicationId = token.getApplicationAttemptId()
|
|
|
|
+ .getApplicationId();
|
|
|
|
+ out.writeLong(applicationId.getClusterTimestamp());
|
|
|
|
+ out.writeInt(applicationId.getId());
|
|
|
|
+ out.writeInt(token.getApplicationAttemptId().getAttemptId());
|
|
|
|
+ out.writeInt(token.getKeyId());
|
|
|
|
+ out.close();
|
|
|
|
+ return baos.toByteArray();
|
|
|
|
+ }
|
|
}
|
|
}
|