You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by su...@apache.org on 2018/08/25 15:50:05 UTC

[48/50] [abbrv] hadoop git commit: YARN-7892. Revisit NodeAttribute class structure. Contributed by Naganarasimha G R.

YARN-7892. Revisit NodeAttribute class structure. Contributed by  Naganarasimha G R.


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

Branch: refs/heads/YARN-3409
Commit: 2b3d54bcd0c59319f2a4b747faf11a3dd9018ef0
Parents: c190002
Author: bibinchundatt <bi...@apache.org>
Authored: Sun May 13 09:35:11 2018 +0530
Committer: Sunil G <su...@apache.org>
Committed: Sat Aug 25 21:10:57 2018 +0530

----------------------------------------------------------------------
 .../hadoop/mapred/ResourceMgrDelegate.java      |   9 +-
 .../GetAttributesToNodesRequest.java            |  34 +++--
 .../GetAttributesToNodesResponse.java           |  33 +++--
 .../GetClusterNodeAttributesResponse.java       |  23 +--
 .../hadoop/yarn/api/records/NodeAttribute.java  |  17 +--
 .../yarn/api/records/NodeAttributeInfo.java     |  62 ++++++++
 .../yarn/api/records/NodeAttributeKey.java      |  66 +++++++++
 .../yarn/api/records/NodeToAttributeValue.java  |  57 ++++++++
 .../src/main/proto/yarn_protos.proto            |  27 +++-
 .../src/main/proto/yarn_service_protos.proto    |   4 +-
 .../hadoop/yarn/client/api/YarnClient.java      |  22 +--
 .../yarn/client/api/impl/YarnClientImpl.java    |   9 +-
 .../pb/GetAttributesToNodesRequestPBImpl.java   |  45 +++---
 .../pb/GetAttributesToNodesResponsePBImpl.java  |  83 +++++++----
 .../GetClusterNodeAttributesResponsePBImpl.java |  68 ++++-----
 .../impl/pb/NodeAttributeInfoPBImpl.java        | 143 +++++++++++++++++++
 .../records/impl/pb/NodeAttributeKeyPBImpl.java | 140 ++++++++++++++++++
 .../records/impl/pb/NodeAttributePBImpl.java    |  79 ++++------
 .../impl/pb/NodeToAttributeValuePBImpl.java     | 137 ++++++++++++++++++
 .../yarn/nodelabels/NodeAttributesManager.java  |  16 ++-
 .../hadoop/yarn/nodelabels/NodeLabelUtil.java   |  14 +-
 .../hadoop/yarn/nodelabels/RMNodeAttribute.java |  18 +--
 .../hadoop/yarn/api/TestPBImplRecords.java      |  45 ++++--
 ...TestConfigurationNodeAttributesProvider.java |  17 ++-
 .../TestScriptBasedNodeAttributesProvider.java  |   5 +-
 .../server/resourcemanager/AdminService.java    |   2 +-
 .../server/resourcemanager/ClientRMService.java |  31 +++-
 .../resourcemanager/ResourceTrackerService.java |   2 +-
 .../nodelabels/NodeAttributesManagerImpl.java   | 115 ++++++++-------
 .../nodelabels/NodeLabelsUtils.java             |  19 +++
 .../webapp/dao/NodeAttributeInfo.java           |   4 +-
 .../resourcemanager/TestClientRMService.java    | 110 ++++++++------
 .../TestResourceTrackerService.java             |   4 +-
 .../TestFileSystemNodeAttributeStore.java       |   6 +-
 .../nodelabels/TestNodeAttributesManager.java   |   9 +-
 35 files changed, 1118 insertions(+), 357 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/main/java/org/apache/hadoop/mapred/ResourceMgrDelegate.java
----------------------------------------------------------------------
diff --git a/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/main/java/org/apache/hadoop/mapred/ResourceMgrDelegate.java b/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/main/java/org/apache/hadoop/mapred/ResourceMgrDelegate.java
index 1a7f308..2cb3716 100644
--- a/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/main/java/org/apache/hadoop/mapred/ResourceMgrDelegate.java
+++ b/hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/main/java/org/apache/hadoop/mapred/ResourceMgrDelegate.java
@@ -59,10 +59,13 @@ import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
 import org.apache.hadoop.yarn.api.records.ContainerId;
 import org.apache.hadoop.yarn.api.records.ContainerReport;
 import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.api.records.NodeAttributeInfo;
 import org.apache.hadoop.yarn.api.records.NodeId;
 import org.apache.hadoop.yarn.api.records.NodeLabel;
 import org.apache.hadoop.yarn.api.records.NodeReport;
 import org.apache.hadoop.yarn.api.records.NodeState;
+import org.apache.hadoop.yarn.api.records.NodeToAttributeValue;
 import org.apache.hadoop.yarn.api.records.Priority;
 import org.apache.hadoop.yarn.api.records.QueueUserACLInfo;
 import org.apache.hadoop.yarn.api.records.Resource;
@@ -541,14 +544,14 @@ public class ResourceMgrDelegate extends YarnClient {
   }
 
   @Override
-  public Set<NodeAttribute> getClusterAttributes()
+  public Set<NodeAttributeInfo> getClusterAttributes()
       throws YarnException, IOException {
     return client.getClusterAttributes();
   }
 
   @Override
-  public Map<NodeAttribute, Set<String>> getAttributesToNodes(
-      Set<NodeAttribute> attributes) throws YarnException, IOException {
+  public Map<NodeAttributeKey, List<NodeToAttributeValue>> getAttributesToNodes(
+      Set<NodeAttributeKey> attributes) throws YarnException, IOException {
     return client.getAttributesToNodes(attributes);
   }
 

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesRequest.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesRequest.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesRequest.java
index d9531b0..94814e9 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesRequest.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesRequest.java
@@ -17,22 +17,24 @@
  */
 package org.apache.hadoop.yarn.api.protocolrecords;
 
-import static org.apache.hadoop.classification.InterfaceAudience.*;
-import static org.apache.hadoop.classification.InterfaceStability.*;
+import java.util.Set;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Evolving;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
 import org.apache.hadoop.yarn.api.ApplicationClientProtocol;
-import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
 import org.apache.hadoop.yarn.util.Records;
 
-import java.util.Set;
-
 /**
  * <p>
- * The request from clients to get attribtues to nodes mapping
- * in the cluster from the <code>ResourceManager</code>.
+ * The request from clients to get node to attribute value mapping for all or
+ * given set of Node AttributeKey's in the cluster from the
+ * <code>ResourceManager</code>.
  * </p>
  *
  * @see ApplicationClientProtocol#getAttributesToNodes
- * (GetAttributesToNodesRequest)
+ *      (GetAttributesToNodesRequest)
  */
 @Public
 @Evolving
@@ -43,7 +45,7 @@ public abstract class GetAttributesToNodesRequest {
   }
 
   public static GetAttributesToNodesRequest newInstance(
-      Set<NodeAttribute> attributes) {
+      Set<NodeAttributeKey> attributes) {
     GetAttributesToNodesRequest request =
         Records.newRecord(GetAttributesToNodesRequest.class);
     request.setNodeAttributes(attributes);
@@ -51,20 +53,22 @@ public abstract class GetAttributesToNodesRequest {
   }
 
   /**
-   * Set node attributes for which the mapping is required.
+   * Set node attributeKeys for which the mapping of hostname to attribute value
+   * is required.
    *
-   * @param attributes Set<NodeAttribute> provided.
+   * @param attributes Set<NodeAttributeKey> provided.
    */
   @Public
   @Unstable
-  public abstract void setNodeAttributes(Set<NodeAttribute> attributes);
+  public abstract void setNodeAttributes(Set<NodeAttributeKey> attributes);
 
   /**
-   * Get node attributes for which mapping mapping is required.
+   * Get node attributeKeys for which mapping of hostname to attribute value is
+   * required.
    *
-   * @return Set<NodeAttribute>
+   * @return Set<NodeAttributeKey>
    */
   @Public
   @Unstable
-  public abstract Set<NodeAttribute> getNodeAttributes();
+  public abstract Set<NodeAttributeKey> getNodeAttributes();
 }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesResponse.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesResponse.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesResponse.java
index 4fdb1f7..9bd529f 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesResponse.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetAttributesToNodesResponse.java
@@ -17,29 +17,30 @@
  */
 package org.apache.hadoop.yarn.api.protocolrecords;
 
-import static org.apache.hadoop.classification.InterfaceAudience.Public;
-import static org.apache.hadoop.classification.InterfaceStability.Evolving;
+import java.util.List;
+import java.util.Map;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Evolving;
 import org.apache.hadoop.yarn.api.ApplicationClientProtocol;
-import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.api.records.NodeToAttributeValue;
 import org.apache.hadoop.yarn.util.Records;
 
-import java.util.Map;
-import java.util.Set;
-
 /**
  * <p>
  * The response sent by the <code>ResourceManager</code> to a client requesting
- * attributes to hostname mapping.
+ * node to attribute value mapping for all or given set of Node AttributeKey's.
  * </p>
  *
  * @see ApplicationClientProtocol#getAttributesToNodes
- * (GetAttributesToNodesRequest)
+ *      (GetAttributesToNodesRequest)
  */
 @Public
 @Evolving
 public abstract class GetAttributesToNodesResponse {
   public static GetAttributesToNodesResponse newInstance(
-      Map<NodeAttribute, Set<String>> map) {
+      Map<NodeAttributeKey, List<NodeToAttributeValue>> map) {
     GetAttributesToNodesResponse response =
         Records.newRecord(GetAttributesToNodesResponse.class);
     response.setAttributeToNodes(map);
@@ -48,15 +49,17 @@ public abstract class GetAttributesToNodesResponse {
 
   @Public
   @Evolving
-  public abstract void setAttributeToNodes(Map<NodeAttribute, Set<String>> map);
+  public abstract void setAttributeToNodes(
+      Map<NodeAttributeKey, List<NodeToAttributeValue>> map);
 
-  /*
-   * Get attributes to node hostname mapping.
+  /**
+   * Get mapping of NodeAttributeKey to its associated mapping of list of
+   * NodeToAttributeValuenode to attribute value.
    *
-   * @return Map<NodeAttribute, Set<String>> node attributes to hostname
-   * mapping.
+   * @return Map<NodeAttributeKey, List<NodeToAttributeValue>> node attributes
+   *         to list of NodeToAttributeValuenode.
    */
   @Public
   @Evolving
-  public abstract Map<NodeAttribute, Set<String>> getAttributesToNodes();
+  public abstract Map<NodeAttributeKey, List<NodeToAttributeValue>> getAttributesToNodes();
 }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetClusterNodeAttributesResponse.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetClusterNodeAttributesResponse.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetClusterNodeAttributesResponse.java
index cc3cae4..b0ccd90 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetClusterNodeAttributesResponse.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/GetClusterNodeAttributesResponse.java
@@ -17,14 +17,15 @@
  */
 package org.apache.hadoop.yarn.api.protocolrecords;
 
-import static org.apache.hadoop.classification.InterfaceAudience.*;
-import static org.apache.hadoop.classification.InterfaceStability.*;
+import java.util.Set;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Evolving;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
 import org.apache.hadoop.yarn.api.ApplicationClientProtocol;
-import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeInfo;
 import org.apache.hadoop.yarn.util.Records;
 
-import java.util.Set;
-
 /**
  * <p>
  * The response sent by the <code>ResourceManager</code> to a client requesting
@@ -45,7 +46,7 @@ public abstract class GetClusterNodeAttributesResponse {
    * @return GetClusterNodeAttributesResponse.
    */
   public static GetClusterNodeAttributesResponse newInstance(
-      Set<NodeAttribute> attributes) {
+      Set<NodeAttributeInfo> attributes) {
     GetClusterNodeAttributesResponse response =
         Records.newRecord(GetClusterNodeAttributesResponse.class);
     response.setNodeAttributes(attributes);
@@ -55,18 +56,18 @@ public abstract class GetClusterNodeAttributesResponse {
   /**
    * Set node attributes to the response.
    *
-   * @param attributes Node attributes
+   * @param attributes Map of Node attributeKey to Type.
    */
   @Public
   @Unstable
-  public abstract void setNodeAttributes(Set<NodeAttribute> attributes);
+  public abstract void setNodeAttributes(Set<NodeAttributeInfo> attributes);
 
   /**
-   * Get node attributes of the response.
+   * Get node attributes from the response.
    *
-   * @return Node attributes
+   * @return Node attributes.
    */
   @Public
   @Unstable
-  public abstract Set<NodeAttribute> getNodeAttributes();
+  public abstract Set<NodeAttributeInfo> getNodeAttributes();
 }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttribute.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttribute.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttribute.java
index 25ac9ab..7064939 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttribute.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttribute.java
@@ -58,8 +58,9 @@ public abstract class NodeAttribute {
       String attributeName, NodeAttributeType attributeType,
       String attributeValue) {
     NodeAttribute nodeAttribute = Records.newRecord(NodeAttribute.class);
-    nodeAttribute.setAttributePrefix(attributePrefix);
-    nodeAttribute.setAttributeName(attributeName);
+    NodeAttributeKey nodeAttributeKey =
+        NodeAttributeKey.newInstance(attributePrefix, attributeName);
+    nodeAttribute.setAttributeKey(nodeAttributeKey);
     nodeAttribute.setAttributeType(attributeType);
     nodeAttribute.setAttributeValue(attributeValue);
     return nodeAttribute;
@@ -67,19 +68,11 @@ public abstract class NodeAttribute {
 
   @Public
   @Unstable
-  public abstract String getAttributePrefix();
+  public abstract NodeAttributeKey getAttributeKey();
 
   @Public
   @Unstable
-  public abstract void setAttributePrefix(String attributePrefix);
-
-  @Public
-  @Unstable
-  public abstract String getAttributeName();
-
-  @Public
-  @Unstable
-  public abstract void setAttributeName(String attributeName);
+  public abstract void setAttributeKey(NodeAttributeKey attributeKey);
 
   @Public
   @Unstable

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeInfo.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeInfo.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeInfo.java
new file mode 100644
index 0000000..d294333
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeInfo.java
@@ -0,0 +1,62 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * Node Attribute Info describes a NodeAttribute.
+ * </p>
+ */
+@Public
+@Unstable
+public abstract class NodeAttributeInfo {
+
+  public static NodeAttributeInfo newInstance(NodeAttribute nodeAttribute) {
+    return newInstance(nodeAttribute.getAttributeKey(),
+        nodeAttribute.getAttributeType());
+  }
+
+  public static NodeAttributeInfo newInstance(NodeAttributeKey nodeAttributeKey,
+      NodeAttributeType attributeType) {
+    NodeAttributeInfo nodeAttribute =
+        Records.newRecord(NodeAttributeInfo.class);
+    nodeAttribute.setAttributeKey(nodeAttributeKey);
+    nodeAttribute.setAttributeType(attributeType);
+    return nodeAttribute;
+  }
+
+  @Public
+  @Unstable
+  public abstract NodeAttributeKey getAttributeKey();
+
+  @Public
+  @Unstable
+  public abstract void setAttributeKey(NodeAttributeKey attributeKey);
+
+  @Public
+  @Unstable
+  public abstract NodeAttributeType getAttributeType();
+
+  @Public
+  @Unstable
+  public abstract void setAttributeType(NodeAttributeType attributeType);
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeKey.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeKey.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeKey.java
new file mode 100644
index 0000000..35ff26f
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeAttributeKey.java
@@ -0,0 +1,66 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.yarn.api.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * Node AttributeKey uniquely identifies a given Node Attribute. Node Attribute
+ * is identified based on attribute prefix and name.
+ * </p>
+ * <p>
+ * Node Attribute Prefix is used as namespace to segregate the attributes.
+ * </p>
+ */
+@Public
+@Unstable
+public abstract class NodeAttributeKey {
+
+  public static NodeAttributeKey newInstance(String attributeName) {
+    return newInstance(NodeAttribute.PREFIX_CENTRALIZED, attributeName);
+  }
+
+  public static NodeAttributeKey newInstance(String attributePrefix,
+      String attributeName) {
+    NodeAttributeKey nodeAttributeKey =
+        Records.newRecord(NodeAttributeKey.class);
+    nodeAttributeKey.setAttributePrefix(attributePrefix);
+    nodeAttributeKey.setAttributeName(attributeName);
+    return nodeAttributeKey;
+  }
+
+  @Public
+  @Unstable
+  public abstract String getAttributePrefix();
+
+  @Public
+  @Unstable
+  public abstract void setAttributePrefix(String attributePrefix);
+
+  @Public
+  @Unstable
+  public abstract String getAttributeName();
+
+  @Public
+  @Unstable
+  public abstract void setAttributeName(String attributeName);
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeToAttributeValue.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeToAttributeValue.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeToAttributeValue.java
new file mode 100644
index 0000000..0bcb8b6
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/NodeToAttributeValue.java
@@ -0,0 +1,57 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.yarn.api.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * Mapping of Attribute Value to a Node.
+ * </p>
+ */
+@Public
+@Unstable
+public abstract class NodeToAttributeValue {
+  public static NodeToAttributeValue newInstance(String hostname,
+      String attributeValue) {
+    NodeToAttributeValue nodeToAttributeValue =
+        Records.newRecord(NodeToAttributeValue.class);
+    nodeToAttributeValue.setAttributeValue(attributeValue);
+    nodeToAttributeValue.setHostname(hostname);
+    return nodeToAttributeValue;
+  }
+
+  @Public
+  @Unstable
+  public abstract String getAttributeValue();
+
+  @Public
+  @Unstable
+  public abstract void setAttributeValue(String attributeValue);
+
+  @Public
+  @Unstable
+  public abstract String getHostname();
+
+  @Public
+  @Unstable
+  public abstract void setHostname(String hostname);
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_protos.proto
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_protos.proto b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_protos.proto
index 5576ee6..aca9471 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_protos.proto
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_protos.proto
@@ -376,16 +376,31 @@ enum NodeAttributeTypeProto {
   STRING = 1;
 }
 
-message NodeAttributeProto {
-  optional string attributePrefix = 1;
+message NodeAttributeKeyProto {
+  optional string attributePrefix = 1 [default="rm.yarn.io"];
   required string attributeName = 2;
-  optional NodeAttributeTypeProto attributeType = 3 [default = STRING];
-  optional string attributeValue = 4 [default=""];
+}
+
+message NodeAttributeProto {
+  required NodeAttributeKeyProto attributeKey = 1;
+  optional NodeAttributeTypeProto attributeType = 2 [default = STRING];
+  optional string attributeValue = 3 [default=""];
+}
+
+
+message NodeAttributeInfoProto {
+  required NodeAttributeKeyProto attributeKey = 1;
+  required NodeAttributeTypeProto attributeType = 2;
+}
+
+message NodeToAttributeValueProto {
+  required string hostname = 1;
+  required string attributeValue = 2;
 }
 
 message AttributeToNodesProto {
-  required NodeAttributeProto nodeAttribute = 1;
-  repeated string hostnames = 2;
+  required NodeAttributeKeyProto nodeAttribute = 1;
+  repeated NodeToAttributeValueProto nodeValueMap = 2;
 }
 
 message NodeToAttributesProto {

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_service_protos.proto
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_service_protos.proto b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_service_protos.proto
index 439780b..248f775 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_service_protos.proto
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/proto/yarn_service_protos.proto
@@ -264,11 +264,11 @@ message GetClusterNodeAttributesRequestProto {
 }
 
 message GetClusterNodeAttributesResponseProto {
-  repeated NodeAttributeProto nodeAttributes = 1;
+  repeated NodeAttributeInfoProto nodeAttributes = 1;
 }
 
 message GetAttributesToNodesRequestProto {
-  repeated NodeAttributeProto nodeAttributes = 1;
+  repeated NodeAttributeKeyProto nodeAttributes = 1;
 }
 
 message GetAttributesToNodesResponseProto {

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/YarnClient.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/YarnClient.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/YarnClient.java
index 0099845..f51b2f9 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/YarnClient.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/YarnClient.java
@@ -52,10 +52,13 @@ import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
 import org.apache.hadoop.yarn.api.records.ContainerId;
 import org.apache.hadoop.yarn.api.records.ContainerReport;
 import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.api.records.NodeAttributeInfo;
 import org.apache.hadoop.yarn.api.records.NodeId;
 import org.apache.hadoop.yarn.api.records.NodeLabel;
 import org.apache.hadoop.yarn.api.records.NodeReport;
 import org.apache.hadoop.yarn.api.records.NodeState;
+import org.apache.hadoop.yarn.api.records.NodeToAttributeValue;
 import org.apache.hadoop.yarn.api.records.Priority;
 import org.apache.hadoop.yarn.api.records.QueueInfo;
 import org.apache.hadoop.yarn.api.records.QueueUserACLInfo;
@@ -914,26 +917,27 @@ public abstract class YarnClient extends AbstractService {
    */
   @Public
   @Unstable
-  public abstract Set<NodeAttribute> getClusterAttributes()
+  public abstract Set<NodeAttributeInfo> getClusterAttributes()
       throws YarnException, IOException;
 
   /**
    * <p>
-   * The interface used by client to get Attributes to nodes mapping
-   * for specified node attributes in existing cluster.
+   * The interface used by client to get mapping of AttributeKey to associated
+   * NodeToAttributeValue list for specified node attributeKeys in the cluster.
    * </p>
    *
-   * @param attributes Attributes for which Attributes to nodes mapping has to
-   *                   be retrieved.If empty or null is set then will return
-   *                   all attributes to node mapping in cluster.
-   * @return Attributes to nodes mappings for specific Attributes.
+   * @param attributes AttributeKeys for which associated NodeToAttributeValue
+   *          mapping value has to be retrieved. If empty or null is set then
+   *          will return mapping for all attributeKeys in the cluster
+   * @return mapping of AttributeKey to List of associated
+   *         NodeToAttributeValue's.
    * @throws YarnException
    * @throws IOException
    */
   @Public
   @Unstable
-  public abstract Map<NodeAttribute, Set<String>> getAttributesToNodes(
-      Set<NodeAttribute> attributes) throws YarnException, IOException;
+  public abstract Map<NodeAttributeKey, List<NodeToAttributeValue>> getAttributesToNodes(
+      Set<NodeAttributeKey> attributes) throws YarnException, IOException;
 
   /**
    * <p>

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/impl/YarnClientImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/impl/YarnClientImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/impl/YarnClientImpl.java
index a08d35d..acfc3ff 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/impl/YarnClientImpl.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-client/src/main/java/org/apache/hadoop/yarn/client/api/impl/YarnClientImpl.java
@@ -99,10 +99,13 @@ import org.apache.hadoop.yarn.api.records.ContainerId;
 import org.apache.hadoop.yarn.api.records.ContainerLaunchContext;
 import org.apache.hadoop.yarn.api.records.ContainerReport;
 import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.api.records.NodeAttributeInfo;
 import org.apache.hadoop.yarn.api.records.NodeId;
 import org.apache.hadoop.yarn.api.records.NodeLabel;
 import org.apache.hadoop.yarn.api.records.NodeReport;
 import org.apache.hadoop.yarn.api.records.NodeState;
+import org.apache.hadoop.yarn.api.records.NodeToAttributeValue;
 import org.apache.hadoop.yarn.api.records.Priority;
 import org.apache.hadoop.yarn.api.records.QueueInfo;
 import org.apache.hadoop.yarn.api.records.QueueUserACLInfo;
@@ -981,7 +984,7 @@ public class YarnClientImpl extends YarnClient {
   }
 
   @Override
-  public Set<NodeAttribute> getClusterAttributes()
+  public Set<NodeAttributeInfo> getClusterAttributes()
       throws YarnException, IOException {
     GetClusterNodeAttributesRequest request =
         GetClusterNodeAttributesRequest.newInstance();
@@ -989,8 +992,8 @@ public class YarnClientImpl extends YarnClient {
   }
 
   @Override
-  public Map<NodeAttribute, Set<String>> getAttributesToNodes(
-      Set<NodeAttribute> attributes) throws YarnException, IOException {
+  public Map<NodeAttributeKey, List<NodeToAttributeValue>> getAttributesToNodes(
+      Set<NodeAttributeKey> attributes) throws YarnException, IOException {
     GetAttributesToNodesRequest request =
         GetAttributesToNodesRequest.newInstance(attributes);
     return rmClient.getAttributesToNodes(request).getAttributesToNodes();

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesRequestPBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesRequestPBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesRequestPBImpl.java
index a84fb44..15a360c 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesRequestPBImpl.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesRequestPBImpl.java
@@ -17,21 +17,21 @@
  */
 package org.apache.hadoop.yarn.api.protocolrecords.impl.pb;
 
-import com.google.protobuf.TextFormat;
-import org.apache.hadoop.yarn.api.protocolrecords.GetAttributesToNodesRequest;
-import org.apache.hadoop.yarn.api.records.NodeAttribute;
-import org.apache.hadoop.yarn.api.records.impl.pb.NodeAttributePBImpl;
-import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeProto;
-import org.apache.hadoop.yarn.proto.YarnServiceProtos;
-import org.apache.hadoop.yarn.proto.YarnServiceProtos.GetAttributesToNodesRequestProto;
-
 import java.util.HashSet;
 import java.util.Iterator;
 import java.util.List;
 import java.util.Set;
 
-import static org.apache.hadoop.classification.InterfaceAudience.*;
-import static org.apache.hadoop.classification.InterfaceStability.*;
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.api.protocolrecords.GetAttributesToNodesRequest;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.api.records.impl.pb.NodeAttributeKeyPBImpl;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeKeyProto;
+import org.apache.hadoop.yarn.proto.YarnServiceProtos;
+import org.apache.hadoop.yarn.proto.YarnServiceProtos.GetAttributesToNodesRequestProto;
+
+import com.google.protobuf.TextFormat;
 
 /**
  * Attributes to nodes mapping request.
@@ -41,7 +41,7 @@ import static org.apache.hadoop.classification.InterfaceStability.*;
 public class GetAttributesToNodesRequestPBImpl
     extends GetAttributesToNodesRequest {
 
-  private Set<NodeAttribute> nodeAttributes = null;
+  private Set<NodeAttributeKey> nodeAttributes = null;
 
   private GetAttributesToNodesRequestProto proto =
       GetAttributesToNodesRequestProto.getDefaultInstance();
@@ -86,9 +86,9 @@ public class GetAttributesToNodesRequestPBImpl
     if (nodeAttributes == null) {
       return;
     }
-    Iterable<NodeAttributeProto> iterable =
-        () -> new Iterator<NodeAttributeProto>() {
-          private Iterator<NodeAttribute> iter = nodeAttributes.iterator();
+    Iterable<NodeAttributeKeyProto> iterable =
+        () -> new Iterator<NodeAttributeKeyProto>() {
+          private Iterator<NodeAttributeKey> iter = nodeAttributes.iterator();
 
           @Override
           public boolean hasNext() {
@@ -96,7 +96,7 @@ public class GetAttributesToNodesRequestPBImpl
           }
 
           @Override
-          public NodeAttributeProto next() {
+          public NodeAttributeKeyProto next() {
             return convertToProtoFormat(iter.next());
           }
 
@@ -110,12 +110,13 @@ public class GetAttributesToNodesRequestPBImpl
     builder.addAllNodeAttributes(iterable);
   }
 
-  private NodeAttributePBImpl convertFromProtoFormat(NodeAttributeProto p) {
-    return new NodeAttributePBImpl(p);
+  private NodeAttributeKeyPBImpl convertFromProtoFormat(
+      NodeAttributeKeyProto p) {
+    return new NodeAttributeKeyPBImpl(p);
   }
 
-  private NodeAttributeProto convertToProtoFormat(NodeAttribute t) {
-    return ((NodeAttributePBImpl) t).getProto();
+  private NodeAttributeKeyProto convertToProtoFormat(NodeAttributeKey t) {
+    return ((NodeAttributeKeyPBImpl) t).getProto();
   }
 
   private void maybeInitBuilder() {
@@ -131,7 +132,7 @@ public class GetAttributesToNodesRequestPBImpl
     }
     YarnServiceProtos.GetAttributesToNodesRequestProtoOrBuilder p =
         viaProto ? proto : builder;
-    List<NodeAttributeProto> nodeAttributesList = p.getNodeAttributesList();
+    List<NodeAttributeKeyProto> nodeAttributesList = p.getNodeAttributesList();
     this.nodeAttributes = new HashSet<>();
     nodeAttributesList
         .forEach((v) -> nodeAttributes.add(convertFromProtoFormat(v)));
@@ -159,7 +160,7 @@ public class GetAttributesToNodesRequestPBImpl
   }
 
   @Override
-  public void setNodeAttributes(Set<NodeAttribute> attributes) {
+  public void setNodeAttributes(Set<NodeAttributeKey> attributes) {
     maybeInitBuilder();
     if (nodeAttributes == null) {
       builder.clearNodeAttributes();
@@ -168,7 +169,7 @@ public class GetAttributesToNodesRequestPBImpl
   }
 
   @Override
-  public Set<NodeAttribute> getNodeAttributes() {
+  public Set<NodeAttributeKey> getNodeAttributes() {
     initNodeAttributes();
     return this.nodeAttributes;
   }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesResponsePBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesResponsePBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesResponsePBImpl.java
index 175c10e..11999bdc 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesResponsePBImpl.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetAttributesToNodesResponsePBImpl.java
@@ -17,23 +17,24 @@
  */
 package org.apache.hadoop.yarn.api.protocolrecords.impl.pb;
 
-import org.apache.hadoop.yarn.api.protocolrecords.GetAttributesToNodesResponse;
-import org.apache.hadoop.yarn.api.records.NodeAttribute;
-import org.apache.hadoop.yarn.api.records.impl.pb.NodeAttributePBImpl;
-import org.apache.hadoop.yarn.proto.YarnProtos.AttributeToNodesProto;
-import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeProto;
-import org.apache.hadoop.yarn.proto.YarnServiceProtos;
-import org.apache.hadoop.yarn.proto.YarnServiceProtos.GetAttributesToNodesResponseProto;
-
+import java.util.ArrayList;
 import java.util.HashMap;
-import java.util.HashSet;
 import java.util.Iterator;
 import java.util.List;
 import java.util.Map;
-import java.util.Set;
 
-import static org.apache.hadoop.classification.InterfaceAudience.*;
-import static org.apache.hadoop.classification.InterfaceStability.*;
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.api.protocolrecords.GetAttributesToNodesResponse;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.api.records.NodeToAttributeValue;
+import org.apache.hadoop.yarn.api.records.impl.pb.NodeAttributeKeyPBImpl;
+import org.apache.hadoop.yarn.api.records.impl.pb.NodeToAttributeValuePBImpl;
+import org.apache.hadoop.yarn.proto.YarnProtos.AttributeToNodesProto;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeKeyProto;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeToAttributeValueProto;
+import org.apache.hadoop.yarn.proto.YarnServiceProtos;
+import org.apache.hadoop.yarn.proto.YarnServiceProtos.GetAttributesToNodesResponseProto;
 
 /**
  * Attributes to nodes response.
@@ -48,7 +49,7 @@ public class GetAttributesToNodesResponsePBImpl
   private GetAttributesToNodesResponseProto.Builder builder = null;
   private boolean viaProto = false;
 
-  private Map<NodeAttribute, Set<String>> attributesToNodes;
+  private Map<NodeAttributeKey, List<NodeToAttributeValue>> attributesToNodes;
 
   public GetAttributesToNodesResponsePBImpl() {
     this.builder = GetAttributesToNodesResponseProto.newBuilder();
@@ -70,10 +71,15 @@ public class GetAttributesToNodesResponsePBImpl
     this.attributesToNodes = new HashMap<>();
 
     for (AttributeToNodesProto c : list) {
-      Set<String> setNodes = new HashSet<>(c.getHostnamesList());
-      if (!setNodes.isEmpty()) {
-        this.attributesToNodes
-            .put(convertFromProtoFormat(c.getNodeAttribute()), setNodes);
+      List<NodeToAttributeValueProto> nodeValueMapList =
+          c.getNodeValueMapList();
+      List<NodeToAttributeValue> nodeToAttributeValue = new ArrayList<>();
+      for (NodeToAttributeValueProto valueProto : nodeValueMapList) {
+        nodeToAttributeValue.add(convertFromProtoFormat(valueProto));
+      }
+      if (!nodeToAttributeValue.isEmpty()) {
+        this.attributesToNodes.put(convertFromProtoFormat(c.getNodeAttribute()),
+            nodeToAttributeValue);
       }
     }
   }
@@ -94,7 +100,7 @@ public class GetAttributesToNodesResponsePBImpl
     Iterable<AttributeToNodesProto> iterable =
         () -> new Iterator<AttributeToNodesProto>() {
 
-          private Iterator<Map.Entry<NodeAttribute, Set<String>>> iter =
+          private Iterator<Map.Entry<NodeAttributeKey, List<NodeToAttributeValue>>> iter =
               attributesToNodes.entrySet().iterator();
 
           @Override
@@ -104,14 +110,18 @@ public class GetAttributesToNodesResponsePBImpl
 
           @Override
           public AttributeToNodesProto next() {
-            Map.Entry<NodeAttribute, Set<String>> now = iter.next();
-            Set<String> hostNames = new HashSet<>();
-            for (String host : now.getValue()) {
-              hostNames.add(host);
+            Map.Entry<NodeAttributeKey, List<NodeToAttributeValue>> attrToNodes
+                      = iter.next();
+
+            AttributeToNodesProto.Builder attrToNodesBuilder =
+                AttributeToNodesProto.newBuilder().setNodeAttribute(
+                    convertToProtoFormat(attrToNodes.getKey()));
+            for (NodeToAttributeValue hostToAttrVal : attrToNodes.getValue()) {
+              attrToNodesBuilder
+                  .addNodeValueMap(convertToProtoFormat(hostToAttrVal));
             }
-            return AttributeToNodesProto.newBuilder()
-                .setNodeAttribute(convertToProtoFormat(now.getKey()))
-                .addAllHostnames(hostNames).build();
+
+            return attrToNodesBuilder.build();
           }
 
           @Override
@@ -122,12 +132,22 @@ public class GetAttributesToNodesResponsePBImpl
     builder.addAllAttributesToNodes(iterable);
   }
 
-  private NodeAttributePBImpl convertFromProtoFormat(NodeAttributeProto p) {
-    return new NodeAttributePBImpl(p);
+  private NodeAttributeKey convertFromProtoFormat(NodeAttributeKeyProto p) {
+    return new NodeAttributeKeyPBImpl(p);
+  }
+
+  private NodeAttributeKeyProto convertToProtoFormat(NodeAttributeKey t) {
+    return ((NodeAttributeKeyPBImpl) t).getProto();
+  }
+
+  private NodeToAttributeValue convertFromProtoFormat(
+      NodeToAttributeValueProto p) {
+    return new NodeToAttributeValuePBImpl(p);
   }
 
-  private NodeAttributeProto convertToProtoFormat(NodeAttribute t) {
-    return ((NodeAttributePBImpl) t).getProto();
+  private NodeToAttributeValueProto convertToProtoFormat(
+      NodeToAttributeValue t) {
+    return ((NodeToAttributeValuePBImpl) t).getProto();
   }
 
   private void mergeLocalToBuilder() {
@@ -170,14 +190,15 @@ public class GetAttributesToNodesResponsePBImpl
   }
 
   @Override
-  public void setAttributeToNodes(Map<NodeAttribute, Set<String>> map) {
+  public void setAttributeToNodes(
+      Map<NodeAttributeKey, List<NodeToAttributeValue>> map) {
     initAttributesToNodes();
     attributesToNodes.clear();
     attributesToNodes.putAll(map);
   }
 
   @Override
-  public Map<NodeAttribute, Set<String>> getAttributesToNodes() {
+  public Map<NodeAttributeKey, List<NodeToAttributeValue>> getAttributesToNodes() {
     initAttributesToNodes();
     return this.attributesToNodes;
   }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetClusterNodeAttributesResponsePBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetClusterNodeAttributesResponsePBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetClusterNodeAttributesResponsePBImpl.java
index 385155f..41cd808 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetClusterNodeAttributesResponsePBImpl.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/protocolrecords/impl/pb/GetClusterNodeAttributesResponsePBImpl.java
@@ -17,19 +17,19 @@
  */
 package org.apache.hadoop.yarn.api.protocolrecords.impl.pb;
 
-import static org.apache.hadoop.classification.InterfaceAudience.*;
-import static org.apache.hadoop.classification.InterfaceStability.*;
-import org.apache.hadoop.yarn.api.protocolrecords.GetClusterNodeAttributesResponse;
-import org.apache.hadoop.yarn.api.records.NodeAttribute;
-import org.apache.hadoop.yarn.api.records.impl.pb.NodeAttributePBImpl;
-import org.apache.hadoop.yarn.proto.YarnServiceProtos;
-import org.apache.hadoop.yarn.proto.YarnServiceProtos.GetClusterNodeAttributesResponseProto;
-import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeProto;
-
-import java.util.ArrayList;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Set;
+import java.util.stream.Collectors;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.api.protocolrecords.GetClusterNodeAttributesResponse;
+import org.apache.hadoop.yarn.api.records.NodeAttributeInfo;
+import org.apache.hadoop.yarn.api.records.impl.pb.NodeAttributeInfoPBImpl;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeInfoProto;
+import org.apache.hadoop.yarn.proto.YarnServiceProtos;
+import org.apache.hadoop.yarn.proto.YarnServiceProtos.GetClusterNodeAttributesResponseProto;
 
 /**
  * Cluster node attributes response.
@@ -42,7 +42,7 @@ public class GetClusterNodeAttributesResponsePBImpl
   private GetClusterNodeAttributesResponseProto proto =
       GetClusterNodeAttributesResponseProto.getDefaultInstance();
   private GetClusterNodeAttributesResponseProto.Builder builder = null;
-  private Set<NodeAttribute> updatedNodeAttributes;
+  private Set<NodeAttributeInfo> clusterNodeAttributes;
   private boolean viaProto = false;
 
   public GetClusterNodeAttributesResponsePBImpl() {
@@ -72,7 +72,7 @@ public class GetClusterNodeAttributesResponsePBImpl
   }
 
   private void mergeLocalToBuilder() {
-    if (this.updatedNodeAttributes != null) {
+    if (this.clusterNodeAttributes != null) {
       addNodeAttributesToProto();
     }
   }
@@ -80,11 +80,12 @@ public class GetClusterNodeAttributesResponsePBImpl
   private void addNodeAttributesToProto() {
     maybeInitBuilder();
     builder.clearNodeAttributes();
-    List<NodeAttributeProto> protoList = new ArrayList<>();
-    for (NodeAttribute r : this.updatedNodeAttributes) {
-      protoList.add(convertToProtoFormat(r));
+    if (clusterNodeAttributes == null || clusterNodeAttributes.isEmpty()) {
+      return;
     }
-    builder.addAllNodeAttributes(protoList);
+
+    builder.addAllNodeAttributes(clusterNodeAttributes.stream()
+        .map(s -> convertToProtoFormat(s)).collect(Collectors.toSet()));
   }
 
   @Override
@@ -112,41 +113,44 @@ public class GetClusterNodeAttributesResponsePBImpl
   }
 
   @Override
-  public synchronized void setNodeAttributes(Set<NodeAttribute> attributes) {
+  public synchronized void setNodeAttributes(
+      Set<NodeAttributeInfo> attributes) {
     maybeInitBuilder();
-    this.updatedNodeAttributes = new HashSet<>();
+    this.clusterNodeAttributes = new HashSet<>();
     if (attributes == null) {
       builder.clearNodeAttributes();
       return;
     }
-    this.updatedNodeAttributes.addAll(attributes);
+    this.clusterNodeAttributes.addAll(attributes);
   }
 
   @Override
-  public synchronized Set<NodeAttribute> getNodeAttributes() {
-    if (this.updatedNodeAttributes != null) {
-      return this.updatedNodeAttributes;
+  public synchronized Set<NodeAttributeInfo> getNodeAttributes() {
+    if (this.clusterNodeAttributes != null) {
+      return this.clusterNodeAttributes;
     }
     initLocalNodeAttributes();
-    return this.updatedNodeAttributes;
+    return this.clusterNodeAttributes;
   }
 
   private void initLocalNodeAttributes() {
     YarnServiceProtos.GetClusterNodeAttributesResponseProtoOrBuilder p =
         viaProto ? proto : builder;
-    List<NodeAttributeProto> attributesProtoList = p.getNodeAttributesList();
-    this.updatedNodeAttributes = new HashSet<>();
-    for (NodeAttributeProto r : attributesProtoList) {
-      this.updatedNodeAttributes.add(convertFromProtoFormat(r));
-    }
+    List<NodeAttributeInfoProto> attributesProtoList =
+        p.getNodeAttributesList();
+    this.clusterNodeAttributes = new HashSet<>();
+    clusterNodeAttributes.addAll(attributesProtoList.stream()
+        .map(attr -> convertFromProtoFormat(attr)).collect(Collectors.toSet()));
   }
 
-  private NodeAttribute convertFromProtoFormat(NodeAttributeProto p) {
-    return new NodeAttributePBImpl(p);
+  private NodeAttributeInfoProto convertToProtoFormat(
+      NodeAttributeInfo attributeInfo) {
+    return ((NodeAttributeInfoPBImpl)attributeInfo).getProto();
   }
 
-  private NodeAttributeProto convertToProtoFormat(NodeAttribute t) {
-    return ((NodeAttributePBImpl) t).getProto();
+  private NodeAttributeInfo convertFromProtoFormat(
+      NodeAttributeInfoProto nodeAttributeInfoProto) {
+    return new NodeAttributeInfoPBImpl(nodeAttributeInfoProto);
   }
 
   @Override

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeInfoPBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeInfoPBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeInfoPBImpl.java
new file mode 100644
index 0000000..bff6335
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeInfoPBImpl.java
@@ -0,0 +1,143 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.impl.pb;
+
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.api.records.NodeAttributeInfo;
+import org.apache.hadoop.yarn.api.records.NodeAttributeType;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeKeyProto;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeInfoProto;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeInfoProtoOrBuilder;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeTypeProto;
+
+/**
+ * Implementation for NodeAttributeInfo.
+ *
+ */
+public class NodeAttributeInfoPBImpl extends NodeAttributeInfo {
+  private NodeAttributeInfoProto proto =
+      NodeAttributeInfoProto.getDefaultInstance();
+  private NodeAttributeInfoProto.Builder builder = null;
+  private boolean viaProto = false;
+
+  public NodeAttributeInfoPBImpl() {
+    builder = NodeAttributeInfoProto.newBuilder();
+  }
+
+  public NodeAttributeInfoPBImpl(NodeAttributeInfoProto proto) {
+    this.proto = proto;
+    viaProto = true;
+  }
+
+  public NodeAttributeInfoProto getProto() {
+    proto = viaProto ? proto : builder.build();
+    viaProto = true;
+    return proto;
+  }
+
+  private void maybeInitBuilder() {
+    if (viaProto || builder == null) {
+      builder = NodeAttributeInfoProto.newBuilder(proto);
+    }
+    viaProto = false;
+  }
+
+  @Override
+  public NodeAttributeKey getAttributeKey() {
+    NodeAttributeInfoProtoOrBuilder p = viaProto ? proto : builder;
+    if (!p.hasAttributeKey()) {
+      return null;
+    }
+    return convertFromProtoFormat(p.getAttributeKey());
+  }
+
+  @Override
+  public void setAttributeKey(NodeAttributeKey attributeKey) {
+    maybeInitBuilder();
+    if (attributeKey == null) {
+      builder.clearAttributeKey();
+      return;
+    }
+    builder.setAttributeKey(convertToProtoFormat(attributeKey));
+  }
+
+  @Override
+  public NodeAttributeType getAttributeType() {
+    NodeAttributeInfoProtoOrBuilder p = viaProto ? proto : builder;
+    if (!p.hasAttributeType()) {
+      return null;
+    }
+    return convertFromProtoFormat(p.getAttributeType());
+  }
+
+  @Override
+  public void setAttributeType(NodeAttributeType attributeType) {
+    maybeInitBuilder();
+    if (attributeType == null) {
+      builder.clearAttributeType();
+      return;
+    }
+    builder.setAttributeType(convertToProtoFormat(attributeType));
+  }
+
+  private NodeAttributeTypeProto convertToProtoFormat(
+      NodeAttributeType attributeType) {
+    return NodeAttributeTypeProto.valueOf(attributeType.name());
+  }
+
+  private NodeAttributeType convertFromProtoFormat(
+      NodeAttributeTypeProto containerState) {
+    return NodeAttributeType.valueOf(containerState.name());
+  }
+
+  private NodeAttributeKeyPBImpl convertFromProtoFormat(
+      NodeAttributeKeyProto attributeKeyProto) {
+    return new NodeAttributeKeyPBImpl(attributeKeyProto);
+  }
+
+  private NodeAttributeKeyProto convertToProtoFormat(
+      NodeAttributeKey attributeKey) {
+    return ((NodeAttributeKeyPBImpl) attributeKey).getProto();
+  }
+
+  @Override
+  public int hashCode() {
+    return getAttributeKey().hashCode();
+  }
+
+  @Override
+  public boolean equals(Object obj) {
+    if (this == obj) {
+      return true;
+    }
+    if (obj == null) {
+      return false;
+    }
+    if (obj instanceof NodeAttributeInfo) {
+      NodeAttributeInfo other = (NodeAttributeInfo) obj;
+      getAttributeKey().equals(other.getAttributeKey());
+      return true;
+    }
+    return false;
+  }
+
+  @Override
+  public String toString() {
+    return getAttributeKey().toString() + ":Type-" + getAttributeType();
+  }
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeKeyPBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeKeyPBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeKeyPBImpl.java
new file mode 100644
index 0000000..921e767
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributeKeyPBImpl.java
@@ -0,0 +1,140 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.yarn.api.records.impl.pb;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeKeyProto;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeKeyProtoOrBuilder;
+
+/**
+ * Implementation for NodeAttributeKey.
+ *
+ */
+@Private
+@Unstable
+public class NodeAttributeKeyPBImpl extends NodeAttributeKey {
+  private NodeAttributeKeyProto proto =
+      NodeAttributeKeyProto.getDefaultInstance();
+  private NodeAttributeKeyProto.Builder builder = null;
+  private boolean viaProto = false;
+
+  public NodeAttributeKeyPBImpl() {
+    builder = NodeAttributeKeyProto.newBuilder();
+  }
+
+  public NodeAttributeKeyPBImpl(NodeAttributeKeyProto proto) {
+    this.proto = proto;
+    viaProto = true;
+  }
+
+  public NodeAttributeKeyProto getProto() {
+    proto = viaProto ? proto : builder.build();
+    viaProto = true;
+    return proto;
+  }
+
+  private void maybeInitBuilder() {
+    if (viaProto || builder == null) {
+      builder = NodeAttributeKeyProto.newBuilder(proto);
+    }
+    viaProto = false;
+  }
+
+  @Override
+  public String getAttributePrefix() {
+    NodeAttributeKeyProtoOrBuilder p = viaProto ? proto : builder;
+    return p.getAttributePrefix();
+  }
+
+  @Override
+  public void setAttributePrefix(String attributePrefix) {
+    maybeInitBuilder();
+    if (attributePrefix == null) {
+      builder.clearAttributePrefix();
+      return;
+    }
+    builder.setAttributePrefix(attributePrefix);
+  }
+
+  @Override
+  public String getAttributeName() {
+    NodeAttributeKeyProtoOrBuilder p = viaProto ? proto : builder;
+    if (!p.hasAttributeName()) {
+      return null;
+    }
+    return p.getAttributeName();
+  }
+
+  @Override
+  public void setAttributeName(String attributeName) {
+    maybeInitBuilder();
+    if (attributeName == null) {
+      builder.clearAttributeName();
+      return;
+    }
+    builder.setAttributeName(attributeName);
+  }
+
+  @Override
+  public int hashCode() {
+    final int prime = 31;
+    int result = 1;
+    result = prime * result + ((getAttributePrefix() == null) ? 0
+        : getAttributePrefix().hashCode());
+    result = prime * result
+        + ((getAttributeName() == null) ? 0 : getAttributeName().hashCode());
+    return result;
+  }
+
+  @Override
+  public boolean equals(Object obj) {
+    if (this == obj) {
+      return true;
+    }
+    if (obj == null) {
+      return false;
+    }
+    if (obj instanceof NodeAttributeKey) {
+      NodeAttributeKey other = (NodeAttributeKey) obj;
+      if (!compare(getAttributePrefix(), other.getAttributePrefix())) {
+        return false;
+      }
+      if (!compare(getAttributeName(), other.getAttributeName())) {
+        return false;
+      }
+      return true;
+    }
+    return false;
+  }
+
+  private static boolean compare(Object left, Object right) {
+    if (left == null) {
+      return right == null;
+    } else {
+      return left.equals(right);
+    }
+  }
+
+  @Override
+  public String toString() {
+    return "Prefix-" + getAttributePrefix() + " :Name-" + getAttributeName();
+  }
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributePBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributePBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributePBImpl.java
index 12a0ecc..0002567 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributePBImpl.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeAttributePBImpl.java
@@ -18,12 +18,21 @@
 
 package org.apache.hadoop.yarn.api.records.impl.pb;
 
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
 import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
 import org.apache.hadoop.yarn.api.records.NodeAttributeType;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeKeyProto;
 import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeProto;
 import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeProtoOrBuilder;
 import org.apache.hadoop.yarn.proto.YarnProtos.NodeAttributeTypeProto;
 
+/**
+ * Implementation for NodeAttribute.
+ */
+@Private
+@Unstable
 public class NodeAttributePBImpl extends NodeAttribute {
   private NodeAttributeProto proto = NodeAttributeProto.getDefaultInstance();
   private NodeAttributeProto.Builder builder = null;
@@ -52,22 +61,22 @@ public class NodeAttributePBImpl extends NodeAttribute {
   }
 
   @Override
-  public String getAttributeName() {
+  public NodeAttributeKey getAttributeKey() {
     NodeAttributeProtoOrBuilder p = viaProto ? proto : builder;
-    if (!p.hasAttributeName()) {
+    if (!p.hasAttributeKey()) {
       return null;
     }
-    return p.getAttributeName();
+    return convertFromProtoFormat(p.getAttributeKey());
   }
 
   @Override
-  public void setAttributeName(String attributeName) {
+  public void setAttributeKey(NodeAttributeKey attributeKey) {
     maybeInitBuilder();
-    if(attributeName == null) {
-      builder.clearAttributeName();
+    if(attributeKey == null) {
+      builder.clearAttributeKey();
       return;
     }
-    builder.setAttributeName(attributeName);
+    builder.setAttributeKey(convertToProtoFormat(attributeKey));
   }
 
   @Override
@@ -118,15 +127,19 @@ public class NodeAttributePBImpl extends NodeAttribute {
     return NodeAttributeType.valueOf(containerState.name());
   }
 
+  private NodeAttributeKeyPBImpl convertFromProtoFormat(
+      NodeAttributeKeyProto attributeKeyProto) {
+    return new NodeAttributeKeyPBImpl(attributeKeyProto);
+  }
+
+  private NodeAttributeKeyProto convertToProtoFormat(
+      NodeAttributeKey attributeKey) {
+    return ((NodeAttributeKeyPBImpl)attributeKey).getProto();
+  }
+
   @Override
   public int hashCode() {
-    final int prime = 31;
-    int result = 1;
-    result = prime * result + ((getAttributePrefix() == null) ? 0
-        : getAttributePrefix().hashCode());
-    result = prime * result
-        + ((getAttributeName() == null) ? 0 : getAttributeName().hashCode());
-    return result;
+    return getAttributeKey().hashCode();
   }
 
   @Override
@@ -139,47 +152,15 @@ public class NodeAttributePBImpl extends NodeAttribute {
     }
     if (obj instanceof NodeAttribute) {
       NodeAttribute other = (NodeAttribute) obj;
-      if (!compare(getAttributePrefix(), other.getAttributePrefix())) {
-        return false;
-      }
-      if (!compare(getAttributeName(), other.getAttributeName())) {
-        return false;
-      }
+      getAttributeKey().equals(other.getAttributeKey());
       return true;
     }
     return false;
   }
 
-  private static boolean compare(Object left, Object right) {
-    if (left == null) {
-      return right == null;
-    } else {
-      return left.equals(right);
-    }
-  }
-
-  @Override
-  public String getAttributePrefix() {
-    NodeAttributeProtoOrBuilder p = viaProto ? proto : builder;
-    if (!p.hasAttributePrefix()) {
-      return null;
-    }
-    return p.getAttributePrefix();
-  }
-
-  @Override
-  public void setAttributePrefix(String attributePrefix) {
-    maybeInitBuilder();
-    if(attributePrefix == null) {
-      builder.clearAttributePrefix();
-      return;
-    }
-    builder.setAttributePrefix(attributePrefix);
-  }
-
   @Override
   public String toString() {
-    return "Prefix-" + getAttributePrefix() + " :Name-" + getAttributeName()
-        + ":Value-" + getAttributeValue() + ":Type-" + getAttributeType();
+    return getAttributeKey().toString() + ":Value-" + getAttributeValue()
+        + ":Type-" + getAttributeType();
   }
 }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeToAttributeValuePBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeToAttributeValuePBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeToAttributeValuePBImpl.java
new file mode 100644
index 0000000..7a79876
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/api/records/impl/pb/NodeToAttributeValuePBImpl.java
@@ -0,0 +1,137 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.yarn.api.records.impl.pb;
+
+import org.apache.hadoop.yarn.api.records.NodeToAttributeValue;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeToAttributeValueProto;
+import org.apache.hadoop.yarn.proto.YarnProtos.NodeToAttributeValueProtoOrBuilder;
+
+/**
+ * PB Implementation for NodeToAttributeValue.
+ *
+ */
+public class NodeToAttributeValuePBImpl extends NodeToAttributeValue {
+  private NodeToAttributeValueProto proto =
+      NodeToAttributeValueProto.getDefaultInstance();
+  private NodeToAttributeValueProto.Builder builder = null;
+  private boolean viaProto = false;
+
+  public NodeToAttributeValuePBImpl() {
+    builder = NodeToAttributeValueProto.newBuilder();
+  }
+
+  public NodeToAttributeValuePBImpl(NodeToAttributeValueProto proto) {
+    this.proto = proto;
+    viaProto = true;
+  }
+
+  public NodeToAttributeValueProto getProto() {
+    proto = viaProto ? proto : builder.build();
+    viaProto = true;
+    return proto;
+  }
+
+  private void maybeInitBuilder() {
+    if (viaProto || builder == null) {
+      builder = NodeToAttributeValueProto.newBuilder(proto);
+    }
+    viaProto = false;
+  }
+
+  @Override
+  public String getAttributeValue() {
+    NodeToAttributeValueProtoOrBuilder p = viaProto ? proto : builder;
+    return p.getAttributeValue();
+  }
+
+  @Override
+  public void setAttributeValue(String attributeValue) {
+    maybeInitBuilder();
+    if (attributeValue == null) {
+      builder.clearAttributeValue();
+      return;
+    }
+    builder.setAttributeValue(attributeValue);
+  }
+
+  @Override
+  public String getHostname() {
+    NodeToAttributeValueProtoOrBuilder p = viaProto ? proto : builder;
+    if (!p.hasHostname()) {
+      return null;
+    }
+    return p.getHostname();
+  }
+
+  @Override
+  public void setHostname(String hostname) {
+    maybeInitBuilder();
+    if (hostname == null) {
+      builder.clearHostname();
+      return;
+    }
+    builder.setHostname(hostname);
+  }
+
+  @Override
+  public int hashCode() {
+    final int prime = 31;
+    int result = 1;
+    result = prime * result
+        + ((getAttributeValue() == null) ? 0 : getAttributeValue().hashCode());
+    result = prime * result
+        + ((getHostname() == null) ? 0 : getHostname().hashCode());
+    return result;
+  }
+
+  @Override
+  public boolean equals(Object obj) {
+    if (this == obj) {
+      return true;
+    }
+    if (obj == null) {
+      return false;
+    }
+    if (obj instanceof NodeToAttributeValue) {
+      NodeToAttributeValue other = (NodeToAttributeValue) obj;
+      if (!compare(getAttributeValue(), other.getAttributeValue())) {
+        return false;
+      }
+      if (!compare(getHostname(), other.getHostname())) {
+        return false;
+      }
+      return true;
+    }
+    return false;
+  }
+
+  private static boolean compare(Object left, Object right) {
+    if (left == null) {
+      return right == null;
+    } else {
+      return left.equals(right);
+    }
+  }
+
+  @Override
+  public String toString() {
+    return "Name-" + getHostname() + " : Attribute Value-"
+        + getAttributeValue();
+  }
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeAttributesManager.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeAttributesManager.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeAttributesManager.java
index 79c53e2..68c6ec6 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeAttributesManager.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeAttributesManager.java
@@ -25,6 +25,7 @@ import java.util.Set;
 
 import org.apache.hadoop.service.AbstractService;
 import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
 import org.apache.hadoop.yarn.server.api.protocolrecords.NodeToAttributes;
 
 /**
@@ -81,20 +82,21 @@ public abstract class NodeAttributesManager extends AbstractService {
    *
    * @param prefix set of prefix string's for which the attributes needs to
    *          returned
-   * @return set of node Attributes
+   * @return Set of node Attributes
    */
   public abstract Set<NodeAttribute> getClusterNodeAttributes(
       Set<String> prefix);
 
   /**
-   * Given a attribute set, return what all Nodes have attribute mapped to it.
-   * If the attributes set is null or empty, all attributes mapping are
-   * returned.
+   * Return a map of Nodes to attribute value for the given NodeAttributeKeys.
+   * If the attributeKeys set is null or empty, then mapping for all attributes
+   * are returned.
    *
-   * @return a Map of attributes to set of hostnames.
+   * @return a Map of attributeKeys to a map of hostnames to its attribute
+   *         values.
    */
-  public abstract Map<NodeAttribute, Set<String>> getAttributesToNodes(
-      Set<NodeAttribute> attributes);
+  public abstract Map<NodeAttributeKey, Map<String, AttributeValue>> getAttributesToNodes(
+      Set<NodeAttributeKey> attributes);
 
   /**
    * NodeAttribute to AttributeValue Map.

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeLabelUtil.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeLabelUtil.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeLabelUtil.java
index 93a27a9..0dd0755 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeLabelUtil.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/NodeLabelUtil.java
@@ -19,6 +19,7 @@ package org.apache.hadoop.yarn.nodelabels;
 
 import com.google.common.base.Strings;
 import org.apache.hadoop.yarn.api.records.NodeAttribute;
+import org.apache.hadoop.yarn.api.records.NodeAttributeKey;
 
 import java.io.IOException;
 import java.util.Set;
@@ -115,14 +116,18 @@ public final class NodeLabelUtil {
       throws IOException {
     if (attributeSet != null && !attributeSet.isEmpty()) {
       for (NodeAttribute nodeAttribute : attributeSet) {
-        String prefix = nodeAttribute.getAttributePrefix();
+        NodeAttributeKey attributeKey = nodeAttribute.getAttributeKey();
+        if (attributeKey == null) {
+          throw new IOException("AttributeKey  must be set");
+        }
+        String prefix = attributeKey.getAttributePrefix();
         if (Strings.isNullOrEmpty(prefix)) {
           throw new IOException("Attribute prefix must be set");
         }
         // Verify attribute prefix format.
         checkAndThrowAttributePrefix(prefix);
         // Verify attribute name format.
-        checkAndThrowLabelName(nodeAttribute.getAttributeName());
+        checkAndThrowLabelName(attributeKey.getAttributeName());
       }
     }
   }
@@ -140,8 +145,9 @@ public final class NodeLabelUtil {
     if (Strings.isNullOrEmpty(prefix)) {
       return attributeSet;
     }
-    return attributeSet.stream().filter(
-        nodeAttribute -> prefix.equals(nodeAttribute.getAttributePrefix()))
+    return attributeSet.stream()
+        .filter(nodeAttribute -> prefix
+            .equals(nodeAttribute.getAttributeKey().getAttributePrefix()))
         .collect(Collectors.toSet());
   }
 }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2b3d54bc/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/RMNodeAttribute.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/RMNodeAttribute.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/RMNodeAttribute.java
index 3b2bd16..ffe36c8 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/RMNodeAttribute.java
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/nodelabels/RMNodeAttribute.java
@@ -18,8 +18,8 @@
 
 package org.apache.hadoop.yarn.nodelabels;
 
-import java.util.HashSet;
-import java.util.Set;
+import java.util.HashMap;
+import java.util.Map;
 
 import org.apache.hadoop.yarn.api.records.NodeAttribute;
 import org.apache.hadoop.yarn.api.records.NodeAttributeType;
@@ -32,11 +32,11 @@ public class RMNodeAttribute extends AbstractLabel {
 
   private NodeAttribute attribute;
   // TODO need to revisit whether we need to make this concurrent implementation
-  private Set<String> nodes = new HashSet<>();
+  private Map<String, AttributeValue> nodes = new HashMap<>();
 
   public RMNodeAttribute(NodeAttribute attribute) {
-    this(attribute.getAttributeName(), Resource.newInstance(0, 0), 0,
-        attribute);
+    this(attribute.getAttributeKey().getAttributeName(),
+        Resource.newInstance(0, 0), 0, attribute);
   }
 
   public RMNodeAttribute(String labelName, Resource res, int activeNMs,
@@ -57,16 +57,16 @@ public class RMNodeAttribute extends AbstractLabel {
     return attribute.getAttributeType();
   }
 
-  public void addNode(String node) {
-    nodes.add(node);
+  public void addNode(String node, AttributeValue attributeValue) {
+    nodes.put(node, attributeValue);
   }
 
   public void removeNode(String node) {
     nodes.remove(node);
   }
 
-  public Set<String> getAssociatedNodeIds() {
-    return new HashSet<String>(nodes);
+  public Map<String, AttributeValue> getAssociatedNodeIds() {
+    return new HashMap<String,  AttributeValue>(nodes);
   }
 
   @Override


---------------------------------------------------------------------
To unsubscribe, e-mail: common-commits-unsubscribe@hadoop.apache.org
For additional commands, e-mail: common-commits-help@hadoop.apache.org