You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by en...@apache.org on 2016/03/15 02:29:19 UTC

[7/7] hbase git commit: HBASE-6721 RegionServer Group based Assignment (Francis Liu)

HBASE-6721 RegionServer Group based Assignment (Francis Liu)


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

Branch: refs/heads/master
Commit: ca816f0780f6e5a117b85810cf35f3b29c964ddc
Parents: 122e6f5
Author: Enis Soztutar <en...@apache.org>
Authored: Mon Mar 14 18:28:50 2016 -0700
Committer: Enis Soztutar <en...@apache.org>
Committed: Mon Mar 14 18:28:50 2016 -0700

----------------------------------------------------------------------
 .../hadoop/hbase/protobuf/ProtobufUtil.java     |    69 +-
 .../org/apache/hadoop/hbase/ServerName.java     |    24 +-
 .../hadoop/hbase/rsgroup/RSGroupInfo.java       |   187 +
 hbase-it/pom.xml                                |    10 +
 .../hbase/rsgroup/IntegrationTestRSGroup.java   |    99 +
 hbase-protocol/pom.xml                          |     2 +
 .../protobuf/generated/RSGroupAdminProtos.java  | 11855 +++++++++++++++++
 .../hbase/protobuf/generated/RSGroupProtos.java |  1331 ++
 hbase-protocol/src/main/protobuf/RSGroup.proto  |    34 +
 .../src/main/protobuf/RSGroupAdmin.proto        |   136 +
 hbase-rsgroup/pom.xml                           |   346 +
 .../hadoop/hbase/rsgroup/RSGroupAdmin.java      |   121 +
 .../hbase/rsgroup/RSGroupAdminClient.java       |   204 +
 .../hbase/rsgroup/RSGroupAdminEndpoint.java     |   965 ++
 .../hbase/rsgroup/RSGroupAdminServer.java       |   501 +
 .../hbase/rsgroup/RSGroupBasedLoadBalancer.java |   422 +
 .../hbase/rsgroup/RSGroupInfoManager.java       |   132 +
 .../hbase/rsgroup/RSGroupInfoManagerImpl.java   |   755 ++
 .../hadoop/hbase/rsgroup/RSGroupSerDe.java      |    88 +
 .../hbase/rsgroup/RSGroupableBalancer.java      |    29 +
 .../balancer/TestRSGroupBasedLoadBalancer.java  |   574 +
 .../hadoop/hbase/rsgroup/TestRSGroups.java      |   287 +
 .../hadoop/hbase/rsgroup/TestRSGroupsBase.java  |   643 +
 .../hbase/rsgroup/TestRSGroupsOfflineMode.java  |   187 +
 .../rsgroup/VerifyingRSGroupAdminClient.java    |   149 +
 .../hbase/tmpl/master/MasterStatusTmpl.jamon    |    34 +-
 .../BaseMasterAndRegionObserver.java            |    53 +
 .../hbase/coprocessor/BaseMasterObserver.java   |    54 +
 .../hbase/coprocessor/MasterObserver.java       |    98 +
 .../hadoop/hbase/master/AssignmentManager.java  |    25 +-
 .../org/apache/hadoop/hbase/master/HMaster.java |    32 +-
 .../hadoop/hbase/master/LoadBalancer.java       |     3 +
 .../hbase/master/MasterCoprocessorHost.java     |   137 +
 .../hadoop/hbase/master/MasterRpcServices.java  |     8 +
 .../hadoop/hbase/master/MasterServices.java     |     5 +
 .../hbase/security/access/AccessController.java |    32 +
 .../hbase/coprocessor/TestMasterObserver.java   |    53 +
 .../master/TestAssignmentManagerOnCluster.java  |   127 +-
 .../hadoop/hbase/master/TestCatalogJanitor.java |     3 +
 .../hbase/master/TestMasterStatusServlet.java   |    12 +-
 .../normalizer/TestSimpleRegionNormalizer.java  |     2 +-
 .../security/access/TestAccessController.java   |    75 +
 hbase-shell/pom.xml                             |    35 +
 hbase-shell/src/main/ruby/hbase.rb              |     1 +
 hbase-shell/src/main/ruby/hbase/hbase.rb        |     4 +
 .../src/main/ruby/hbase/rsgroup_admin.rb        |   150 +
 hbase-shell/src/main/ruby/shell.rb              |    21 +
 hbase-shell/src/main/ruby/shell/commands.rb     |     4 +
 .../src/main/ruby/shell/commands/add_rsgroup.rb |    39 +
 .../main/ruby/shell/commands/balance_rsgroup.rb |    37 +
 .../src/main/ruby/shell/commands/get_rsgroup.rb |    44 +
 .../ruby/shell/commands/get_server_rsgroup.rb   |    40 +
 .../ruby/shell/commands/get_table_rsgroup.rb    |    41 +
 .../main/ruby/shell/commands/list_rsgroups.rb   |    50 +
 .../ruby/shell/commands/move_rsgroup_servers.rb |    37 +
 .../ruby/shell/commands/move_rsgroup_tables.rb  |    37 +
 .../main/ruby/shell/commands/remove_rsgroup.rb  |    37 +
 .../apache/hadoop/hbase/client/TestShell.java   |     2 +-
 .../hbase/client/rsgroup/TestShellRSGroups.java |   111 +
 .../src/test/ruby/shell/rsgroup_shell_test.rb   |    96 +
 hbase-shell/src/test/ruby/test_helper.rb        |     4 +
 pom.xml                                         |    23 +
 62 files changed, 20652 insertions(+), 64 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-client/src/main/java/org/apache/hadoop/hbase/protobuf/ProtobufUtil.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/protobuf/ProtobufUtil.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/protobuf/ProtobufUtil.java
index b052e63..9c71d97 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/protobuf/ProtobufUtil.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/protobuf/ProtobufUtil.java
@@ -18,7 +18,19 @@
 package org.apache.hadoop.hbase.protobuf;
 
 
-import static org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionSpecifier.RegionSpecifierType.REGION_NAME;
+import com.google.common.collect.ArrayListMultimap;
+import com.google.common.collect.ListMultimap;
+import com.google.common.collect.Lists;
+import com.google.common.net.HostAndPort;
+import com.google.protobuf.ByteString;
+import com.google.protobuf.CodedInputStream;
+import com.google.protobuf.InvalidProtocolBufferException;
+import com.google.protobuf.Message;
+import com.google.protobuf.Parser;
+import com.google.protobuf.RpcChannel;
+import com.google.protobuf.Service;
+import com.google.protobuf.ServiceException;
+import com.google.protobuf.TextFormat;
 
 import java.io.ByteArrayOutputStream;
 import java.io.IOException;
@@ -124,6 +136,7 @@ import org.apache.hadoop.hbase.protobuf.generated.MasterProtos.CreateTableReques
 import org.apache.hadoop.hbase.protobuf.generated.MasterProtos.GetTableDescriptorsResponse;
 import org.apache.hadoop.hbase.protobuf.generated.MasterProtos.MasterService;
 import org.apache.hadoop.hbase.protobuf.generated.QuotaProtos;
+import org.apache.hadoop.hbase.protobuf.generated.RSGroupProtos;
 import org.apache.hadoop.hbase.protobuf.generated.RegionServerStatusProtos.RegionServerReportRequest;
 import org.apache.hadoop.hbase.protobuf.generated.RegionServerStatusProtos.RegionServerStartupRequest;
 import org.apache.hadoop.hbase.protobuf.generated.WALProtos;
@@ -139,6 +152,7 @@ import org.apache.hadoop.hbase.quotas.QuotaType;
 import org.apache.hadoop.hbase.quotas.ThrottleType;
 import org.apache.hadoop.hbase.replication.ReplicationLoadSink;
 import org.apache.hadoop.hbase.replication.ReplicationLoadSource;
+import org.apache.hadoop.hbase.rsgroup.RSGroupInfo;
 import org.apache.hadoop.hbase.security.User;
 import org.apache.hadoop.hbase.security.access.Permission;
 import org.apache.hadoop.hbase.security.access.TablePermission;
@@ -157,18 +171,8 @@ import org.apache.hadoop.io.Text;
 import org.apache.hadoop.ipc.RemoteException;
 import org.apache.hadoop.security.token.Token;
 
-import com.google.common.collect.ArrayListMultimap;
-import com.google.common.collect.ListMultimap;
-import com.google.common.collect.Lists;
-import com.google.protobuf.ByteString;
-import com.google.protobuf.CodedInputStream;
-import com.google.protobuf.InvalidProtocolBufferException;
-import com.google.protobuf.Message;
-import com.google.protobuf.Parser;
-import com.google.protobuf.RpcChannel;
-import com.google.protobuf.Service;
-import com.google.protobuf.ServiceException;
-import com.google.protobuf.TextFormat;
+import static org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionSpecifier
+    .RegionSpecifierType.REGION_NAME;
 
 /**
  * Protobufs utility.
@@ -3057,7 +3061,7 @@ public final class ProtobufUtil {
    * @param builder current message builder
    * @param in InputStream containing protobuf data
    * @param size known size of protobuf data
-   * @throws IOException 
+   * @throws IOException
    */
   public static void mergeFrom(Message.Builder builder, InputStream in, int size)
       throws IOException {
@@ -3072,7 +3076,7 @@ public final class ProtobufUtil {
    * buffers where the message size is not known
    * @param builder current message builder
    * @param in InputStream containing protobuf data
-   * @throws IOException 
+   * @throws IOException
    */
   public static void mergeFrom(Message.Builder builder, InputStream in)
       throws IOException {
@@ -3086,8 +3090,8 @@ public final class ProtobufUtil {
    * This version of protobuf's mergeFrom avoids the hard-coded 64MB limit for decoding
    * buffers when working with ByteStrings
    * @param builder current message builder
-   * @param bs ByteString containing the 
-   * @throws IOException 
+   * @param bs ByteString containing the
+   * @throws IOException
    */
   public static void mergeFrom(Message.Builder builder, ByteString bs) throws IOException {
     final CodedInputStream codedInput = bs.newCodedInput();
@@ -3101,7 +3105,7 @@ public final class ProtobufUtil {
    * buffers when working with byte arrays
    * @param builder current message builder
    * @param b byte array
-   * @throws IOException 
+   * @throws IOException
    */
   public static void mergeFrom(Message.Builder builder, byte[] b) throws IOException {
     final CodedInputStream codedInput = CodedInputStream.newInstance(b);
@@ -3221,4 +3225,33 @@ public final class ProtobufUtil {
     return new TimeRange(minStamp, maxStamp);
   }
 
+  public static RSGroupInfo toGroupInfo(RSGroupProtos.RSGroupInfo proto) {
+    RSGroupInfo RSGroupInfo = new RSGroupInfo(proto.getName());
+    for(HBaseProtos.ServerName el: proto.getServersList()) {
+      RSGroupInfo.addServer(HostAndPort.fromParts(el.getHostName(), el.getPort()));
+    }
+    for(HBaseProtos.TableName pTableName: proto.getTablesList()) {
+      RSGroupInfo.addTable(ProtobufUtil.toTableName(pTableName));
+    }
+    return RSGroupInfo;
+  }
+
+  public static RSGroupProtos.RSGroupInfo toProtoGroupInfo(RSGroupInfo pojo) {
+    List<HBaseProtos.TableName> tables =
+        new ArrayList<HBaseProtos.TableName>(pojo.getTables().size());
+    for(TableName arg: pojo.getTables()) {
+      tables.add(ProtobufUtil.toProtoTableName(arg));
+    }
+    List<HBaseProtos.ServerName> hostports =
+        new ArrayList<HBaseProtos.ServerName>(pojo.getServers().size());
+    for(HostAndPort el: pojo.getServers()) {
+      hostports.add(HBaseProtos.ServerName.newBuilder()
+          .setHostName(el.getHostText())
+          .setPort(el.getPort())
+          .build());
+    }
+    return RSGroupProtos.RSGroupInfo.newBuilder().setName(pojo.getName())
+        .addAllServers(hostports)
+        .addAllTables(tables).build();
+  }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-common/src/main/java/org/apache/hadoop/hbase/ServerName.java
----------------------------------------------------------------------
diff --git a/hbase-common/src/main/java/org/apache/hadoop/hbase/ServerName.java b/hbase-common/src/main/java/org/apache/hadoop/hbase/ServerName.java
index 059dcb8..c0c43ed 100644
--- a/hbase-common/src/main/java/org/apache/hadoop/hbase/ServerName.java
+++ b/hbase-common/src/main/java/org/apache/hadoop/hbase/ServerName.java
@@ -18,6 +18,10 @@
  */
 package org.apache.hadoop.hbase;
 
+import com.google.common.net.HostAndPort;
+import com.google.common.net.InetAddresses;
+import com.google.protobuf.InvalidProtocolBufferException;
+
 import java.io.Serializable;
 import java.util.ArrayList;
 import java.util.List;
@@ -31,9 +35,6 @@ import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
 import org.apache.hadoop.hbase.util.Addressing;
 import org.apache.hadoop.hbase.util.Bytes;
 
-import com.google.common.net.InetAddresses;
-import com.google.protobuf.InvalidProtocolBufferException;
-
 /**
  * Instance of an HBase ServerName.
  * A server name is used uniquely identifying a server instance in a cluster and is made
@@ -54,7 +55,7 @@ import com.google.protobuf.InvalidProtocolBufferException;
  */
 @InterfaceAudience.Public
 @InterfaceStability.Evolving
-public class ServerName implements Comparable<ServerName>, Serializable {
+  public class ServerName implements Comparable<ServerName>, Serializable {
   private static final long serialVersionUID = 1367463982557264981L;
 
   /**
@@ -91,6 +92,7 @@ public class ServerName implements Comparable<ServerName>, Serializable {
   private final String hostnameOnly;
   private final int port;
   private final long startcode;
+  private transient HostAndPort hostAndPort;
 
   /**
    * Cached versioned bytes of this ServerName instance.
@@ -105,7 +107,7 @@ public class ServerName implements Comparable<ServerName>, Serializable {
     this.hostnameOnly = hostname;
     this.port = port;
     this.startcode = startcode;
-    this.servername = getServerName(this.hostnameOnly, port, startcode);
+    this.servername = getServerName(hostname, port, startcode);
   }
 
   /**
@@ -189,7 +191,8 @@ public class ServerName implements Comparable<ServerName>, Serializable {
    * in compares, etc.
    */
   public String toShortString() {
-    return Addressing.createHostAndPortStr(getHostNameMinusDomain(this.hostnameOnly), this.port);
+    return Addressing.createHostAndPortStr(
+        getHostNameMinusDomain(hostnameOnly), port);
   }
 
   /**
@@ -256,7 +259,14 @@ public class ServerName implements Comparable<ServerName>, Serializable {
    * {@link Addressing#createHostAndPortStr(String, int)}
    */
   public String getHostAndPort() {
-    return Addressing.createHostAndPortStr(this.hostnameOnly, this.port);
+    return Addressing.createHostAndPortStr(hostnameOnly, port);
+  }
+
+  public HostAndPort getHostPort() {
+    if (hostAndPort == null) {
+      hostAndPort = HostAndPort.fromParts(hostnameOnly, port);
+    }
+    return hostAndPort;
   }
 
   /**

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-common/src/main/java/org/apache/hadoop/hbase/rsgroup/RSGroupInfo.java
----------------------------------------------------------------------
diff --git a/hbase-common/src/main/java/org/apache/hadoop/hbase/rsgroup/RSGroupInfo.java b/hbase-common/src/main/java/org/apache/hadoop/hbase/rsgroup/RSGroupInfo.java
new file mode 100644
index 0000000..0fb02d8
--- /dev/null
+++ b/hbase-common/src/main/java/org/apache/hadoop/hbase/rsgroup/RSGroupInfo.java
@@ -0,0 +1,187 @@
+/**
+ * Copyright The Apache Software Foundation
+ *
+ * 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.hbase.rsgroup;
+
+import com.google.common.collect.Sets;
+import com.google.common.net.HostAndPort;
+
+import java.util.Collection;
+import java.util.NavigableSet;
+import java.util.Set;
+
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.classification.InterfaceAudience;
+import org.apache.hadoop.hbase.classification.InterfaceStability;
+
+/**
+ * Stores the group information of region server groups.
+ */
+@InterfaceAudience.Public
+@InterfaceStability.Evolving
+public class RSGroupInfo {
+
+  public static final String DEFAULT_GROUP = "default";
+  public static final String NAMESPACEDESC_PROP_GROUP = "hbase.rsgroup.name";
+
+  private String name;
+  private Set<HostAndPort> servers;
+  private NavigableSet<TableName> tables;
+
+  public RSGroupInfo(String name) {
+    this(name, Sets.<HostAndPort>newHashSet(), Sets.<TableName>newTreeSet());
+  }
+
+  RSGroupInfo(String name,
+              Set<HostAndPort> servers,
+              NavigableSet<TableName> tables) {
+    this.name = name;
+    this.servers = servers;
+    this.tables = tables;
+  }
+
+  public RSGroupInfo(RSGroupInfo src) {
+    name = src.getName();
+    servers = Sets.newHashSet(src.getServers());
+    tables = Sets.newTreeSet(src.getTables());
+  }
+
+  /**
+   * Get group name.
+   *
+   * @return group name
+   */
+  public String getName() {
+    return name;
+  }
+
+  /**
+   * Adds the server to the group.
+   *
+   * @param hostPort the server
+   */
+  public void addServer(HostAndPort hostPort){
+    servers.add(hostPort);
+  }
+
+  /**
+   * Adds a group of servers.
+   *
+   * @param hostPort the servers
+   */
+  public void addAllServers(Collection<HostAndPort> hostPort){
+    servers.addAll(hostPort);
+  }
+
+  /**
+   * @param hostPort hostPort of the server
+   * @return true, if a server with hostPort is found
+   */
+  public boolean containsServer(HostAndPort hostPort) {
+    return servers.contains(hostPort);
+  }
+
+  /**
+   * Get list of servers.
+   *
+   * @return set of servers
+   */
+  public Set<HostAndPort> getServers() {
+    return servers;
+  }
+
+  /**
+   * Remove a server from this group.
+   *
+   * @param hostPort HostPort of the server to remove
+   */
+  public boolean removeServer(HostAndPort hostPort) {
+    return servers.remove(hostPort);
+  }
+
+  /**
+   * Set of tables that are members of this group
+   * @return set of tables
+   */
+  public NavigableSet<TableName> getTables() {
+    return tables;
+  }
+
+  public void addTable(TableName table) {
+    tables.add(table);
+  }
+
+  public void addAllTables(Collection<TableName> arg) {
+    tables.addAll(arg);
+  }
+
+  public boolean containsTable(TableName table) {
+    return tables.contains(table);
+  }
+
+  public boolean removeTable(TableName table) {
+    return tables.remove(table);
+  }
+
+  @Override
+  public String toString() {
+    StringBuffer sb = new StringBuffer();
+    sb.append("Name:");
+    sb.append(this.name);
+    sb.append(", ");
+    sb.append(" Servers:");
+    sb.append(this.servers);
+    return sb.toString();
+
+  }
+
+  @Override
+  public boolean equals(Object o) {
+    if (this == o) {
+      return true;
+    }
+    if (o == null || getClass() != o.getClass()) {
+      return false;
+    }
+
+    RSGroupInfo RSGroupInfo = (RSGroupInfo) o;
+
+    if (!name.equals(RSGroupInfo.name)) {
+      return false;
+    }
+    if (!servers.equals(RSGroupInfo.servers)) {
+      return false;
+    }
+    if (!tables.equals(RSGroupInfo.tables)) {
+      return false;
+    }
+
+    return true;
+  }
+
+  @Override
+  public int hashCode() {
+    int result = servers.hashCode();
+    result = 31 * result + tables.hashCode();
+    result = 31 * result + name.hashCode();
+    return result;
+  }
+
+}

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-it/pom.xml
----------------------------------------------------------------------
diff --git a/hbase-it/pom.xml b/hbase-it/pom.xml
index 92243f0..70d13ad 100644
--- a/hbase-it/pom.xml
+++ b/hbase-it/pom.xml
@@ -209,6 +209,16 @@
     </dependency>
     <dependency>
       <groupId>org.apache.hbase</groupId>
+      <artifactId>hbase-rsgroup</artifactId>
+    </dependency>
+    <dependency>
+      <groupId>org.apache.hbase</groupId>
+      <artifactId>hbase-rsgroup</artifactId>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+    <dependency>
+      <groupId>org.apache.hbase</groupId>
       <artifactId>hbase-server</artifactId>
     </dependency>
     <dependency>

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-it/src/test/java/org/apache/hadoop/hbase/rsgroup/IntegrationTestRSGroup.java
----------------------------------------------------------------------
diff --git a/hbase-it/src/test/java/org/apache/hadoop/hbase/rsgroup/IntegrationTestRSGroup.java b/hbase-it/src/test/java/org/apache/hadoop/hbase/rsgroup/IntegrationTestRSGroup.java
new file mode 100644
index 0000000..fe9c58c2
--- /dev/null
+++ b/hbase-it/src/test/java/org/apache/hadoop/hbase/rsgroup/IntegrationTestRSGroup.java
@@ -0,0 +1,99 @@
+/**
+ * Copyright The Apache Software Foundation
+ *
+ * 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.hbase.rsgroup;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.hbase.IntegrationTestingUtility;
+import org.apache.hadoop.hbase.Waiter;
+import org.apache.hadoop.hbase.testclassification.IntegrationTests;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.experimental.categories.Category;
+
+/**
+ * Runs all of the units tests defined in TestGroupBase
+ * as an integration test.
+ * Requires TestRSGroupBase.NUM_SLAVE_BASE servers to run.
+ */
+@Category(IntegrationTests.class)
+public class IntegrationTestRSGroup extends TestRSGroupsBase {
+  //Integration specific
+  private final static Log LOG = LogFactory.getLog(IntegrationTestRSGroup.class);
+  private static boolean initialized = false;
+
+  @Before
+  public void beforeMethod() throws Exception {
+    if(!initialized) {
+      LOG.info("Setting up IntegrationTestGroup");
+      LOG.info("Initializing cluster with " + NUM_SLAVES_BASE + " servers");
+      TEST_UTIL = new IntegrationTestingUtility();
+      ((IntegrationTestingUtility)TEST_UTIL).initializeCluster(NUM_SLAVES_BASE);
+      //set shared configs
+      admin = TEST_UTIL.getHBaseAdmin();
+      cluster = TEST_UTIL.getHBaseClusterInterface();
+      rsGroupAdmin = new VerifyingRSGroupAdminClient(rsGroupAdmin.newClient(TEST_UTIL.getConnection()),
+          TEST_UTIL.getConfiguration());
+      LOG.info("Done initializing cluster");
+      initialized = true;
+      //cluster may not be clean
+      //cleanup when initializing
+      afterMethod();
+    }
+  }
+
+  @After
+  public void afterMethod() throws Exception {
+    LOG.info("Cleaning up previous test run");
+    //cleanup previous artifacts
+    deleteTableIfNecessary();
+    deleteNamespaceIfNecessary();
+    deleteGroups();
+    admin.setBalancerRunning(true, true);
+
+    LOG.info("Restoring the cluster");
+    ((IntegrationTestingUtility)TEST_UTIL).restoreCluster();
+    LOG.info("Done restoring the cluster");
+
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        LOG.info("Waiting for cleanup to finish "+ rsGroupAdmin.listRSGroups());
+        //Might be greater since moving servers back to default
+        //is after starting a server
+        return rsGroupAdmin.getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP).getServers().size()
+            >= NUM_SLAVES_BASE;
+      }
+    });
+
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        LOG.info("Waiting for regionservers to be registered "+ rsGroupAdmin.listRSGroups());
+        //Might be greater since moving servers back to default
+        //is after starting a server
+        return rsGroupAdmin.getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP).getServers().size()
+            == getNumServers();
+      }
+    });
+
+    LOG.info("Done cleaning up previous test run");
+  }
+}

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-protocol/pom.xml
----------------------------------------------------------------------
diff --git a/hbase-protocol/pom.xml b/hbase-protocol/pom.xml
index 8034576..d43b7b9 100644
--- a/hbase-protocol/pom.xml
+++ b/hbase-protocol/pom.xml
@@ -193,6 +193,8 @@
                           <include>RegionServerStatus.proto</include>
                           <include>RowProcessor.proto</include>
                           <include>RPC.proto</include>
+                          <include>RSGroup.proto</include>
+                          <include>RSGroupAdmin.proto</include>
                           <include>SecureBulkLoad.proto</include>
                           <include>Tracing.proto</include>
                           <include>VisibilityLabels.proto</include>