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 2017/06/07 21:49:00 UTC

[09/50] [abbrv] hadoop git commit: YARN-3662. Federation Membership State Store internal APIs.

YARN-3662. Federation Membership State Store internal APIs.


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

Branch: refs/heads/YARN-2915
Commit: 8b02ec0b973dcfb74d9330f987685f79e1bd0beb
Parents: cb2b927
Author: Subru Krishnan <su...@apache.org>
Authored: Fri Jul 29 16:53:40 2016 -0700
Committer: Subru Krishnan <su...@apache.org>
Committed: Wed Jun 7 14:46:40 2017 -0700

----------------------------------------------------------------------
 .../hadoop-yarn-server-common/pom.xml           |   8 +
 .../store/FederationMembershipStateStore.java   | 126 +++++++++
 .../server/federation/store/package-info.java   |  17 ++
 .../store/records/GetSubClusterInfoRequest.java |  62 +++++
 .../records/GetSubClusterInfoResponse.java      |  62 +++++
 .../records/GetSubClustersInfoRequest.java      |  66 +++++
 .../records/GetSubClustersInfoResponse.java     |  66 +++++
 .../records/SubClusterDeregisterRequest.java    |  89 +++++++
 .../records/SubClusterDeregisterResponse.java   |  42 +++
 .../records/SubClusterHeartbeatRequest.java     | 149 +++++++++++
 .../records/SubClusterHeartbeatResponse.java    |  45 ++++
 .../federation/store/records/SubClusterId.java  | 100 +++++++
 .../store/records/SubClusterInfo.java           | 263 ++++++++++++++++++
 .../records/SubClusterRegisterRequest.java      |  74 +++++
 .../records/SubClusterRegisterResponse.java     |  44 +++
 .../store/records/SubClusterState.java          |  60 +++++
 .../impl/pb/GetSubClusterInfoRequestPBImpl.java | 125 +++++++++
 .../pb/GetSubClusterInfoResponsePBImpl.java     | 134 ++++++++++
 .../pb/GetSubClustersInfoRequestPBImpl.java     | 108 ++++++++
 .../pb/GetSubClustersInfoResponsePBImpl.java    | 184 +++++++++++++
 .../pb/SubClusterDeregisterRequestPBImpl.java   | 156 +++++++++++
 .../pb/SubClusterDeregisterResponsePBImpl.java  |  77 ++++++
 .../pb/SubClusterHeartbeatRequestPBImpl.java    | 192 +++++++++++++
 .../pb/SubClusterHeartbeatResponsePBImpl.java   |  77 ++++++
 .../records/impl/pb/SubClusterIdPBImpl.java     |  75 ++++++
 .../records/impl/pb/SubClusterInfoPBImpl.java   | 267 +++++++++++++++++++
 .../pb/SubClusterRegisterRequestPBImpl.java     | 134 ++++++++++
 .../pb/SubClusterRegisterResponsePBImpl.java    |  77 ++++++
 .../store/records/impl/pb/package-info.java     |  17 ++
 .../federation/store/records/package-info.java  |  17 ++
 .../proto/yarn_server_federation_protos.proto   |  93 +++++++
 .../records/TestFederationProtocolRecords.java  | 133 +++++++++
 32 files changed, 3139 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/pom.xml
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/pom.xml b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/pom.xml
index 465f71a..c52e5e6 100644
--- a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/pom.xml
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/pom.xml
@@ -58,6 +58,13 @@
       <groupId>org.apache.hadoop</groupId>
       <artifactId>hadoop-yarn-common</artifactId>
     </dependency>
+    <!-- 'mvn dependency:analyze' fails to detect use of this dependency -->
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-yarn-common</artifactId>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
 
     <dependency>
       <groupId>com.google.guava</groupId>
@@ -146,6 +153,7 @@
                   <include>yarn_server_common_protos.proto</include>
                   <include>yarn_server_common_service_protos.proto</include>
                   <include>yarn_server_common_service_protos.proto</include>
+                  <include>yarn_server_federation_protos.proto</include>
                   <include>ResourceTracker.proto</include>
                   <include>SCMUploader.proto</include>
                   <include>collectornodemanager_protocol.proto</include>

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/FederationMembershipStateStore.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/FederationMembershipStateStore.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/FederationMembershipStateStore.java
new file mode 100644
index 0000000..378eadc
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/FederationMembershipStateStore.java
@@ -0,0 +1,126 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.exceptions.YarnException;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClusterInfoRequest;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClusterInfoResponse;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClustersInfoRequest;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClustersInfoResponse;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterDeregisterRequest;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterDeregisterResponse;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterHeartbeatRequest;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterHeartbeatResponse;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterRegisterRequest;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterRegisterResponse;
+import org.apache.hadoop.yarn.server.records.Version;
+
+/**
+ * FederationMembershipStateStore maintains the state of all
+ * <em>subcluster(s)</em> as encapsulated by {@code SubClusterInfo} for all the
+ * subcluster(s) that are participating in federation.
+ */
+@Private
+@Unstable
+public interface FederationMembershipStateStore {
+
+  /**
+   * Get the {@link Version} of the underlying federation membership state
+   * store.
+   *
+   * @return the {@link Version} of the underlying federation membership state
+   *         store
+   */
+  Version getMembershipStateStoreVersion();
+
+  /**
+   * Register a <em>subcluster</em> by publishing capabilities as represented by
+   * {@code SubClusterInfo} to indicate participation in federation. This is
+   * typically done during initialization or restart/failover of the
+   * subcluster's <code>ResourceManager</code>. Upon successful registration, an
+   * identifier for the <em>subcluster</em> which is unique across the federated
+   * cluster is returned. The identifier is static, i.e. preserved across
+   * restarts and failover.
+   *
+   * @param registerSubClusterRequest the capabilities of the subcluster that
+   *          wants to participate in federation. The subcluster id is also
+   *          specified in case registration is triggered by restart/failover
+   * @return response empty on successfully if registration was successful
+   * @throws YarnException if the request is invalid/fails
+   */
+  SubClusterRegisterResponse registerSubCluster(
+      SubClusterRegisterRequest registerSubClusterRequest) throws YarnException;
+
+  /**
+   * Deregister a <em>subcluster</em> identified by {@code SubClusterId} to
+   * change state in federation. This can be done to mark the sub cluster lost,
+   * deregistered, or decommissioned.
+   *
+   * @param subClusterDeregisterRequest - the request to deregister the
+   *          sub-cluster from federation.
+   * @return response empty on successfully deregistering the subcluster state
+   * @throws YarnException if the request is invalid/fails
+   */
+  SubClusterDeregisterResponse deregisterSubCluster(
+      SubClusterDeregisterRequest subClusterDeregisterRequest)
+      throws YarnException;
+
+  /**
+   * Periodic heartbeat from a <code>ResourceManager</code> participating in
+   * federation to indicate liveliness. The heartbeat publishes the current
+   * capabilities as represented by {@code SubClusterInfo} of the subcluster.
+   * Currently response is empty if the operation was successful, if not an
+   * exception reporting reason for a failure.
+   *
+   * @param subClusterHeartbeatRequest the capabilities of the subcluster that
+   *          wants to keep alive its participation in federation
+   * @return response currently empty on if heartbeat was successfully processed
+   * @throws YarnException if the request is invalid/fails
+   */
+  SubClusterHeartbeatResponse subClusterHeartbeat(
+      SubClusterHeartbeatRequest subClusterHeartbeatRequest)
+      throws YarnException;
+
+  /**
+   * Get the membership information of <em>subcluster</em> as identified by
+   * {@code SubClusterId}. The membership information includes the cluster
+   * endpoint and current capabilities as represented by {@code SubClusterInfo}.
+   *
+   * @param subClusterRequest the subcluster whose information is required
+   * @return the {@code SubClusterInfo}
+   * @throws YarnException if the request is invalid/fails
+   */
+  GetSubClusterInfoResponse getSubCluster(
+      GetSubClusterInfoRequest subClusterRequest) throws YarnException;
+
+  /**
+   * Get the membership information of all the <em>subclusters</em> that are
+   * currently participating in federation. The membership information includes
+   * the cluster endpoint and current capabilities as represented by
+   * {@code SubClusterInfo}.
+   *
+   * @param subClustersRequest request for sub-clusters information
+   * @return a map of {@code SubClusterInfo} keyed by the {@code SubClusterId}
+   * @throws YarnException if the request is invalid/fails
+   */
+  GetSubClustersInfoResponse getSubClusters(
+      GetSubClustersInfoRequest subClustersRequest) throws YarnException;
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/package-info.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/package-info.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/package-info.java
new file mode 100644
index 0000000..33179e9
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/package-info.java
@@ -0,0 +1,17 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+package org.apache.hadoop.yarn.server.federation.store;
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoRequest.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoRequest.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoRequest.java
new file mode 100644
index 0000000..656dea9
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoRequest.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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * Request class to obtain information about a sub-cluster identified by its
+ * {@link SubClusterId}.
+ */
+@Private
+@Unstable
+public abstract class GetSubClusterInfoRequest {
+
+  @Private
+  @Unstable
+  public static GetSubClusterInfoRequest newInstance(
+      SubClusterId subClusterId) {
+    GetSubClusterInfoRequest subClusterRequest =
+        Records.newRecord(GetSubClusterInfoRequest.class);
+    subClusterRequest.setSubClusterId(subClusterId);
+    return subClusterRequest;
+  }
+
+  /**
+   * Get the {@link SubClusterId} representing the unique identifier of the
+   * subcluster.
+   *
+   * @return the subcluster identifier
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterId getSubClusterId();
+
+  /**
+   * Set the {@link SubClusterId} representing the unique identifier of the
+   * subcluster.
+   *
+   * @param subClusterId the subcluster identifier
+   */
+  @Public
+  @Unstable
+  public abstract void setSubClusterId(SubClusterId subClusterId);
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoResponse.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoResponse.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoResponse.java
new file mode 100644
index 0000000..f7bc74d
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClusterInfoResponse.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
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * Response to a query with {@link SubClusterInfo} about a sub-cluster.
+ */
+@Private
+@Unstable
+public abstract class GetSubClusterInfoResponse {
+
+  @Private
+  @Unstable
+  public static GetSubClusterInfoResponse newInstance(
+      SubClusterInfo subClusterInfo) {
+    GetSubClusterInfoResponse registerSubClusterRequest =
+        Records.newRecord(GetSubClusterInfoResponse.class);
+    registerSubClusterRequest.setSubClusterInfo(subClusterInfo);
+    return registerSubClusterRequest;
+  }
+
+  /**
+   * Get the {@link SubClusterInfo} encapsulating the information about the
+   * sub-cluster.
+   *
+   * @return the information pertaining to the sub-cluster
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterInfo getSubClusterInfo();
+
+  /**
+   * Set the {@link SubClusterInfo} encapsulating the information about the
+   * sub-cluster.
+   *
+   * @param subClusterInfo the information pertaining to the sub-cluster
+   */
+  @Private
+  @Unstable
+  public abstract void setSubClusterInfo(SubClusterInfo subClusterInfo);
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoRequest.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoRequest.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoRequest.java
new file mode 100644
index 0000000..3264d81
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoRequest.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.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * Request class to obtain information about all sub-clusters that are
+ * participating in federation.
+ */
+@Private
+@Unstable
+public abstract class GetSubClustersInfoRequest {
+
+  @Public
+  @Unstable
+  public static GetSubClustersInfoRequest newInstance(
+      boolean filterInactiveSubClusters) {
+    GetSubClustersInfoRequest request =
+        Records.newRecord(GetSubClustersInfoRequest.class);
+    request.setFilterInactiveSubClusters(filterInactiveSubClusters);
+    return request;
+  }
+
+  /**
+   * Get the flag that indicates whether only active sub-clusters should be
+   * returned.
+   *
+   * @return whether to filter out inactive sub-clusters
+   */
+  @Public
+  @Unstable
+  public abstract boolean getFilterInactiveSubClusters();
+
+  /**
+   * Set the flag that indicates whether only active sub-clusters should be
+   * returned.
+   *
+   * @param filterInactiveSubClusters whether to filter out inactive
+   *          sub-clusters
+   */
+  @Public
+  @Unstable
+  public abstract void setFilterInactiveSubClusters(
+      boolean filterInactiveSubClusters);
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoResponse.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoResponse.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoResponse.java
new file mode 100644
index 0000000..bcf75ab
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/GetSubClustersInfoResponse.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.server.federation.store.records;
+
+import java.util.List;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * Response to a query with list of {@link SubClusterInfo} about all
+ * sub-clusters that are currently participating in Federation.
+ */
+@Private
+@Unstable
+public abstract class GetSubClustersInfoResponse {
+
+  @Public
+  @Unstable
+  public static GetSubClustersInfoResponse newInstance(
+      List<SubClusterInfo> subClusters) {
+    GetSubClustersInfoResponse subClusterInfos =
+        Records.newRecord(GetSubClustersInfoResponse.class);
+    subClusterInfos.setSubClusters(subClusters);
+    return subClusterInfos;
+  }
+
+  /**
+   * Get the list of {@link SubClusterInfo} representing the information about
+   * all sub-clusters that are currently participating in Federation.
+   *
+   * @return the list of {@link SubClusterInfo}
+   */
+  @Public
+  @Unstable
+  public abstract List<SubClusterInfo> getSubClusters();
+
+  /**
+   * Set the list of {@link SubClusterInfo} representing the information about
+   * all sub-clusters that are currently participating in Federation.
+   *
+   * @param subClusters the list of {@link SubClusterInfo}
+   */
+  @Private
+  @Unstable
+  public abstract void setSubClusters(List<SubClusterInfo> subClusters);
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterRequest.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterRequest.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterRequest.java
new file mode 100644
index 0000000..50a50a1
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterRequest.java
@@ -0,0 +1,89 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * The request sent to set the state of a subcluster to either
+ * SC_DECOMMISSIONED, SC_LOST, or SC_DEREGISTERED.
+ *
+ * <p>
+ * The update includes details such as:
+ * <ul>
+ * <li>{@link SubClusterId}</li>
+ * <li>{@link SubClusterState}</li>
+ * </ul>
+ */
+@Private
+@Unstable
+public abstract class SubClusterDeregisterRequest {
+
+  @Private
+  @Unstable
+  public static SubClusterDeregisterRequest newInstance(
+      SubClusterId subClusterId, SubClusterState subClusterState) {
+    SubClusterDeregisterRequest registerRequest =
+        Records.newRecord(SubClusterDeregisterRequest.class);
+    registerRequest.setSubClusterId(subClusterId);
+    registerRequest.setState(subClusterState);
+    return registerRequest;
+  }
+
+  /**
+   * Get the {@link SubClusterId} representing the unique identifier of the
+   * subcluster.
+   *
+   * @return the subcluster identifier
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterId getSubClusterId();
+
+  /**
+   * Set the {@link SubClusterId} representing the unique identifier of the
+   * subcluster.
+   *
+   * @param subClusterId the subcluster identifier
+   */
+  @Private
+  @Unstable
+  public abstract void setSubClusterId(SubClusterId subClusterId);
+
+  /**
+   * Get the {@link SubClusterState} of the subcluster.
+   *
+   * @return the state of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterState getState();
+
+  /**
+   * Set the {@link SubClusterState} of the subcluster.
+   *
+   * @param state the state of the subCluster
+   */
+  @Private
+  @Unstable
+  public abstract void setState(SubClusterState state);
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterResponse.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterResponse.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterResponse.java
new file mode 100644
index 0000000..74fe994
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterDeregisterResponse.java
@@ -0,0 +1,42 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * SubClusterDeregisterResponse contains the answer from the {@code
+ * FederationMembershipStateStore} to a request to deregister the sub cluster.
+ * Currently response is empty if the operation was successful, if not an
+ * exception reporting reason for a failure.
+ */
+@Private
+@Unstable
+public abstract class SubClusterDeregisterResponse {
+
+  @Private
+  @Unstable
+  public static SubClusterDeregisterResponse newInstance() {
+    SubClusterDeregisterResponse response =
+        Records.newRecord(SubClusterDeregisterResponse.class);
+    return response;
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatRequest.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatRequest.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatRequest.java
new file mode 100644
index 0000000..3a07c18
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatRequest.java
@@ -0,0 +1,149 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * SubClusterHeartbeatRequest is a report of the runtime information of the
+ * subcluster that is participating in federation.
+ *
+ * <p>
+ * It includes information such as:
+ * <ul>
+ * <li>{@link SubClusterId}</li>
+ * <li>The URL of the subcluster</li>
+ * <li>The timestamp representing the last start time of the subCluster</li>
+ * <li>{@code FederationsubClusterState}</li>
+ * <li>The current capacity and utilization of the subCluster</li>
+ * </ul>
+ */
+@Private
+@Unstable
+public abstract class SubClusterHeartbeatRequest {
+
+  @Private
+  @Unstable
+  public static SubClusterHeartbeatRequest newInstance(
+      SubClusterId subClusterId, SubClusterState state, String capability) {
+    return newInstance(subClusterId, 0, state, capability);
+  }
+
+  @Private
+  @Unstable
+  public static SubClusterHeartbeatRequest newInstance(
+      SubClusterId subClusterId, long lastHeartBeat, SubClusterState state,
+      String capability) {
+    SubClusterHeartbeatRequest subClusterHeartbeatRequest =
+        Records.newRecord(SubClusterHeartbeatRequest.class);
+    subClusterHeartbeatRequest.setSubClusterId(subClusterId);
+    subClusterHeartbeatRequest.setLastHeartBeat(lastHeartBeat);
+    subClusterHeartbeatRequest.setState(state);
+    subClusterHeartbeatRequest.setCapability(capability);
+    return subClusterHeartbeatRequest;
+  }
+
+  /**
+   * Get the {@link SubClusterId} representing the unique identifier of the
+   * subcluster.
+   *
+   * @return the subcluster identifier
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterId getSubClusterId();
+
+  /**
+   * Set the {@link SubClusterId} representing the unique identifier of the
+   * subCluster.
+   *
+   * @param subClusterId the subCluster identifier
+   */
+  @Private
+  @Unstable
+  public abstract void setSubClusterId(SubClusterId subClusterId);
+
+  /**
+   * Get the last heart beat time of the subcluster.
+   *
+   * @return the state of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract long getLastHeartBeat();
+
+  /**
+   * Set the last heartbeat time of the subcluster.
+   *
+   * @param time the last heartbeat time of the subcluster
+   */
+  @Private
+  @Unstable
+  public abstract void setLastHeartBeat(long time);
+
+  /**
+   * Get the {@link SubClusterState} of the subcluster.
+   *
+   * @return the state of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterState getState();
+
+  /**
+   * Set the {@link SubClusterState} of the subcluster.
+   *
+   * @param state the state of the subCluster
+   */
+  @Private
+  @Unstable
+  public abstract void setState(SubClusterState state);
+
+  /**
+   * Get the current capacity and utilization of the subcluster. This is the
+   * JAXB marshalled string representation of the <code>ClusterMetrics</code>.
+   *
+   * @return the current capacity and utilization of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract String getCapability();
+
+  /**
+   * Set the current capacity and utilization of the subCluster. This is the
+   * JAXB marshalled string representation of the <code>ClusterMetrics</code>.
+   *
+   * @param capability the current capacity and utilization of the subcluster
+   */
+  @Private
+  @Unstable
+  public abstract void setCapability(String capability);
+
+  @Override
+  public String toString() {
+    return "SubClusterHeartbeatRequest [getSubClusterId() = "
+        + getSubClusterId() + ", getState() = " + getState()
+        + ", getLastHeartBeat = " + getLastHeartBeat() + ", getCapability() = "
+        + getCapability() + "]";
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatResponse.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatResponse.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatResponse.java
new file mode 100644
index 0000000..0b7fd8c
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterHeartbeatResponse.java
@@ -0,0 +1,45 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * SubClusterHeartbeatResponse contains the response from the {@code
+ * FederationMembershipStateStore} to a periodic heartbeat to indicate
+ * liveliness from a <code>ResourceManager</code> participating in federation.
+ * Currently response is empty if the operation was successful, if not an
+ * exception reporting reason for a failure.
+ * <p>
+ * NOTE: This can be extended to push down policies in future
+ */
+@Private
+@Unstable
+public abstract class SubClusterHeartbeatResponse {
+
+  @Private
+  @Unstable
+  public static SubClusterHeartbeatResponse newInstance() {
+    SubClusterHeartbeatResponse response =
+        Records.newRecord(SubClusterHeartbeatResponse.class);
+    return response;
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterId.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterId.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterId.java
new file mode 100644
index 0000000..fec967d
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterId.java
@@ -0,0 +1,100 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * SubClusterId represents the <em>globally unique</em> identifier for a
+ * subcluster that is participating in federation.
+ *
+ * <p>
+ * The globally unique nature of the identifier is obtained from the
+ * <code>FederationMembershipStateStore</code> on initialization.
+ */
+@Private
+@Unstable
+public abstract class SubClusterId implements Comparable<SubClusterId> {
+
+  @Private
+  @Unstable
+  public static SubClusterId newInstance(String subClusterId) {
+    SubClusterId id = Records.newRecord(SubClusterId.class);
+    id.setId(subClusterId);
+    return id;
+  }
+
+  /**
+   * Get the string identifier of the <em>subcluster</em> which is unique across
+   * the federated cluster. The identifier is static, i.e. preserved across
+   * restarts and failover.
+   *
+   * @return unique identifier of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract String getId();
+
+  /**
+   * Set the string identifier of the <em>subcluster</em> which is unique across
+   * the federated cluster. The identifier is static, i.e. preserved across
+   * restarts and failover.
+   *
+   * @param subClusterId unique identifier of the subcluster
+   */
+  @Private
+  @Unstable
+  protected abstract void setId(String subClusterId);
+
+  @Override
+  public boolean equals(Object obj) {
+    if (this == obj) {
+      return true;
+    }
+    if (obj == null) {
+      return false;
+    }
+    if (getClass() != obj.getClass()) {
+      return false;
+    }
+    SubClusterId other = (SubClusterId) obj;
+    return this.getId().equals(other.getId());
+  }
+
+  @Override
+  public int hashCode() {
+    return getId().hashCode();
+  }
+
+  @Override
+  public int compareTo(SubClusterId other) {
+    return getId().compareTo(other.getId());
+  }
+
+  @Override
+  public String toString() {
+    StringBuilder sb = new StringBuilder();
+    sb.append(getId());
+    return sb.toString();
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterInfo.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterInfo.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterInfo.java
new file mode 100644
index 0000000..f13c8f1
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterInfo.java
@@ -0,0 +1,263 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * SubClusterInfo is a report of the runtime information of the subcluster that
+ * is participating in federation.
+ *
+ * <p>
+ * It includes information such as:
+ * <ul>
+ * <li>{@link SubClusterId}</li>
+ * <li>The URL of the subcluster</li>
+ * <li>The timestamp representing the last start time of the subCluster</li>
+ * <li>{@code FederationsubClusterState}</li>
+ * <li>The current capacity and utilization of the subCluster</li>
+ * </ul>
+ */
+@Private
+@Unstable
+public abstract class SubClusterInfo {
+
+  @Private
+  @Unstable
+  public static SubClusterInfo newInstance(SubClusterId subClusterId,
+      String amRMServiceAddress, String clientRMServiceAddress,
+      String rmAdminServiceAddress, String rmWebServiceAddress,
+      SubClusterState state, long lastStartTime, String capability) {
+    return newInstance(subClusterId, amRMServiceAddress, clientRMServiceAddress,
+        rmAdminServiceAddress, rmWebServiceAddress, 0, state, lastStartTime,
+        capability);
+  }
+
+  @Private
+  @Unstable
+  public static SubClusterInfo newInstance(SubClusterId subClusterId,
+      String amRMServiceAddress, String clientRMServiceAddress,
+      String rmAdminServiceAddress, String rmWebServiceAddress,
+      long lastHeartBeat, SubClusterState state, long lastStartTime,
+      String capability) {
+    SubClusterInfo subClusterInfo = Records.newRecord(SubClusterInfo.class);
+    subClusterInfo.setSubClusterId(subClusterId);
+    subClusterInfo.setAMRMServiceAddress(amRMServiceAddress);
+    subClusterInfo.setClientRMServiceAddress(clientRMServiceAddress);
+    subClusterInfo.setRMAdminServiceAddress(rmAdminServiceAddress);
+    subClusterInfo.setRMWebServiceAddress(rmWebServiceAddress);
+    subClusterInfo.setLastHeartBeat(lastHeartBeat);
+    subClusterInfo.setState(state);
+    subClusterInfo.setLastStartTime(lastStartTime);
+    subClusterInfo.setCapability(capability);
+    return subClusterInfo;
+  }
+
+  /**
+   * Get the {@link SubClusterId} representing the unique identifier of the
+   * subcluster.
+   *
+   * @return the subcluster identifier
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterId getSubClusterId();
+
+  /**
+   * Set the {@link SubClusterId} representing the unique identifier of the
+   * subCluster.
+   *
+   * @param subClusterId the subCluster identifier
+   */
+  @Private
+  @Unstable
+  public abstract void setSubClusterId(SubClusterId subClusterId);
+
+  /**
+   * Get the URL of the AM-RM service endpoint of the subcluster
+   * <code>ResourceManager</code>.
+   *
+   * @return the URL of the AM-RM service endpoint of the subcluster
+   *         <code>ResourceManager</code>
+   */
+  @Public
+  @Unstable
+  public abstract String getAMRMServiceAddress();
+
+  /**
+   * Set the URL of the AM-RM service endpoint of the subcluster
+   * <code>ResourceManager</code>.
+   *
+   * @param amRMServiceAddress the URL of the AM-RM service endpoint of the
+   *          subcluster <code>ResourceManager</code>
+   */
+  @Private
+  @Unstable
+  public abstract void setAMRMServiceAddress(String amRMServiceAddress);
+
+  /**
+   * Get the URL of the client-RM service endpoint of the subcluster
+   * <code>ResourceManager</code>.
+   *
+   * @return the URL of the client-RM service endpoint of the subcluster
+   *         <code>ResourceManager</code>
+   */
+  @Public
+  @Unstable
+  public abstract String getClientRMServiceAddress();
+
+  /**
+   * Set the URL of the client-RM service endpoint of the subcluster
+   * <code>ResourceManager</code>.
+   *
+   * @param clientRMServiceAddress the URL of the client-RM service endpoint of
+   *          the subCluster <code>ResourceManager</code>
+   */
+  @Private
+  @Unstable
+  public abstract void setClientRMServiceAddress(String clientRMServiceAddress);
+
+  /**
+   * Get the URL of the <code>ResourceManager</code> administration service.
+   *
+   * @return the URL of the <code>ResourceManager</code> administration service
+   */
+  @Public
+  @Unstable
+  public abstract String getRMAdminServiceAddress();
+
+  /**
+   * Set the URL of the <code>ResourceManager</code> administration service.
+   *
+   * @param rmAdminServiceAddress the URL of the <code>ResourceManager</code>
+   *          administration service.
+   */
+  @Private
+  @Unstable
+  public abstract void setRMAdminServiceAddress(String rmAdminServiceAddress);
+
+  /**
+   * Get the URL of the <code>ResourceManager</code> web application interface.
+   *
+   * @return the URL of the <code>ResourceManager</code> web application
+   *         interface.
+   */
+  @Public
+  @Unstable
+  public abstract String getRMWebServiceAddress();
+
+  /**
+   * Set the URL of the <code>ResourceManager</code> web application interface.
+   *
+   * @param rmWebServiceAddress the URL of the <code>ResourceManager</code> web
+   *          application interface.
+   */
+  @Private
+  @Unstable
+  public abstract void setRMWebServiceAddress(String rmWebServiceAddress);
+
+  /**
+   * Get the last heart beat time of the subcluster.
+   *
+   * @return the state of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract long getLastHeartBeat();
+
+  /**
+   * Set the last heartbeat time of the subcluster.
+   *
+   * @param time the last heartbeat time of the subcluster
+   */
+  @Private
+  @Unstable
+  public abstract void setLastHeartBeat(long time);
+
+  /**
+   * Get the {@link SubClusterState} of the subcluster.
+   *
+   * @return the state of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterState getState();
+
+  /**
+   * Set the {@link SubClusterState} of the subcluster.
+   *
+   * @param state the state of the subCluster
+   */
+  @Private
+  @Unstable
+  public abstract void setState(SubClusterState state);
+
+  /**
+   * Get the timestamp representing the last start time of the subcluster.
+   *
+   * @return the timestamp representing the last start time of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract long getLastStartTime();
+
+  /**
+   * Set the timestamp representing the last start time of the subcluster.
+   *
+   * @param lastStartTime the timestamp representing the last start time of the
+   *          subcluster
+   */
+  @Private
+  @Unstable
+  public abstract void setLastStartTime(long lastStartTime);
+
+  /**
+   * Get the current capacity and utilization of the subcluster. This is the
+   * JAXB marshalled string representation of the <code>ClusterMetrics</code>.
+   *
+   * @return the current capacity and utilization of the subcluster
+   */
+  @Public
+  @Unstable
+  public abstract String getCapability();
+
+  /**
+   * Set the current capacity and utilization of the subCluster. This is the
+   * JAXB marshalled string representation of the <code>ClusterMetrics</code>.
+   *
+   * @param capability the current capacity and utilization of the subcluster
+   */
+  @Private
+  @Unstable
+  public abstract void setCapability(String capability);
+
+  @Override
+  public String toString() {
+    return "SubClusterInfo [getSubClusterId() = " + getSubClusterId()
+        + ", getAMRMServiceAddress() = " + getAMRMServiceAddress()
+        + ", getClientRMServiceAddress() = " + getClientRMServiceAddress()
+        + ", getRMAdminServiceAddress() = " + getRMAdminServiceAddress()
+        + ", getRMWebServiceAddress() = " + getRMWebServiceAddress()
+        + ", getState() = " + getState() + ", getLastStartTime() = "
+        + getLastStartTime() + ", getCapability() = " + getCapability() + "]";
+  }
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterRequest.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterRequest.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterRequest.java
new file mode 100644
index 0000000..8864fe3
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterRequest.java
@@ -0,0 +1,74 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceAudience.Public;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * <p>
+ * SubClusterRegisterRequest is a request by a sub-cluster
+ * {@code ResourceManager} to participate in federation.
+ *
+ * <p>
+ * It includes information such as:
+ * <ul>
+ * <li>{@link SubClusterId}</li>
+ * <li>The URL of the subcluster</li>
+ * <li>The timestamp representing the last start time of the subCluster</li>
+ * <li>{@code FederationsubClusterState}</li>
+ * <li>The current capacity and utilization of the subCluster</li>
+ * </ul>
+ */
+@Private
+@Unstable
+public abstract class SubClusterRegisterRequest {
+
+  @Private
+  @Unstable
+  public static SubClusterRegisterRequest newInstance(
+      SubClusterInfo subClusterInfo) {
+    SubClusterRegisterRequest registerSubClusterRequest =
+        Records.newRecord(SubClusterRegisterRequest.class);
+    registerSubClusterRequest.setSubClusterInfo(subClusterInfo);
+    return registerSubClusterRequest;
+  }
+
+  /**
+   * Get the {@link SubClusterInfo} encapsulating the information about the
+   * sub-cluster.
+   *
+   * @return the information pertaining to the sub-cluster
+   */
+  @Public
+  @Unstable
+  public abstract SubClusterInfo getSubClusterInfo();
+
+  /**
+   * Set the {@link SubClusterInfo} encapsulating the information about the
+   * sub-cluster.
+   *
+   * @param subClusterInfo the information pertaining to the sub-cluster
+   */
+  @Public
+  @Unstable
+  public abstract void setSubClusterInfo(SubClusterInfo subClusterInfo);
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterResponse.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterResponse.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterResponse.java
new file mode 100644
index 0000000..060a857
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterRegisterResponse.java
@@ -0,0 +1,44 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.util.Records;
+
+/**
+ * SubClusterRegisterResponse contains the response from the {@code
+ * FederationMembershipStateStore} to a registration request from a
+ * <code>ResourceManager</code> to participate in federation.
+ *
+ * Currently response is empty if the operation was successful, if not an
+ * exception reporting reason for a failure.
+ */
+@Private
+@Unstable
+public abstract class SubClusterRegisterResponse {
+
+  @Private
+  @Unstable
+  public static SubClusterRegisterResponse newInstance() {
+    SubClusterRegisterResponse response =
+        Records.newRecord(SubClusterRegisterResponse.class);
+    return response;
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterState.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterState.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterState.java
new file mode 100644
index 0000000..22cec99
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/SubClusterState.java
@@ -0,0 +1,60 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+
+/**
+ * <p>
+ * State of a <code>SubCluster</code>.
+ * </p>
+ */
+@Private
+@Unstable
+public enum SubClusterState {
+  /** Newly registered subcluster, before the first heartbeat. */
+  SC_NEW,
+
+  /** Subcluster is registered and the RM sent a heartbeat recently. */
+  SC_RUNNING,
+
+  /** Subcluster is unhealthy. */
+  SC_UNHEALTHY,
+
+  /** Subcluster is in the process of being out of service. */
+  SC_DECOMMISSIONING,
+
+  /** Subcluster is out of service. */
+  SC_DECOMMISSIONED,
+
+  /** RM has not sent a heartbeat for some configured time threshold. */
+  SC_LOST,
+
+  /** Subcluster has unregistered. */
+  SC_UNREGISTERED;
+
+  public boolean isUnusable() {
+    return (this != SC_RUNNING && this != SC_NEW);
+  }
+
+  public boolean isFinal() {
+    return (this == SC_UNREGISTERED || this == SC_DECOMMISSIONED
+        || this == SC_LOST);
+  }
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoRequestPBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoRequestPBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoRequestPBImpl.java
new file mode 100644
index 0000000..c61c419
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoRequestPBImpl.java
@@ -0,0 +1,125 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records.impl.pb;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClusterInfoRequestProto;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClusterInfoRequestProtoOrBuilder;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.SubClusterIdProto;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClusterInfoRequest;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterId;
+
+import com.google.protobuf.TextFormat;
+
+/**
+ * Protocol buffer based implementation of {@link GetSubClusterInfoRequest}.
+ */
+@Private
+@Unstable
+public class GetSubClusterInfoRequestPBImpl extends GetSubClusterInfoRequest {
+
+  private GetSubClusterInfoRequestProto proto =
+      GetSubClusterInfoRequestProto.getDefaultInstance();
+  private GetSubClusterInfoRequestProto.Builder builder = null;
+  private boolean viaProto = false;
+
+  public GetSubClusterInfoRequestPBImpl() {
+    builder = GetSubClusterInfoRequestProto.newBuilder();
+  }
+
+  public GetSubClusterInfoRequestPBImpl(GetSubClusterInfoRequestProto proto) {
+    this.proto = proto;
+    viaProto = true;
+  }
+
+  public GetSubClusterInfoRequestProto getProto() {
+    mergeLocalToProto();
+    proto = viaProto ? proto : builder.build();
+    viaProto = true;
+    return proto;
+  }
+
+  private void mergeLocalToProto() {
+    if (viaProto) {
+      maybeInitBuilder();
+    }
+    mergeLocalToBuilder();
+    proto = builder.build();
+    viaProto = true;
+  }
+
+  private void maybeInitBuilder() {
+    if (viaProto || builder == null) {
+      builder = GetSubClusterInfoRequestProto.newBuilder(proto);
+    }
+    viaProto = false;
+  }
+
+  private void mergeLocalToBuilder() {
+  }
+
+  @Override
+  public int hashCode() {
+    return getProto().hashCode();
+  }
+
+  @Override
+  public boolean equals(Object other) {
+    if (other == null) {
+      return false;
+    }
+    if (other.getClass().isAssignableFrom(this.getClass())) {
+      return this.getProto().equals(this.getClass().cast(other).getProto());
+    }
+    return false;
+  }
+
+  @Override
+  public String toString() {
+    return TextFormat.shortDebugString(getProto());
+  }
+
+  @Override
+  public SubClusterId getSubClusterId() {
+    GetSubClusterInfoRequestProtoOrBuilder p = viaProto ? proto : builder;
+    if (!p.hasSubClusterId()) {
+      return null;
+    }
+    return convertFromProtoFormat(p.getSubClusterId());
+  }
+
+  @Override
+  public void setSubClusterId(SubClusterId subClusterId) {
+    maybeInitBuilder();
+    if (subClusterId == null) {
+      builder.clearSubClusterId();
+      return;
+    }
+    builder.setSubClusterId(convertToProtoFormat(subClusterId));
+  }
+
+  private SubClusterId convertFromProtoFormat(SubClusterIdProto sc) {
+    return new SubClusterIdPBImpl(sc);
+  }
+
+  private SubClusterIdProto convertToProtoFormat(SubClusterId sc) {
+    return ((SubClusterIdPBImpl) sc).getProto();
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoResponsePBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoResponsePBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoResponsePBImpl.java
new file mode 100644
index 0000000..d0bcc33
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClusterInfoResponsePBImpl.java
@@ -0,0 +1,134 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records.impl.pb;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClusterInfoResponseProto;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClusterInfoResponseProtoOrBuilder;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.SubClusterInfoProto;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClusterInfoResponse;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterInfo;
+
+import com.google.protobuf.TextFormat;
+
+/**
+ * Protocol buffer based implementation of {@link GetSubClusterInfoResponse}.
+ */
+@Private
+@Unstable
+public class GetSubClusterInfoResponsePBImpl extends GetSubClusterInfoResponse {
+
+  private GetSubClusterInfoResponseProto proto =
+      GetSubClusterInfoResponseProto.getDefaultInstance();
+  private GetSubClusterInfoResponseProto.Builder builder = null;
+  private boolean viaProto = false;
+
+  private SubClusterInfo subClusterInfo = null;
+
+  public GetSubClusterInfoResponsePBImpl() {
+    builder = GetSubClusterInfoResponseProto.newBuilder();
+  }
+
+  public GetSubClusterInfoResponsePBImpl(GetSubClusterInfoResponseProto proto) {
+    this.proto = proto;
+    viaProto = true;
+  }
+
+  public GetSubClusterInfoResponseProto getProto() {
+    mergeLocalToProto();
+    proto = viaProto ? proto : builder.build();
+    viaProto = true;
+    return proto;
+  }
+
+  private void mergeLocalToProto() {
+    if (viaProto) {
+      maybeInitBuilder();
+    }
+    mergeLocalToBuilder();
+    proto = builder.build();
+    viaProto = true;
+  }
+
+  private void maybeInitBuilder() {
+    if (viaProto || builder == null) {
+      builder = GetSubClusterInfoResponseProto.newBuilder(proto);
+    }
+    viaProto = false;
+  }
+
+  private void mergeLocalToBuilder() {
+    if (this.subClusterInfo != null) {
+      builder.setSubClusterInfo(convertToProtoFormat(this.subClusterInfo));
+    }
+  }
+
+  @Override
+  public int hashCode() {
+    return getProto().hashCode();
+  }
+
+  @Override
+  public boolean equals(Object other) {
+    if (other == null) {
+      return false;
+    }
+    if (other.getClass().isAssignableFrom(this.getClass())) {
+      return this.getProto().equals(this.getClass().cast(other).getProto());
+    }
+    return false;
+  }
+
+  @Override
+  public String toString() {
+    return TextFormat.shortDebugString(getProto());
+  }
+
+  @Override
+  public SubClusterInfo getSubClusterInfo() {
+    GetSubClusterInfoResponseProtoOrBuilder p = viaProto ? proto : builder;
+    if (this.subClusterInfo != null) {
+      return this.subClusterInfo;
+    }
+    if (!p.hasSubClusterInfo()) {
+      return null;
+    }
+    this.subClusterInfo = convertFromProtoFormat(p.getSubClusterInfo());
+    return this.subClusterInfo;
+  }
+
+  @Override
+  public void setSubClusterInfo(SubClusterInfo subClusterInfo) {
+    maybeInitBuilder();
+    if (subClusterInfo == null) {
+      builder.clearSubClusterInfo();
+    }
+    this.subClusterInfo = subClusterInfo;
+  }
+
+  private SubClusterInfo convertFromProtoFormat(
+      SubClusterInfoProto clusterInfo) {
+    return new SubClusterInfoPBImpl(clusterInfo);
+  }
+
+  private SubClusterInfoProto convertToProtoFormat(SubClusterInfo clusterInfo) {
+    return ((SubClusterInfoPBImpl) clusterInfo).getProto();
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoRequestPBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoRequestPBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoRequestPBImpl.java
new file mode 100644
index 0000000..2b848c0
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoRequestPBImpl.java
@@ -0,0 +1,108 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.yarn.server.federation.store.records.impl.pb;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClustersInfoRequestProto;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClustersInfoRequestProtoOrBuilder;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClustersInfoRequest;
+
+import com.google.protobuf.TextFormat;
+
+/**
+ * Protocol buffer based implementation of {@link GetSubClustersInfoRequest}.
+ */
+@Private
+@Unstable
+public class GetSubClustersInfoRequestPBImpl extends GetSubClustersInfoRequest {
+
+  private GetSubClustersInfoRequestProto proto =
+      GetSubClustersInfoRequestProto.getDefaultInstance();
+  private GetSubClustersInfoRequestProto.Builder builder = null;
+  private boolean viaProto = false;
+
+  public GetSubClustersInfoRequestPBImpl() {
+    builder = GetSubClustersInfoRequestProto.newBuilder();
+  }
+
+  public GetSubClustersInfoRequestPBImpl(GetSubClustersInfoRequestProto proto) {
+    this.proto = proto;
+    viaProto = true;
+  }
+
+  public GetSubClustersInfoRequestProto getProto() {
+    mergeLocalToProto();
+    proto = viaProto ? proto : builder.build();
+    viaProto = true;
+    return proto;
+  }
+
+  private void mergeLocalToProto() {
+    if (viaProto) {
+      maybeInitBuilder();
+    }
+    mergeLocalToBuilder();
+    proto = builder.build();
+    viaProto = true;
+  }
+
+  private void maybeInitBuilder() {
+    if (viaProto || builder == null) {
+      builder = GetSubClustersInfoRequestProto.newBuilder(proto);
+    }
+    viaProto = false;
+  }
+
+  private void mergeLocalToBuilder() {
+  }
+
+  @Override
+  public int hashCode() {
+    return getProto().hashCode();
+  }
+
+  @Override
+  public boolean equals(Object other) {
+    if (other == null) {
+      return false;
+    }
+    if (other.getClass().isAssignableFrom(this.getClass())) {
+      return this.getProto().equals(this.getClass().cast(other).getProto());
+    }
+    return false;
+  }
+
+  @Override
+  public String toString() {
+    return TextFormat.shortDebugString(getProto());
+  }
+
+  @Override
+  public boolean getFilterInactiveSubClusters() {
+    GetSubClustersInfoRequestProtoOrBuilder p = viaProto ? proto : builder;
+    return p.getFilterInactiveSubclusters();
+  }
+
+  @Override
+  public void setFilterInactiveSubClusters(boolean filterInactiveSubClusters) {
+    maybeInitBuilder();
+    builder.setFilterInactiveSubclusters(filterInactiveSubClusters);
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/8b02ec0b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoResponsePBImpl.java
----------------------------------------------------------------------
diff --git a/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoResponsePBImpl.java b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoResponsePBImpl.java
new file mode 100644
index 0000000..d39ef7f
--- /dev/null
+++ b/hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/federation/store/records/impl/pb/GetSubClustersInfoResponsePBImpl.java
@@ -0,0 +1,184 @@
+/**
+ * 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.federation.store.records.impl.pb;
+
+import java.util.ArrayList;
+import java.util.Iterator;
+import java.util.List;
+
+import org.apache.hadoop.classification.InterfaceAudience.Private;
+import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClustersInfoResponseProto;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.GetSubClustersInfoResponseProtoOrBuilder;
+import org.apache.hadoop.yarn.federation.proto.YarnServerFederationProtos.SubClusterInfoProto;
+import org.apache.hadoop.yarn.server.federation.store.records.GetSubClustersInfoResponse;
+import org.apache.hadoop.yarn.server.federation.store.records.SubClusterInfo;
+
+import com.google.protobuf.TextFormat;
+
+/**
+ * Protocol buffer based implementation of {@link GetSubClustersInfoResponse}.
+ */
+@Private
+@Unstable
+public class GetSubClustersInfoResponsePBImpl
+    extends GetSubClustersInfoResponse {
+
+  private GetSubClustersInfoResponseProto proto =
+      GetSubClustersInfoResponseProto.getDefaultInstance();
+  private GetSubClustersInfoResponseProto.Builder builder = null;
+  private boolean viaProto = false;
+
+  private List<SubClusterInfo> subClusterInfos;
+
+  public GetSubClustersInfoResponsePBImpl() {
+    builder = GetSubClustersInfoResponseProto.newBuilder();
+  }
+
+  public GetSubClustersInfoResponsePBImpl(
+      GetSubClustersInfoResponseProto proto) {
+    this.proto = proto;
+    viaProto = true;
+  }
+
+  public GetSubClustersInfoResponseProto getProto() {
+    mergeLocalToProto();
+    proto = viaProto ? proto : builder.build();
+    viaProto = true;
+    return proto;
+  }
+
+  private void mergeLocalToBuilder() {
+    if (this.subClusterInfos != null) {
+      addReservationResourcesToProto();
+    }
+  }
+
+  private void mergeLocalToProto() {
+    if (viaProto) {
+      maybeInitBuilder();
+    }
+    mergeLocalToBuilder();
+    proto = builder.build();
+    viaProto = true;
+  }
+
+  private void maybeInitBuilder() {
+    if (viaProto || builder == null) {
+      builder = GetSubClustersInfoResponseProto.newBuilder(proto);
+    }
+    viaProto = false;
+  }
+
+  @Override
+  public List<SubClusterInfo> getSubClusters() {
+    initSubClustersInfoList();
+    return subClusterInfos;
+  }
+
+  @Override
+  public void setSubClusters(List<SubClusterInfo> subClusters) {
+    if (subClusters == null) {
+      builder.clearSubClusterInfos();
+      return;
+    }
+    this.subClusterInfos = subClusters;
+  }
+
+  private void initSubClustersInfoList() {
+    if (this.subClusterInfos != null) {
+      return;
+    }
+    GetSubClustersInfoResponseProtoOrBuilder p = viaProto ? proto : builder;
+    List<SubClusterInfoProto> subClusterInfosList = p.getSubClusterInfosList();
+    subClusterInfos = new ArrayList<SubClusterInfo>();
+
+    for (SubClusterInfoProto r : subClusterInfosList) {
+      subClusterInfos.add(convertFromProtoFormat(r));
+    }
+  }
+
+  private void addReservationResourcesToProto() {
+    maybeInitBuilder();
+    builder.clearSubClusterInfos();
+    if (subClusterInfos == null) {
+      return;
+    }
+    Iterable<SubClusterInfoProto> iterable =
+        new Iterable<SubClusterInfoProto>() {
+          @Override
+          public Iterator<SubClusterInfoProto> iterator() {
+            return new Iterator<SubClusterInfoProto>() {
+
+              private Iterator<SubClusterInfo> iter =
+                  subClusterInfos.iterator();
+
+              @Override
+              public boolean hasNext() {
+                return iter.hasNext();
+              }
+
+              @Override
+              public SubClusterInfoProto next() {
+                return convertToProtoFormat(iter.next());
+              }
+
+              @Override
+              public void remove() {
+                throw new UnsupportedOperationException();
+              }
+
+            };
+
+          }
+
+        };
+    builder.addAllSubClusterInfos(iterable);
+  }
+
+  private SubClusterInfoProto convertToProtoFormat(SubClusterInfo r) {
+    return ((SubClusterInfoPBImpl) r).getProto();
+  }
+
+  private SubClusterInfoPBImpl convertFromProtoFormat(SubClusterInfoProto r) {
+    return new SubClusterInfoPBImpl(r);
+  }
+
+  @Override
+  public int hashCode() {
+    return getProto().hashCode();
+  }
+
+  @Override
+  public boolean equals(Object other) {
+    if (other == null) {
+      return false;
+    }
+    if (other.getClass().isAssignableFrom(this.getClass())) {
+      return this.getProto().equals(this.getClass().cast(other).getProto());
+    }
+    return false;
+  }
+
+  @Override
+  public String toString() {
+    return TextFormat.shortDebugString(getProto());
+  }
+
+}


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