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:14 UTC

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

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsBase.java
----------------------------------------------------------------------
diff --git a/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsBase.java b/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsBase.java
new file mode 100644
index 0000000..9225e09
--- /dev/null
+++ b/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsBase.java
@@ -0,0 +1,643 @@
+/**
+ * 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.Maps;
+import com.google.common.collect.Sets;
+import com.google.common.net.HostAndPort;
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.hbase.ClusterStatus;
+import org.apache.hadoop.hbase.HBaseCluster;
+import org.apache.hadoop.hbase.HBaseTestingUtility;
+import org.apache.hadoop.hbase.HColumnDescriptor;
+import org.apache.hadoop.hbase.HRegionInfo;
+import org.apache.hadoop.hbase.HTableDescriptor;
+import org.apache.hadoop.hbase.NamespaceDescriptor;
+import org.apache.hadoop.hbase.RegionLoad;
+import org.apache.hadoop.hbase.ServerName;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.Waiter;
+import org.apache.hadoop.hbase.client.HBaseAdmin;
+import org.apache.hadoop.hbase.constraint.ConstraintException;
+import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
+import org.apache.hadoop.hbase.protobuf.generated.AdminProtos;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.junit.Assert;
+import org.junit.Test;
+
+import java.io.IOException;
+import java.security.SecureRandom;
+import java.util.HashSet;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.TreeMap;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertNull;
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.fail;
+
+public abstract class TestRSGroupsBase {
+  protected static final Log LOG = LogFactory.getLog(TestRSGroupsBase.class);
+
+  //shared
+  protected final static String groupPrefix = "Group";
+  protected final static String tablePrefix = "Group";
+  protected final static SecureRandom rand = new SecureRandom();
+
+  //shared, cluster type specific
+  protected static HBaseTestingUtility TEST_UTIL;
+  protected static HBaseAdmin admin;
+  protected static HBaseCluster cluster;
+  protected static RSGroupAdmin rsGroupAdmin;
+
+  public final static long WAIT_TIMEOUT = 60000*5;
+  public final static int NUM_SLAVES_BASE = 4; //number of slaves for the smallest cluster
+
+
+
+  protected RSGroupInfo addGroup(RSGroupAdmin gAdmin, String groupName,
+                                 int serverCount) throws IOException, InterruptedException {
+    RSGroupInfo defaultInfo = gAdmin
+        .getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP);
+    assertTrue(defaultInfo != null);
+    assertTrue(defaultInfo.getServers().size() >= serverCount);
+    gAdmin.addRSGroup(groupName);
+
+    Set<HostAndPort> set = new HashSet<HostAndPort>();
+    for(HostAndPort server: defaultInfo.getServers()) {
+      if(set.size() == serverCount) {
+        break;
+      }
+      set.add(server);
+    }
+    gAdmin.moveServers(set, groupName);
+    RSGroupInfo result = gAdmin.getRSGroupInfo(groupName);
+    assertTrue(result.getServers().size() >= serverCount);
+    return result;
+  }
+
+  static void removeGroup(RSGroupAdminClient groupAdmin, String groupName) throws IOException {
+    RSGroupInfo RSGroupInfo = groupAdmin.getRSGroupInfo(groupName);
+    groupAdmin.moveTables(RSGroupInfo.getTables(), RSGroupInfo.DEFAULT_GROUP);
+    groupAdmin.moveServers(RSGroupInfo.getServers(), RSGroupInfo.DEFAULT_GROUP);
+    groupAdmin.removeRSGroup(groupName);
+  }
+
+  protected void deleteTableIfNecessary() throws IOException {
+    for (HTableDescriptor desc : TEST_UTIL.getHBaseAdmin().listTables(tablePrefix+".*")) {
+      TEST_UTIL.deleteTable(desc.getTableName());
+    }
+  }
+
+  protected void deleteNamespaceIfNecessary() throws IOException {
+    for (NamespaceDescriptor desc : TEST_UTIL.getHBaseAdmin().listNamespaceDescriptors()) {
+      if(desc.getName().startsWith(tablePrefix)) {
+        admin.deleteNamespace(desc.getName());
+      }
+    }
+  }
+
+  protected void deleteGroups() throws IOException {
+    RSGroupAdmin groupAdmin = rsGroupAdmin.newClient(TEST_UTIL.getConnection());
+    for(RSGroupInfo group: groupAdmin.listRSGroups()) {
+      if(!group.getName().equals(RSGroupInfo.DEFAULT_GROUP)) {
+        groupAdmin.moveTables(group.getTables(), RSGroupInfo.DEFAULT_GROUP);
+        groupAdmin.moveServers(group.getServers(), RSGroupInfo.DEFAULT_GROUP);
+        groupAdmin.removeRSGroup(group.getName());
+      }
+    }
+  }
+
+  public Map<TableName, List<String>> getTableRegionMap() throws IOException {
+    Map<TableName, List<String>> map = Maps.newTreeMap();
+    Map<TableName, Map<ServerName, List<String>>> tableServerRegionMap
+        = getTableServerRegionMap();
+    for(TableName tableName : tableServerRegionMap.keySet()) {
+      if(!map.containsKey(tableName)) {
+        map.put(tableName, new LinkedList<String>());
+      }
+      for(List<String> subset: tableServerRegionMap.get(tableName).values()) {
+        map.get(tableName).addAll(subset);
+      }
+    }
+    return map;
+  }
+
+  public Map<TableName, Map<ServerName, List<String>>> getTableServerRegionMap()
+      throws IOException {
+    Map<TableName, Map<ServerName, List<String>>> map = Maps.newTreeMap();
+    ClusterStatus status = TEST_UTIL.getHBaseClusterInterface().getClusterStatus();
+    for(ServerName serverName : status.getServers()) {
+      for(RegionLoad rl : status.getLoad(serverName).getRegionsLoad().values()) {
+        TableName tableName = HRegionInfo.getTable(rl.getName());
+        if(!map.containsKey(tableName)) {
+          map.put(tableName, new TreeMap<ServerName, List<String>>());
+        }
+        if(!map.get(tableName).containsKey(serverName)) {
+          map.get(tableName).put(serverName, new LinkedList<String>());
+        }
+        map.get(tableName).get(serverName).add(rl.getNameAsString());
+      }
+    }
+    return map;
+  }
+
+  @Test
+  public void testBogusArgs() throws Exception {
+    assertNull(rsGroupAdmin.getRSGroupInfoOfTable(TableName.valueOf("nonexistent")));
+    assertNull(rsGroupAdmin.getRSGroupOfServer(HostAndPort.fromParts("bogus",123)));
+    assertNull(rsGroupAdmin.getRSGroupInfo("bogus"));
+
+    try {
+      rsGroupAdmin.removeRSGroup("bogus");
+      fail("Expected removing bogus group to fail");
+    } catch(ConstraintException ex) {
+      //expected
+    }
+
+    try {
+      rsGroupAdmin.moveTables(Sets.newHashSet(TableName.valueOf("bogustable")), "bogus");
+      fail("Expected move with bogus group to fail");
+    } catch(ConstraintException ex) {
+      //expected
+    }
+
+    try {
+      rsGroupAdmin.moveServers(Sets.newHashSet(HostAndPort.fromParts("bogus",123)), "bogus");
+      fail("Expected move with bogus group to fail");
+    } catch(ConstraintException ex) {
+      //expected
+    }
+
+    try {
+      rsGroupAdmin.balanceRSGroup("bogus");
+      fail("Expected move with bogus group to fail");
+    } catch(ConstraintException ex) {
+      //expected
+    }
+  }
+
+  @Test
+  public void testCreateMultiRegion() throws IOException {
+    LOG.info("testCreateMultiRegion");
+    TableName tableName = TableName.valueOf(tablePrefix + "_testCreateMultiRegion");
+    byte[] end = {1,3,5,7,9};
+    byte[] start = {0,2,4,6,8};
+    byte[][] f = {Bytes.toBytes("f")};
+    TEST_UTIL.createTable(tableName, f,1,start,end,10);
+  }
+
+  @Test
+  public void testCreateAndDrop() throws Exception {
+    LOG.info("testCreateAndDrop");
+
+    final TableName tableName = TableName.valueOf(tablePrefix + "_testCreateAndDrop");
+    TEST_UTIL.createTable(tableName, Bytes.toBytes("cf"));
+    //wait for created table to be assigned
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return getTableRegionMap().get(tableName) != null;
+      }
+    });
+    TEST_UTIL.deleteTable(tableName);
+  }
+
+
+  @Test
+  public void testSimpleRegionServerMove() throws IOException,
+      InterruptedException {
+    LOG.info("testSimpleRegionServerMove");
+
+    int initNumGroups = rsGroupAdmin.listRSGroups().size();
+    RSGroupInfo appInfo = addGroup(rsGroupAdmin, getGroupName("testSimpleRegionServerMove"), 1);
+    RSGroupInfo adminInfo = addGroup(rsGroupAdmin, getGroupName("testSimpleRegionServerMove"), 1);
+    RSGroupInfo dInfo = rsGroupAdmin.getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP);
+    Assert.assertEquals(initNumGroups + 2, rsGroupAdmin.listRSGroups().size());
+    assertEquals(1, adminInfo.getServers().size());
+    assertEquals(1, appInfo.getServers().size());
+    assertEquals(getNumServers() - 2, dInfo.getServers().size());
+    rsGroupAdmin.moveServers(appInfo.getServers(),
+        RSGroupInfo.DEFAULT_GROUP);
+    rsGroupAdmin.removeRSGroup(appInfo.getName());
+    rsGroupAdmin.moveServers(adminInfo.getServers(),
+        RSGroupInfo.DEFAULT_GROUP);
+    rsGroupAdmin.removeRSGroup(adminInfo.getName());
+    Assert.assertEquals(rsGroupAdmin.listRSGroups().size(), initNumGroups);
+  }
+
+  // return the real number of region servers, excluding the master embedded region server in 2.0+
+  public int getNumServers() throws IOException {
+    ClusterStatus status = admin.getClusterStatus();
+    ServerName master = status.getMaster();
+    int count = 0;
+    for (ServerName sn : status.getServers()) {
+      if (!sn.equals(master)) {
+        count++;
+      }
+    }
+    return count;
+  }
+
+  @Test
+  public void testMoveServers() throws Exception {
+    LOG.info("testMoveServers");
+
+    //create groups and assign servers
+    addGroup(rsGroupAdmin, "bar", 3);
+    rsGroupAdmin.addRSGroup("foo");
+
+    RSGroupInfo barGroup = rsGroupAdmin.getRSGroupInfo("bar");
+    RSGroupInfo fooGroup = rsGroupAdmin.getRSGroupInfo("foo");
+    assertEquals(3, barGroup.getServers().size());
+    assertEquals(0, fooGroup.getServers().size());
+
+    //test fail bogus server move
+    try {
+      rsGroupAdmin.moveServers(Sets.newHashSet(HostAndPort.fromString("foo:9999")),"foo");
+      fail("Bogus servers shouldn't have been successfully moved.");
+    } catch(IOException ex) {
+      String exp = "Server foo:9999 does not have a group.";
+      String msg = "Expected '"+exp+"' in exception message: ";
+      assertTrue(msg+" "+ex.getMessage(), ex.getMessage().contains(exp));
+    }
+
+    //test success case
+    LOG.info("moving servers "+barGroup.getServers()+" to group foo");
+    rsGroupAdmin.moveServers(barGroup.getServers(), fooGroup.getName());
+
+    barGroup = rsGroupAdmin.getRSGroupInfo("bar");
+    fooGroup = rsGroupAdmin.getRSGroupInfo("foo");
+    assertEquals(0,barGroup.getServers().size());
+    assertEquals(3,fooGroup.getServers().size());
+
+    LOG.info("moving servers "+fooGroup.getServers()+" to group default");
+    rsGroupAdmin.moveServers(fooGroup.getServers(), RSGroupInfo.DEFAULT_GROUP);
+
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return getNumServers() ==
+        rsGroupAdmin.getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP).getServers().size();
+      }
+    });
+
+    fooGroup = rsGroupAdmin.getRSGroupInfo("foo");
+    assertEquals(0,fooGroup.getServers().size());
+
+    //test group removal
+    LOG.info("Remove group "+barGroup.getName());
+    rsGroupAdmin.removeRSGroup(barGroup.getName());
+    Assert.assertEquals(null, rsGroupAdmin.getRSGroupInfo(barGroup.getName()));
+    LOG.info("Remove group "+fooGroup.getName());
+    rsGroupAdmin.removeRSGroup(fooGroup.getName());
+    Assert.assertEquals(null, rsGroupAdmin.getRSGroupInfo(fooGroup.getName()));
+  }
+
+  @Test
+  public void testTableMoveTruncateAndDrop() throws Exception {
+    LOG.info("testTableMove");
+
+    final TableName tableName = TableName.valueOf(tablePrefix + "_testTableMoveAndDrop");
+    final byte[] familyNameBytes = Bytes.toBytes("f");
+    String newGroupName = getGroupName("testTableMove");
+    final RSGroupInfo newGroup = addGroup(rsGroupAdmin, newGroupName, 2);
+
+    TEST_UTIL.createMultiRegionTable(tableName, familyNameBytes, 5);
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        List<String> regions = getTableRegionMap().get(tableName);
+        if (regions == null)
+          return false;
+        return getTableRegionMap().get(tableName).size() >= 5;
+      }
+    });
+
+    RSGroupInfo tableGrp = rsGroupAdmin.getRSGroupInfoOfTable(tableName);
+    assertTrue(tableGrp.getName().equals(RSGroupInfo.DEFAULT_GROUP));
+
+    //change table's group
+    LOG.info("Moving table "+tableName+" to "+newGroup.getName());
+    rsGroupAdmin.moveTables(Sets.newHashSet(tableName), newGroup.getName());
+
+    //verify group change
+    Assert.assertEquals(newGroup.getName(),
+        rsGroupAdmin.getRSGroupInfoOfTable(tableName).getName());
+
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        Map<ServerName, List<String>> serverMap = getTableServerRegionMap().get(tableName);
+        int count = 0;
+        if (serverMap != null) {
+          for (ServerName rs : serverMap.keySet()) {
+            if (newGroup.containsServer(rs.getHostPort())) {
+              count += serverMap.get(rs).size();
+            }
+          }
+        }
+        return count == 5;
+      }
+    });
+
+    //test truncate
+    admin.disableTable(tableName);
+    admin.truncateTable(tableName, true);
+    Assert.assertEquals(1, rsGroupAdmin.getRSGroupInfo(newGroup.getName()).getTables().size());
+    Assert.assertEquals(tableName, rsGroupAdmin.getRSGroupInfo(
+        newGroup.getName()).getTables().first());
+
+    //verify removed table is removed from group
+    TEST_UTIL.deleteTable(tableName);
+    Assert.assertEquals(0, rsGroupAdmin.getRSGroupInfo(newGroup.getName()).getTables().size());
+  }
+
+  @Test
+  public void testGroupBalance() throws Exception {
+    LOG.info("testGroupBalance");
+    String newGroupName = getGroupName("testGroupBalance");
+    final RSGroupInfo newGroup = addGroup(rsGroupAdmin, newGroupName, 3);
+
+    final TableName tableName = TableName.valueOf(tablePrefix+"_ns", "testGroupBalance");
+    admin.createNamespace(
+        NamespaceDescriptor.create(tableName.getNamespaceAsString())
+            .addConfiguration(RSGroupInfo.NAMESPACEDESC_PROP_GROUP, newGroupName).build());
+    final byte[] familyNameBytes = Bytes.toBytes("f");
+    final HTableDescriptor desc = new HTableDescriptor(tableName);
+    desc.addFamily(new HColumnDescriptor("f"));
+    byte [] startKey = Bytes.toBytes("aaaaa");
+    byte [] endKey = Bytes.toBytes("zzzzz");
+    admin.createTable(desc, startKey, endKey, 6);
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        List<String> regions = getTableRegionMap().get(tableName);
+        if (regions == null) {
+          return false;
+        }
+        return regions.size() >= 6;
+      }
+    });
+
+    //make assignment uneven, move all regions to one server
+    Map<ServerName,List<String>> assignMap =
+        getTableServerRegionMap().get(tableName);
+    final ServerName first = assignMap.entrySet().iterator().next().getKey();
+    for(HRegionInfo region: admin.getTableRegions(tableName)) {
+      if(!assignMap.get(first).contains(region)) {
+        admin.move(region.getEncodedNameAsBytes(), Bytes.toBytes(first.getServerName()));
+      }
+    }
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        Map<ServerName, List<String>> map = getTableServerRegionMap().get(tableName);
+        if (map == null) {
+          return true;
+        }
+        List<String> regions = map.get(first);
+        if (regions == null) {
+          return true;
+        }
+        return regions.size() >= 6;
+      }
+    });
+
+    //balance the other group and make sure it doesn't affect the new group
+    rsGroupAdmin.balanceRSGroup(RSGroupInfo.DEFAULT_GROUP);
+    assertEquals(6, getTableServerRegionMap().get(tableName).get(first).size());
+
+    rsGroupAdmin.balanceRSGroup(newGroupName);
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        for (List<String> regions : getTableServerRegionMap().get(tableName).values()) {
+          if (2 != regions.size()) {
+            return false;
+          }
+        }
+        return true;
+      }
+    });
+  }
+
+  @Test
+  public void testRegionMove() throws Exception {
+    LOG.info("testRegionMove");
+
+    final RSGroupInfo newGroup = addGroup(rsGroupAdmin, getGroupName("testRegionMove"), 1);
+    final TableName tableName = TableName.valueOf(tablePrefix + rand.nextInt());
+    final byte[] familyNameBytes = Bytes.toBytes("f");
+    // All the regions created below will be assigned to the default group.
+    TEST_UTIL.createMultiRegionTable(tableName, familyNameBytes, 6);
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        List<String> regions = getTableRegionMap().get(tableName);
+        if (regions == null)
+          return false;
+        return getTableRegionMap().get(tableName).size() >= 6;
+      }
+    });
+
+    //get target region to move
+    Map<ServerName,List<String>> assignMap =
+        getTableServerRegionMap().get(tableName);
+    String targetRegion = null;
+    for(ServerName server : assignMap.keySet()) {
+      targetRegion = assignMap.get(server).size() > 0 ? assignMap.get(server).get(0) : null;
+      if(targetRegion != null) {
+        break;
+      }
+    }
+    //get server which is not a member of new group
+    ServerName targetServer = null;
+    for(ServerName server : admin.getClusterStatus().getServers()) {
+      if(!newGroup.containsServer(server.getHostPort())) {
+        targetServer = server;
+        break;
+      }
+    }
+
+    final AdminProtos.AdminService.BlockingInterface targetRS =
+        admin.getConnection().getAdmin(targetServer);
+
+    //move target server to group
+    rsGroupAdmin.moveServers(Sets.newHashSet(targetServer.getHostPort()),
+        newGroup.getName());
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return ProtobufUtil.getOnlineRegions(targetRS).size() <= 0;
+      }
+    });
+
+    // Lets move this region to the new group.
+    TEST_UTIL.getHBaseAdmin().move(Bytes.toBytes(HRegionInfo.encodeRegionName(Bytes.toBytes(targetRegion))),
+        Bytes.toBytes(targetServer.getServerName()));
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return
+            getTableRegionMap().get(tableName) != null &&
+                getTableRegionMap().get(tableName).size() == 6 &&
+                admin.getClusterStatus().getRegionsInTransition().size() < 1;
+      }
+    });
+
+    //verify that targetServer didn't open it
+    assertFalse(ProtobufUtil.getOnlineRegions(targetRS).contains(targetRegion));
+  }
+
+  @Test
+  public void testFailRemoveGroup() throws IOException, InterruptedException {
+    LOG.info("testFailRemoveGroup");
+
+    int initNumGroups = rsGroupAdmin.listRSGroups().size();
+    addGroup(rsGroupAdmin, "bar", 3);
+    TableName tableName = TableName.valueOf(tablePrefix+"_my_table");
+    TEST_UTIL.createTable(tableName, Bytes.toBytes("f"));
+    rsGroupAdmin.moveTables(Sets.newHashSet(tableName), "bar");
+    RSGroupInfo barGroup = rsGroupAdmin.getRSGroupInfo("bar");
+    //group is not empty therefore it should fail
+    try {
+      rsGroupAdmin.removeRSGroup(barGroup.getName());
+      fail("Expected remove group to fail");
+    } catch(IOException e) {
+    }
+    //group cannot lose all it's servers therefore it should fail
+    try {
+      rsGroupAdmin.moveServers(barGroup.getServers(), RSGroupInfo.DEFAULT_GROUP);
+      fail("Expected move servers to fail");
+    } catch(IOException e) {
+    }
+
+    rsGroupAdmin.moveTables(barGroup.getTables(), RSGroupInfo.DEFAULT_GROUP);
+    try {
+      rsGroupAdmin.removeRSGroup(barGroup.getName());
+      fail("Expected move servers to fail");
+    } catch(IOException e) {
+    }
+
+    rsGroupAdmin.moveServers(barGroup.getServers(), RSGroupInfo.DEFAULT_GROUP);
+    rsGroupAdmin.removeRSGroup(barGroup.getName());
+
+    Assert.assertEquals(initNumGroups, rsGroupAdmin.listRSGroups().size());
+  }
+
+  @Test
+  public void testKillRS() throws Exception {
+    LOG.info("testKillRS");
+    RSGroupInfo appInfo = addGroup(rsGroupAdmin, "appInfo", 1);
+
+
+    final TableName tableName = TableName.valueOf(tablePrefix+"_ns", "_testKillRS");
+    admin.createNamespace(
+        NamespaceDescriptor.create(tableName.getNamespaceAsString())
+            .addConfiguration(RSGroupInfo.NAMESPACEDESC_PROP_GROUP, appInfo.getName()).build());
+    final HTableDescriptor desc = new HTableDescriptor(tableName);
+    desc.addFamily(new HColumnDescriptor("f"));
+    admin.createTable(desc);
+    //wait for created table to be assigned
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return getTableRegionMap().get(desc.getTableName()) != null;
+      }
+    });
+
+    ServerName targetServer = ServerName.parseServerName(
+        appInfo.getServers().iterator().next().toString());
+    AdminProtos.AdminService.BlockingInterface targetRS =
+        admin.getConnection().getAdmin(targetServer);
+    HRegionInfo targetRegion = ProtobufUtil.getOnlineRegions(targetRS).get(0);
+    Assert.assertEquals(1, ProtobufUtil.getOnlineRegions(targetRS).size());
+
+    try {
+      //stopping may cause an exception
+      //due to the connection loss
+      targetRS.stopServer(null,
+          AdminProtos.StopServerRequest.newBuilder().setReason("Die").build());
+    } catch(Exception e) {
+    }
+    assertFalse(cluster.getClusterStatus().getServers().contains(targetServer));
+
+    //wait for created table to be assigned
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return cluster.getClusterStatus().getRegionsInTransition().size() == 0;
+      }
+    });
+    Set<HostAndPort> newServers = Sets.newHashSet();
+    newServers.add(
+        rsGroupAdmin.getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP).getServers().iterator().next());
+    rsGroupAdmin.moveServers(newServers, appInfo.getName());
+
+    //Make sure all the table's regions get reassigned
+    //disabling the table guarantees no conflicting assign/unassign (ie SSH) happens
+    admin.disableTable(tableName);
+    admin.enableTable(tableName);
+
+    //wait for region to be assigned
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return cluster.getClusterStatus().getRegionsInTransition().size() == 0;
+      }
+    });
+
+    targetServer = ServerName.parseServerName(
+        newServers.iterator().next().toString());
+    targetRS =
+        admin.getConnection().getAdmin(targetServer);
+    Assert.assertEquals(1, ProtobufUtil.getOnlineRegions(targetRS).size());
+    Assert.assertEquals(tableName,
+        ProtobufUtil.getOnlineRegions(targetRS).get(0).getTable());
+  }
+
+  @Test
+  public void testValidGroupNames() throws IOException {
+    String[] badNames = {"foo*","foo@","-"};
+    String[] goodNames = {"foo_123"};
+
+    for(String entry: badNames) {
+      try {
+        rsGroupAdmin.addRSGroup(entry);
+        fail("Expected a constraint exception for: "+entry);
+      } catch(ConstraintException ex) {
+        //expected
+      }
+    }
+
+    for(String entry: goodNames) {
+      rsGroupAdmin.addRSGroup(entry);
+    }
+  }
+
+  private String getGroupName(String baseName) {
+    return groupPrefix+"_"+baseName+"_"+rand.nextInt(Integer.MAX_VALUE);
+  }
+}

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsOfflineMode.java
----------------------------------------------------------------------
diff --git a/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsOfflineMode.java b/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsOfflineMode.java
new file mode 100644
index 0000000..b89ea0e
--- /dev/null
+++ b/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/TestRSGroupsOfflineMode.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 org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.hbase.HBaseCluster;
+import org.apache.hadoop.hbase.HBaseTestingUtility;
+import org.apache.hadoop.hbase.HConstants;
+import org.apache.hadoop.hbase.HRegionInfo;
+import org.apache.hadoop.hbase.MiniHBaseCluster;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.Waiter;
+import org.apache.hadoop.hbase.client.HBaseAdmin;
+import org.apache.hadoop.hbase.coprocessor.CoprocessorHost;
+import org.apache.hadoop.hbase.master.HMaster;
+import org.apache.hadoop.hbase.master.ServerManager;
+import org.apache.hadoop.hbase.regionserver.HRegionServer;
+import org.apache.hadoop.hbase.testclassification.MediumTests;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.junit.AfterClass;
+import org.junit.Assert;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+
+
+//This tests that GroupBasedBalancer will use data in zk
+//to do balancing during master startup
+//This does not test retain assignment
+@Category(MediumTests.class)
+public class TestRSGroupsOfflineMode {
+  private static final org.apache.commons.logging.Log LOG =
+      LogFactory.getLog(TestRSGroupsOfflineMode.class);
+  private static HMaster master;
+  private static HBaseAdmin hbaseAdmin;
+  private static HBaseTestingUtility TEST_UTIL;
+  private static HBaseCluster cluster;
+  private static RSGroupAdminEndpoint RSGroupAdminEndpoint;
+  public final static long WAIT_TIMEOUT = 60000*5;
+
+  @BeforeClass
+  public static void setUp() throws Exception {
+    TEST_UTIL = new HBaseTestingUtility();
+    TEST_UTIL.getConfiguration().set(
+        HConstants.HBASE_MASTER_LOADBALANCER_CLASS,
+        RSGroupBasedLoadBalancer.class.getName());
+    TEST_UTIL.getConfiguration().set(CoprocessorHost.MASTER_COPROCESSOR_CONF_KEY,
+        RSGroupAdminEndpoint.class.getName());
+    TEST_UTIL.getConfiguration().set(
+        ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART,
+        "1");
+    TEST_UTIL.startMiniCluster(2, 3);
+    cluster = TEST_UTIL.getHBaseCluster();
+    master = ((MiniHBaseCluster)cluster).getMaster();
+    master.balanceSwitch(false);
+    hbaseAdmin = TEST_UTIL.getHBaseAdmin();
+    //wait till the balancer is in online mode
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return master.isInitialized() &&
+            ((RSGroupBasedLoadBalancer) master.getLoadBalancer()).isOnline() &&
+            master.getServerManager().getOnlineServersList().size() >= 3;
+      }
+    });
+    RSGroupAdminEndpoint =
+        master.getMasterCoprocessorHost().findCoprocessors(RSGroupAdminEndpoint.class).get(0);
+  }
+
+  @AfterClass
+  public static void tearDown() throws Exception {
+    TEST_UTIL.shutdownMiniCluster();
+  }
+
+  @Test
+  public void testOffline() throws Exception, InterruptedException {
+    //table should be after group table name
+    //so it gets assigned later
+    final TableName failoverTable = TableName.valueOf("testOffline");
+    TEST_UTIL.createTable(failoverTable, Bytes.toBytes("f"));
+
+    RSGroupAdmin groupAdmin = RSGroupAdmin.newClient(TEST_UTIL.getConnection());
+
+    final HRegionServer killRS = ((MiniHBaseCluster)cluster).getRegionServer(0);
+    final HRegionServer groupRS = ((MiniHBaseCluster)cluster).getRegionServer(1);
+    final HRegionServer failoverRS = ((MiniHBaseCluster)cluster).getRegionServer(2);
+
+    String newGroup =  "my_group";
+    groupAdmin.addRSGroup(newGroup);
+    if(master.getAssignmentManager().getRegionStates().getRegionAssignments()
+        .containsValue(failoverRS.getServerName())) {
+      for(HRegionInfo regionInfo: hbaseAdmin.getOnlineRegions(failoverRS.getServerName())) {
+        hbaseAdmin.move(regionInfo.getEncodedNameAsBytes(),
+            Bytes.toBytes(failoverRS.getServerName().getServerName()));
+      }
+      LOG.info("Waiting for region unassignments on failover RS...");
+      TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+        @Override
+        public boolean evaluate() throws Exception {
+          return master.getServerManager().getLoad(failoverRS.getServerName())
+              .getRegionsLoad().size() > 0;
+        }
+      });
+    }
+
+    //move server to group and make sure all tables are assigned
+    groupAdmin.moveServers(Sets.newHashSet(groupRS.getServerName().getHostPort()), newGroup);
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return groupRS.getNumberOfOnlineRegions() < 1 &&
+            master.getAssignmentManager().getRegionStates().getRegionsInTransition().size() < 1;
+      }
+    });
+    //move table to group and wait
+    groupAdmin.moveTables(Sets.newHashSet(RSGroupInfoManager.RSGROUP_TABLE_NAME), newGroup);
+    LOG.info("Waiting for move table...");
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return groupRS.getNumberOfOnlineRegions() == 1;
+      }
+    });
+
+    groupRS.stop("die");
+    //race condition here
+    TEST_UTIL.getHBaseCluster().getMaster().stopMaster();
+    LOG.info("Waiting for offline mode...");
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return TEST_UTIL.getHBaseCluster().getMaster() != null &&
+            TEST_UTIL.getHBaseCluster().getMaster().isActiveMaster() &&
+            TEST_UTIL.getHBaseCluster().getMaster().isInitialized() &&
+            TEST_UTIL.getHBaseCluster().getMaster().getServerManager().getOnlineServers().size()
+                <= 3;
+      }
+    });
+
+
+    RSGroupInfoManager groupMgr = RSGroupAdminEndpoint.getGroupInfoManager();
+    //make sure balancer is in offline mode, since this is what we're testing
+    assertFalse(groupMgr.isOnline());
+    //verify the group affiliation that's loaded from ZK instead of tables
+    assertEquals(newGroup,
+        groupMgr.getRSGroupOfTable(RSGroupInfoManager.RSGROUP_TABLE_NAME));
+    assertEquals(RSGroupInfo.DEFAULT_GROUP, groupMgr.getRSGroupOfTable(failoverTable));
+
+    //kill final regionserver to see the failover happens for all tables
+    //except GROUP table since it's group does not have any online RS
+    killRS.stop("die");
+    master = TEST_UTIL.getHBaseCluster().getMaster();
+    LOG.info("Waiting for new table assignment...");
+    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
+      @Override
+      public boolean evaluate() throws Exception {
+        return failoverRS.getOnlineRegions(failoverTable).size() >= 1;
+      }
+    });
+    Assert.assertEquals(0, failoverRS.getOnlineRegions(RSGroupInfoManager.RSGROUP_TABLE_NAME).size());
+
+    //need this for minicluster to shutdown cleanly
+    master.stopMaster();
+  }
+}

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/VerifyingRSGroupAdminClient.java
----------------------------------------------------------------------
diff --git a/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/VerifyingRSGroupAdminClient.java b/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/VerifyingRSGroupAdminClient.java
new file mode 100644
index 0000000..d1f4898
--- /dev/null
+++ b/hbase-rsgroup/src/test/java/org/apache/hadoop/hbase/rsgroup/VerifyingRSGroupAdminClient.java
@@ -0,0 +1,149 @@
+/**
+ * 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.Maps;
+import com.google.common.collect.Sets;
+import com.google.common.net.HostAndPort;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.client.ConnectionFactory;
+import org.apache.hadoop.hbase.client.Result;
+import org.apache.hadoop.hbase.client.Scan;
+import org.apache.hadoop.hbase.client.Table;
+import org.apache.hadoop.hbase.exceptions.DeserializationException;
+import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
+import org.apache.hadoop.hbase.protobuf.generated.RSGroupProtos;
+import org.apache.hadoop.hbase.zookeeper.ZKUtil;
+import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
+import org.apache.zookeeper.KeeperException;
+import org.junit.Assert;
+
+import java.io.ByteArrayInputStream;
+import java.io.IOException;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+
+public class VerifyingRSGroupAdminClient extends RSGroupAdmin {
+  private Table table;
+  private ZooKeeperWatcher zkw;
+  private RSGroupSerDe serDe;
+  private RSGroupAdmin wrapped;
+
+  public VerifyingRSGroupAdminClient(RSGroupAdmin RSGroupAdmin, Configuration conf)
+      throws IOException {
+    wrapped = RSGroupAdmin;
+    table = ConnectionFactory.createConnection(conf).getTable(RSGroupInfoManager.RSGROUP_TABLE_NAME);
+    zkw = new ZooKeeperWatcher(conf, this.getClass().getSimpleName(), null);
+    serDe = new RSGroupSerDe();
+  }
+
+  @Override
+  public void addRSGroup(String groupName) throws IOException {
+    wrapped.addRSGroup(groupName);
+    verify();
+  }
+
+  @Override
+  public RSGroupInfo getRSGroupInfo(String groupName) throws IOException {
+    return wrapped.getRSGroupInfo(groupName);
+  }
+
+  @Override
+  public RSGroupInfo getRSGroupInfoOfTable(TableName tableName) throws IOException {
+    return wrapped.getRSGroupInfoOfTable(tableName);
+  }
+
+  @Override
+  public void moveServers(Set<HostAndPort> servers, String targetGroup) throws IOException {
+    wrapped.moveServers(servers, targetGroup);
+    verify();
+  }
+
+  @Override
+  public void moveTables(Set<TableName> tables, String targetGroup) throws IOException {
+    wrapped.moveTables(tables, targetGroup);
+    verify();
+  }
+
+  @Override
+  public void removeRSGroup(String name) throws IOException {
+    wrapped.removeRSGroup(name);
+    verify();
+  }
+
+  @Override
+  public boolean balanceRSGroup(String name) throws IOException {
+    return wrapped.balanceRSGroup(name);
+  }
+
+  @Override
+  public List<RSGroupInfo> listRSGroups() throws IOException {
+    return wrapped.listRSGroups();
+  }
+
+  @Override
+  public RSGroupInfo getRSGroupOfServer(HostAndPort hostPort) throws IOException {
+    return wrapped.getRSGroupOfServer(hostPort);
+  }
+
+  public void verify() throws IOException {
+    Map<String, RSGroupInfo> groupMap = Maps.newHashMap();
+    Set<RSGroupInfo> zList = Sets.newHashSet();
+
+    for (Result result : table.getScanner(new Scan())) {
+      RSGroupProtos.RSGroupInfo proto =
+          RSGroupProtos.RSGroupInfo.parseFrom(
+              result.getValue(
+                  RSGroupInfoManager.META_FAMILY_BYTES,
+                  RSGroupInfoManager.META_QUALIFIER_BYTES));
+      groupMap.put(proto.getName(), ProtobufUtil.toGroupInfo(proto));
+    }
+    Assert.assertEquals(Sets.newHashSet(groupMap.values()),
+        Sets.newHashSet(wrapped.listRSGroups()));
+    try {
+      String groupBasePath = ZKUtil.joinZNode(zkw.baseZNode, "rsgroup");
+      for(String znode: ZKUtil.listChildrenNoWatch(zkw, groupBasePath)) {
+        byte[] data = ZKUtil.getData(zkw, ZKUtil.joinZNode(groupBasePath, znode));
+        if(data.length > 0) {
+          ProtobufUtil.expectPBMagicPrefix(data);
+          ByteArrayInputStream bis = new ByteArrayInputStream(
+              data, ProtobufUtil.lengthOfPBMagic(), data.length);
+          zList.add(ProtobufUtil.toGroupInfo(RSGroupProtos.RSGroupInfo.parseFrom(bis)));
+        }
+      }
+      Assert.assertEquals(zList.size(), groupMap.size());
+      for(RSGroupInfo RSGroupInfo : zList) {
+        Assert.assertTrue(groupMap.get(RSGroupInfo.getName()).equals(RSGroupInfo));
+      }
+    } catch (KeeperException e) {
+      throw new IOException("ZK verification failed", e);
+    } catch (DeserializationException e) {
+      throw new IOException("ZK verification failed", e);
+    } catch (InterruptedException e) {
+      throw new IOException("ZK verification failed", e);
+    }
+  }
+
+  @Override
+  public void close() throws IOException {
+  }
+}

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/jamon/org/apache/hadoop/hbase/tmpl/master/MasterStatusTmpl.jamon
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/jamon/org/apache/hadoop/hbase/tmpl/master/MasterStatusTmpl.jamon b/hbase-server/src/main/jamon/org/apache/hadoop/hbase/tmpl/master/MasterStatusTmpl.jamon
index 1af3db9..951a95e 100644
--- a/hbase-server/src/main/jamon/org/apache/hadoop/hbase/tmpl/master/MasterStatusTmpl.jamon
+++ b/hbase-server/src/main/jamon/org/apache/hadoop/hbase/tmpl/master/MasterStatusTmpl.jamon
@@ -30,30 +30,30 @@ AssignmentManager assignmentManager = null;
 </%args>
 <%import>
 java.util.*;
-org.apache.hadoop.util.StringUtils;
-org.apache.hadoop.hbase.util.Bytes;
-org.apache.hadoop.hbase.util.JvmVersion;
-org.apache.hadoop.hbase.util.FSUtils;
-org.apache.hadoop.hbase.master.HMaster;
-org.apache.hadoop.hbase.master.AssignmentManager;
-org.apache.hadoop.hbase.master.ServerManager;
+org.apache.hadoop.hbase.HBaseConfiguration;
 org.apache.hadoop.hbase.HConstants;
+org.apache.hadoop.hbase.HRegionInfo;
+org.apache.hadoop.hbase.HTableDescriptor;
 org.apache.hadoop.hbase.NamespaceDescriptor;
 org.apache.hadoop.hbase.ServerLoad;
 org.apache.hadoop.hbase.ServerName;
-org.apache.hadoop.hbase.client.Admin;
-org.apache.hadoop.hbase.HRegionInfo;
-org.apache.hadoop.hbase.master.RegionState;
-org.apache.hadoop.hbase.HTableDescriptor;
-org.apache.hadoop.hbase.HBaseConfiguration;
 org.apache.hadoop.hbase.TableName;
-org.apache.hadoop.hbase.tool.Canary;
-org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription;
+org.apache.hadoop.hbase.client.Admin;
+org.apache.hadoop.hbase.master.AssignmentManager;
 org.apache.hadoop.hbase.master.DeadServer;
+org.apache.hadoop.hbase.master.HMaster;
+org.apache.hadoop.hbase.master.RegionState;
+org.apache.hadoop.hbase.master.ServerManager;
 org.apache.hadoop.hbase.protobuf.ProtobufUtil;
-org.apache.hadoop.hbase.security.visibility.VisibilityConstants;
-org.apache.hadoop.hbase.security.access.AccessControlLists;
+org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription;
 org.apache.hadoop.hbase.quotas.QuotaUtil;
+org.apache.hadoop.hbase.security.access.AccessControlLists;
+org.apache.hadoop.hbase.security.visibility.VisibilityConstants;
+org.apache.hadoop.hbase.tool.Canary;
+org.apache.hadoop.hbase.util.Bytes;
+org.apache.hadoop.hbase.util.FSUtils;
+org.apache.hadoop.hbase.util.JvmVersion;
+org.apache.hadoop.util.StringUtils;
 </%import>
 
 <%if format.equals("json") %>
@@ -380,6 +380,8 @@ AssignmentManager assignmentManager = master.getAssignmentManager();
         } else if (tableName.equals(QuotaUtil.QUOTA_TABLE_NAME)){
             description = "The hbase:quota table holds quota information about number" +
             " or size of requests in a given time frame.";
+        } else if (tableName.equals(TableName.valueOf("hbase:rsgroup"))){
+            description = "The hbase:rsgroup table holds information about regionserver groups";
         }
     </%java>
     <td><% description %></td>

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterAndRegionObserver.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterAndRegionObserver.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterAndRegionObserver.java
index d601491..4748056 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterAndRegionObserver.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterAndRegionObserver.java
@@ -19,8 +19,11 @@
 
 package org.apache.hadoop.hbase.coprocessor;
 
+import com.google.common.net.HostAndPort;
+
 import java.io.IOException;
 import java.util.List;
+import java.util.Set;
 
 import org.apache.hadoop.hbase.CoprocessorEnvironment;
 import org.apache.hadoop.hbase.HBaseInterfaceAudience;
@@ -631,4 +634,54 @@ public abstract class BaseMasterAndRegionObserver extends BaseRegionObserver
   public void postSetNamespaceQuota(final ObserverContext<MasterCoprocessorEnvironment> ctx,
       final String namespace, final Quotas quotas) throws IOException {
   }
+
+  @Override
+  public void postAddRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+  }
+
+  @Override
+  public void postBalanceRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                 String groupName, boolean balancerRan) throws IOException {
+  }
+
+  @Override
+  public void postMoveServers(ObserverContext<MasterCoprocessorEnvironment> ctx, Set<HostAndPort>
+      servers, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void postMoveTables(ObserverContext<MasterCoprocessorEnvironment> ctx, Set<TableName>
+      tables, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void postRemoveRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+  }
+
+  @Override
+  public void preAddRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+  }
+
+  @Override
+  public void preBalanceRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String groupName)
+      throws IOException {
+  }
+
+  @Override
+  public void preMoveServers(ObserverContext<MasterCoprocessorEnvironment> ctx,
+      Set<HostAndPort> servers, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void preMoveTables(ObserverContext<MasterCoprocessorEnvironment> ctx,
+      Set<TableName> tables, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void preRemoveRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+  }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterObserver.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterObserver.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterObserver.java
index 0cdf0ad..0adb179 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterObserver.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/BaseMasterObserver.java
@@ -19,8 +19,11 @@
 
 package org.apache.hadoop.hbase.coprocessor;
 
+import com.google.common.net.HostAndPort;
+
 import java.io.IOException;
 import java.util.List;
+import java.util.Set;
 
 import org.apache.hadoop.hbase.CoprocessorEnvironment;
 import org.apache.hadoop.hbase.HBaseInterfaceAudience;
@@ -634,4 +637,55 @@ public class BaseMasterObserver implements MasterObserver {
   public void postSetNamespaceQuota(final ObserverContext<MasterCoprocessorEnvironment> ctx,
       final String namespace, final Quotas quotas) throws IOException {
   }
+
+  @Override
+  public void preMoveServers(ObserverContext<MasterCoprocessorEnvironment> ctx, Set<HostAndPort>
+      servers, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void postMoveServers(ObserverContext<MasterCoprocessorEnvironment> ctx, Set<HostAndPort>
+      servers, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void preMoveTables(ObserverContext<MasterCoprocessorEnvironment> ctx, Set<TableName>
+      tables, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void postMoveTables(ObserverContext<MasterCoprocessorEnvironment> ctx,
+      Set<TableName> tables, String targetGroup) throws IOException {
+  }
+
+  @Override
+  public void preAddRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+  }
+
+  @Override
+  public void postAddRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+  }
+
+  @Override
+  public void preRemoveRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+
+  }
+
+  @Override
+  public void postRemoveRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String name)
+      throws IOException {
+  }
+
+  @Override
+  public void preBalanceRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx, String groupName)
+      throws IOException {
+  }
+
+  @Override
+  public void postBalanceRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                 String groupName, boolean balancerRan) throws IOException {
+  }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/MasterObserver.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/MasterObserver.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/MasterObserver.java
index b7a535b..82b6ffe 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/MasterObserver.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/coprocessor/MasterObserver.java
@@ -19,8 +19,11 @@
 
 package org.apache.hadoop.hbase.coprocessor;
 
+import com.google.common.net.HostAndPort;
+
 import java.io.IOException;
 import java.util.List;
+import java.util.Set;
 
 import org.apache.hadoop.hbase.Coprocessor;
 import org.apache.hadoop.hbase.HBaseInterfaceAudience;
@@ -1238,4 +1241,99 @@ public interface MasterObserver extends Coprocessor {
    */
   void postDispatchMerge(final ObserverContext<MasterCoprocessorEnvironment> c,
       final HRegionInfo regionA, final HRegionInfo regionB) throws IOException;
+
+  /**
+   * Called before servers are moved to target region server group
+   * @param ctx the environment to interact with the framework and master
+   * @param servers set of servers to move
+   * @param targetGroup destination group
+   * @throws IOException on failure
+   */
+  void preMoveServers(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                      Set<HostAndPort> servers, String targetGroup) throws IOException;
+
+  /**
+   * Called after servers are moved to target region server group
+   * @param ctx the environment to interact with the framework and master
+   * @param servers set of servers to move
+   * @param targetGroup name of group
+   * @throws IOException on failure
+   */
+  void postMoveServers(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                       Set<HostAndPort> servers, String targetGroup) throws IOException;
+
+  /**
+   * Called before tables are moved to target region server group
+   * @param ctx the environment to interact with the framework and master
+   * @param tables set of tables to move
+   * @param targetGroup name of group
+   * @throws IOException on failure
+   */
+  void preMoveTables(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                     Set<TableName> tables, String targetGroup) throws IOException;
+
+  /**
+   * Called after servers are moved to target region server group
+   * @param ctx the environment to interact with the framework and master
+   * @param tables set of tables to move
+   * @param targetGroup name of group
+   * @throws IOException on failure
+   */
+  void postMoveTables(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                      Set<TableName> tables, String targetGroup) throws IOException;
+
+  /**
+   * Called before a new region server group is added
+   * @param ctx the environment to interact with the framework and master
+   * @param name group name
+   * @throws IOException on failure
+   */
+  void preAddRSGroup(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                     String name) throws IOException;
+
+  /**
+   * Called after a new region server group is added
+   * @param ctx the environment to interact with the framework and master
+   * @param name group name
+   * @throws IOException on failure
+   */
+  void postAddRSGroup(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                      String name) throws IOException;
+
+  /**
+   * Called before a region server group is removed
+   * @param ctx the environment to interact with the framework and master
+   * @param name group name
+   * @throws IOException on failure
+   */
+  void preRemoveRSGroup(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                        String name) throws IOException;
+
+  /**
+   * Called after a region server group is removed
+   * @param ctx the environment to interact with the framework and master
+   * @param name group name
+   * @throws IOException on failure
+   */
+  void postRemoveRSGroup(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                         String name) throws IOException;
+
+  /**
+   * Called before a region server group is removed
+   * @param ctx the environment to interact with the framework and master
+   * @param groupName group name
+   * @throws IOException on failure
+   */
+  void preBalanceRSGroup(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                         String groupName) throws IOException;
+
+  /**
+   * Called after a region server group is removed
+   * @param ctx the environment to interact with the framework and master
+   * @param groupName group name
+   * @throws IOException on failure
+   */
+  void postBalanceRSGroup(final ObserverContext<MasterCoprocessorEnvironment> ctx,
+                          String groupName, boolean balancerRan) throws IOException;
+
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/master/AssignmentManager.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/AssignmentManager.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/AssignmentManager.java
index 1110db3..2b7713b 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/AssignmentManager.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/AssignmentManager.java
@@ -1078,7 +1078,7 @@ public class AssignmentManager {
           return;
         }
         LOG.info("Assigning " + region.getRegionNameAsString() +
-            " to " + plan.getDestination().toString());
+            " to " + plan.getDestination());
         // Transition RegionState to PENDING_OPEN
        regionStates.updateRegionState(region,
           State.PENDING_OPEN, plan.getDestination());
@@ -1267,8 +1267,13 @@ public class AssignmentManager {
           || existingPlan.getDestination() == null
           || !destServers.contains(existingPlan.getDestination())) {
         newPlan = true;
-        randomPlan = new RegionPlan(region, null,
-            balancer.randomAssignment(region, destServers));
+        try {
+          randomPlan = new RegionPlan(region, null,
+              balancer.randomAssignment(region, destServers));
+        } catch (IOException ex) {
+          LOG.warn("Failed to create new plan.",ex);
+          return null;
+        }
         if (!region.isMetaTable() && shouldAssignRegionsWithFavoredNodes) {
           List<HRegionInfo> regions = new ArrayList<HRegionInfo>(1);
           regions.add(region);
@@ -1513,6 +1518,8 @@ public class AssignmentManager {
       throw new IOException("Unable to determine a plan to assign region(s)");
     }
 
+    processBogusAssignments(bulkPlan);
+
     assign(regions.size(), servers.size(),
       "retainAssignment=true", bulkPlan);
   }
@@ -1542,6 +1549,8 @@ public class AssignmentManager {
       throw new IOException("Unable to determine a plan to assign region(s)");
     }
 
+    processBogusAssignments(bulkPlan);
+
     processFavoredNodes(regions);
     assign(regions.size(), servers.size(), "round-robin=true", bulkPlan);
   }
@@ -2954,6 +2963,16 @@ public class AssignmentManager {
     }
   }
 
+  private void processBogusAssignments(Map<ServerName, List<HRegionInfo>> bulkPlan) {
+    if (bulkPlan.containsKey(LoadBalancer.BOGUS_SERVER_NAME)) {
+      // Found no plan for some regions, put those regions in RIT
+      for (HRegionInfo hri : bulkPlan.get(LoadBalancer.BOGUS_SERVER_NAME)) {
+        regionStates.updateRegionState(hri, State.FAILED_OPEN);
+      }
+      bulkPlan.remove(LoadBalancer.BOGUS_SERVER_NAME);
+    }
+  }
+
   /**
    * @return Instance of load balancer
    */

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
index b4bffb4..142502a 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
@@ -18,6 +18,13 @@
  */
 package org.apache.hadoop.hbase.master;
 
+import com.google.common.annotations.VisibleForTesting;
+import com.google.common.collect.Lists;
+import com.google.common.collect.Maps;
+import com.google.protobuf.Descriptors;
+import com.google.protobuf.Service;
+
+
 import java.io.IOException;
 import java.io.InterruptedIOException;
 import java.lang.reflect.Constructor;
@@ -165,11 +172,6 @@ import org.mortbay.jetty.Connector;
 import org.mortbay.jetty.nio.SelectChannelConnector;
 import org.mortbay.jetty.servlet.Context;
 
-import com.google.common.annotations.VisibleForTesting;
-import com.google.common.collect.Maps;
-import com.google.protobuf.Descriptors;
-import com.google.protobuf.Service;
-
 /**
  * HMaster is the "master server" for HBase. An HBase cluster has one active
  * master.  If many masters are started, all compete.  Whichever wins goes on to
@@ -1396,11 +1398,14 @@ public class HMaster extends HRegionServer implements MasterServices {
       final byte[] destServerName) throws HBaseIOException {
     RegionState regionState = assignmentManager.getRegionStates().
       getRegionState(Bytes.toString(encodedRegionName));
-    if (regionState == null) {
+
+    HRegionInfo hri;
+    if (regionState != null) {
+      hri = regionState.getRegion();
+    } else {
       throw new UnknownRegionException(Bytes.toStringBinary(encodedRegionName));
     }
 
-    HRegionInfo hri = regionState.getRegion();
     ServerName dest;
     if (destServerName == null || destServerName.length == 0) {
       LOG.info("Passed destination servername is null/empty so " +
@@ -1413,7 +1418,12 @@ public class HMaster extends HRegionServer implements MasterServices {
         return;
       }
     } else {
-      dest = ServerName.valueOf(Bytes.toString(destServerName));
+      ServerName candidate = ServerName.valueOf(Bytes.toString(destServerName));
+      dest = balancer.randomAssignment(hri, Lists.newArrayList(candidate));
+      if (dest == null) {
+        LOG.debug("Unable to determine a plan to assign " + hri);
+        return;
+      }
       if (dest.equals(serverName) && balancer instanceof BaseLoadBalancer
           && !((BaseLoadBalancer)balancer).shouldBeOnMaster(hri)) {
         // To avoid unnecessary region moving later by balancer. Don't put user
@@ -1476,7 +1486,6 @@ public class HMaster extends HRegionServer implements MasterServices {
     HRegionInfo[] newRegions = ModifyRegionUtils.createHRegionInfos(hTableDescriptor, splitKeys);
     checkInitialized();
     sanityCheckTableDescriptor(hTableDescriptor);
-
     if (cpHost != null) {
       cpHost.preCreateTable(hTableDescriptor, newRegions);
     }
@@ -2827,4 +2836,9 @@ public class HMaster extends HRegionServer implements MasterServices {
   public SplitOrMergeTracker getSplitOrMergeTracker() {
     return splitOrMergeTracker;
   }
+
+  @Override
+  public LoadBalancer getLoadBalancer() {
+    return balancer;
+  }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/master/LoadBalancer.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/LoadBalancer.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/LoadBalancer.java
index 6a618e1..5d6be9e 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/LoadBalancer.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/LoadBalancer.java
@@ -48,6 +48,9 @@ import org.apache.hadoop.hbase.TableName;
 @InterfaceAudience.Private
 public interface LoadBalancer extends Configurable, Stoppable, ConfigurationObserver {
 
+  //used to signal to the caller that the region(s) cannot be assigned
+  ServerName BOGUS_SERVER_NAME = ServerName.parseServerName("localhost,1,1");
+
   /**
    * Set the current cluster status.  This allows a LoadBalancer to map host name to a server
    * @param st

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterCoprocessorHost.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterCoprocessorHost.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterCoprocessorHost.java
index 5888b3e..b16c232 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterCoprocessorHost.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterCoprocessorHost.java
@@ -19,8 +19,11 @@
 
 package org.apache.hadoop.hbase.master;
 
+import com.google.common.net.HostAndPort;
+
 import java.io.IOException;
 import java.util.List;
+import java.util.Set;
 
 import org.apache.commons.lang.ClassUtils;
 import org.apache.commons.logging.Log;
@@ -63,12 +66,15 @@ public class MasterCoprocessorHost
   static class MasterEnvironment extends CoprocessorHost.Environment
       implements MasterCoprocessorEnvironment {
     private MasterServices masterServices;
+    final boolean supportGroupCPs;
 
     public MasterEnvironment(final Class<?> implClass, final Coprocessor impl,
         final int priority, final int seq, final Configuration conf,
         final MasterServices services) {
       super(impl, priority, seq, conf);
       this.masterServices = services;
+      supportGroupCPs = !useLegacyMethod(impl.getClass(),
+          "preBalanceRSGroup", ObserverContext.class, String.class);
     }
 
     public MasterServices getMasterServices() {
@@ -1170,4 +1176,135 @@ public class MasterCoprocessorHost
     }
     return bypass;
   }
+
+  public void preMoveServers(final Set<HostAndPort> servers, final String targetGroup)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.preMoveServers(ctx, servers, targetGroup);
+        }
+      }
+    });
+  }
+
+  public void postMoveServers(final Set<HostAndPort> servers, final String targetGroup)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.postMoveServers(ctx, servers, targetGroup);
+        }
+      }
+    });
+  }
+
+  public void preMoveTables(final Set<TableName> tables, final String targetGroup)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.preMoveTables(ctx, tables, targetGroup);
+        }
+      }
+    });
+  }
+
+  public void postMoveTables(final Set<TableName> tables, final String targetGroup)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.postMoveTables(ctx, tables, targetGroup);
+        }
+      }
+    });
+  }
+
+  public void preAddRSGroup(final String name)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.preAddRSGroup(ctx, name);
+        }
+      }
+    });
+  }
+
+  public void postAddRSGroup(final String name)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if (((MasterEnvironment) ctx.getEnvironment()).supportGroupCPs) {
+          oserver.postAddRSGroup(ctx, name);
+        }
+      }
+    });
+  }
+
+  public void preRemoveRSGroup(final String name)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.preRemoveRSGroup(ctx, name);
+        }
+      }
+    });
+  }
+
+  public void postRemoveRSGroup(final String name)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.postRemoveRSGroup(ctx, name);
+        }
+      }
+    });
+  }
+
+  public void preBalanceRSGroup(final String name)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.preBalanceRSGroup(ctx, name);
+        }
+      }
+    });
+  }
+
+  public void postBalanceRSGroup(final String name, final boolean balanceRan)
+      throws IOException {
+    execOperation(coprocessors.isEmpty() ? null : new CoprocessorOperation() {
+      @Override
+      public void call(MasterObserver oserver,
+          ObserverContext<MasterCoprocessorEnvironment> ctx) throws IOException {
+        if(((MasterEnvironment)ctx.getEnvironment()).supportGroupCPs) {
+          oserver.postBalanceRSGroup(ctx, name, balanceRan);
+        }
+      }
+    });
+  }
+
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterRpcServices.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterRpcServices.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterRpcServices.java
index d6a53cd..753ecb6 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterRpcServices.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterRpcServices.java
@@ -1309,6 +1309,14 @@ public class MasterRpcServices extends RSRpcServices
       }
       Pair<HRegionInfo, ServerName> pair =
         MetaTableAccessor.getRegion(master.getConnection(), regionName);
+      if (Bytes.equals(HRegionInfo.FIRST_META_REGIONINFO.getRegionName(),regionName)) {
+        pair = new Pair<HRegionInfo, ServerName>(HRegionInfo.FIRST_META_REGIONINFO,
+            master.getMetaTableLocator().getMetaRegionLocation(master.getZooKeeper()));
+      }
+      if (pair == null) {
+        throw new UnknownRegionException(Bytes.toString(regionName));
+      }
+
       if (pair == null) throw new UnknownRegionException(Bytes.toString(regionName));
       HRegionInfo hri = pair.getFirst();
       if (master.cpHost != null) {

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java
index 59c7a88..8ce21b0 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java
@@ -328,4 +328,9 @@ public interface MasterServices extends Server {
    * @throws IOException
    */
   public long getLastMajorCompactionTimestampForRegion(byte[] regionName) throws IOException;
+
+  /**
+   * @return load balancer
+   */
+  public LoadBalancer getLoadBalancer();
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/main/java/org/apache/hadoop/hbase/security/access/AccessController.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/security/access/AccessController.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/security/access/AccessController.java
index 43f9833..0a88a86 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/security/access/AccessController.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/security/access/AccessController.java
@@ -18,6 +18,8 @@
  */
 package org.apache.hadoop.hbase.security.access;
 
+import com.google.common.net.HostAndPort;
+
 import java.io.IOException;
 import java.net.InetAddress;
 import java.security.PrivilegedExceptionAction;
@@ -2619,4 +2621,34 @@ public class AccessController extends BaseMasterAndRegionObserver
   public void postReplicateLogEntries(ObserverContext<RegionServerCoprocessorEnvironment> ctx,
       List<WALEntry> entries, CellScanner cells) throws IOException {
   }
+
+  @Override
+  public void preMoveServers(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                             Set<HostAndPort> servers, String targetGroup) throws IOException {
+    requirePermission("moveServers", Action.ADMIN);
+  }
+
+  @Override
+  public void preMoveTables(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                            Set<TableName> tables, String targetGroup) throws IOException {
+    requirePermission("moveTables", Action.ADMIN);
+  }
+
+  @Override
+  public void preAddRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                            String name) throws IOException {
+    requirePermission("addRSGroup", Action.ADMIN);
+  }
+
+  @Override
+  public void preRemoveRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                               String name) throws IOException {
+    requirePermission("removeRSGroup", Action.ADMIN);
+  }
+
+  @Override
+  public void preBalanceRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                String groupName) throws IOException {
+    requirePermission("balanceRSGroup", Action.ADMIN);
+  }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
index 7e79b64..3810f1e 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/coprocessor/TestMasterObserver.java
@@ -28,6 +28,7 @@ import java.io.IOException;
 import java.util.Arrays;
 import java.util.Collection;
 import java.util.List;
+import java.util.Set;
 import java.util.concurrent.CountDownLatch;
 
 import org.apache.commons.logging.Log;
@@ -74,6 +75,8 @@ import org.junit.Test;
 import org.junit.experimental.categories.Category;
 import org.junit.rules.TestName;
 
+import com.google.common.net.HostAndPort;
+
 /**
  * Tests invocation of the {@link org.apache.hadoop.hbase.coprocessor.MasterObserver}
  * interface hooks at all appropriate times during normal HMaster operations.
@@ -1312,6 +1315,56 @@ public class TestMasterObserver {
     public void postSetNamespaceQuota(final ObserverContext<MasterCoprocessorEnvironment> ctx,
         final String namespace, final Quotas quotas) throws IOException {
     }
+
+    @Override
+    public void preMoveServers(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                               Set<HostAndPort> servers, String targetGroup) throws IOException {
+    }
+
+    @Override
+    public void postMoveServers(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                Set<HostAndPort> servers, String targetGroup) throws IOException {
+    }
+
+    @Override
+    public void preMoveTables(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                              Set<TableName> tables, String targetGroupGroup) throws IOException {
+    }
+
+    @Override
+    public void postMoveTables(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                               Set<TableName> tables, String targetGroup) throws IOException {
+    }
+
+    @Override
+    public void preAddRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                              String name) throws IOException {
+    }
+
+    @Override
+    public void postAddRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                               String name) throws IOException {
+    }
+
+    @Override
+    public void preRemoveRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                 String name) throws IOException {
+    }
+
+    @Override
+    public void postRemoveRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                  String name) throws IOException {
+    }
+
+    @Override
+    public void preBalanceRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                  String groupName) throws IOException {
+    }
+
+    @Override
+    public void postBalanceRSGroup(ObserverContext<MasterCoprocessorEnvironment> ctx,
+                                   String groupName, boolean balancerRan) throws IOException {
+    }
   }
 
   private static HBaseTestingUtility UTIL = new HBaseTestingUtility();

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestAssignmentManagerOnCluster.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestAssignmentManagerOnCluster.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestAssignmentManagerOnCluster.java
index 703d54e..fdfc1b3 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestAssignmentManagerOnCluster.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestAssignmentManagerOnCluster.java
@@ -27,12 +27,16 @@ import static org.junit.Assert.fail;
 
 import java.io.IOException;
 import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
 import java.util.Set;
 import java.util.concurrent.atomic.AtomicBoolean;
 import java.util.concurrent.atomic.AtomicInteger;
 
+import com.google.common.collect.Lists;
+import com.google.common.collect.Maps;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
@@ -536,7 +540,7 @@ public class TestAssignmentManagerOnCluster {
         desc.getTableName(), Bytes.toBytes("A"), Bytes.toBytes("Z"));
       MetaTableAccessor.addRegionToMeta(meta, hri);
 
-      MyLoadBalancer.controledRegion = hri.getEncodedName();
+      MyLoadBalancer.controledRegion = hri;
 
       HMaster master = TEST_UTIL.getHBaseCluster().getMaster();
       master.assignRegion(hri);
@@ -562,6 +566,105 @@ public class TestAssignmentManagerOnCluster {
   }
 
   /**
+   * This tests round-robin assignment failed due to no bulkplan
+   */
+  @Test (timeout=60000)
+  public void testRoundRobinAssignmentFailed() throws Exception {
+    TableName tableName = TableName.valueOf("testRoundRobinAssignmentFailed");
+    try {
+      HTableDescriptor desc = new HTableDescriptor(tableName);
+      desc.addFamily(new HColumnDescriptor(FAMILY));
+      admin.createTable(desc);
+
+      Table meta = admin.getConnection().getTable(TableName.META_TABLE_NAME);
+      HRegionInfo hri = new HRegionInfo(
+        desc.getTableName(), Bytes.toBytes("A"), Bytes.toBytes("Z"));
+      MetaTableAccessor.addRegionToMeta(meta, hri);
+
+      MyLoadBalancer.controledRegion = hri;
+
+      HMaster master = TEST_UTIL.getHBaseCluster().getMaster();
+      AssignmentManager am = master.getAssignmentManager();
+      // round-robin assignment but balancer cannot find a plan
+      // assignment should fail
+      am.assign(Arrays.asList(hri));
+
+      // if bulk assignment cannot update region state to online
+      // or failed_open this waits until timeout
+      assertFalse(am.waitForAssignment(hri));
+      RegionState state = am.getRegionStates().getRegionState(hri);
+      assertEquals(RegionState.State.FAILED_OPEN, state.getState());
+      // Failed to open since no plan, so it's on no server
+      assertNull(state.getServerName());
+
+      // try again with valid plan
+      MyLoadBalancer.controledRegion = null;
+      am.assign(Arrays.asList(hri));
+      assertTrue(am.waitForAssignment(hri));
+
+      ServerName serverName = master.getAssignmentManager().
+        getRegionStates().getRegionServerOfRegion(hri);
+      TEST_UTIL.assertRegionOnServer(hri, serverName, 200);
+    } finally {
+      MyLoadBalancer.controledRegion = null;
+      TEST_UTIL.deleteTable(tableName);
+    }
+  }
+
+  /**
+   * This tests retain assignment failed due to no bulkplan
+   */
+  @Test (timeout=60000)
+  public void testRetainAssignmentFailed() throws Exception {
+    TableName tableName = TableName.valueOf("testRetainAssignmentFailed");
+    try {
+      HTableDescriptor desc = new HTableDescriptor(tableName);
+      desc.addFamily(new HColumnDescriptor(FAMILY));
+      admin.createTable(desc);
+
+      Table meta = TEST_UTIL.getConnection().getTable(TableName.META_TABLE_NAME);
+      HRegionInfo hri = new HRegionInfo(
+        desc.getTableName(), Bytes.toBytes("A"), Bytes.toBytes("Z"));
+      MetaTableAccessor.addRegionToMeta(meta, hri);
+
+      MyLoadBalancer.controledRegion = hri;
+
+      HMaster master = TEST_UTIL.getHBaseCluster().getMaster();
+      AssignmentManager am = master.getAssignmentManager();
+
+      Map<HRegionInfo, ServerName> regions = new HashMap<HRegionInfo, ServerName>();
+      ServerName dest = TEST_UTIL.getHBaseCluster().getRegionServer(0).getServerName();
+      regions.put(hri, dest);
+      // retainAssignment but balancer cannot find a plan
+      // assignment should fail
+      am.assign(regions);
+
+      // if retain assignment cannot update region state to online
+      // or failed_open this waits until timeout
+      assertFalse(am.waitForAssignment(hri));
+      RegionState state = am.getRegionStates().getRegionState(hri);
+      assertEquals(RegionState.State.FAILED_OPEN, state.getState());
+      // Failed to open since no plan, so it's on no server
+      assertNull(state.getServerName());
+
+      // try retainAssigment again with valid plan
+      MyLoadBalancer.controledRegion = null;
+      am.assign(regions);
+      assertTrue(am.waitForAssignment(hri));
+
+      ServerName serverName = master.getAssignmentManager().
+        getRegionStates().getRegionServerOfRegion(hri);
+      TEST_UTIL.assertRegionOnServer(hri, serverName, 200);
+
+      // it retains on same server as specified
+      assertEquals(serverName, dest);
+    } finally {
+      MyLoadBalancer.controledRegion = null;
+      TEST_UTIL.deleteTable(tableName);
+    }
+  }
+
+  /**
    * This tests region open failure which is not recoverable
    */
   @Test (timeout=60000)
@@ -1169,7 +1272,7 @@ public class TestAssignmentManagerOnCluster {
 
   static class MyLoadBalancer extends StochasticLoadBalancer {
     // For this region, if specified, always assign to nowhere
-    static volatile String controledRegion = null;
+    static volatile HRegionInfo controledRegion = null;
 
     static volatile Integer countRegionServers = null;
     static AtomicInteger counter = new AtomicInteger(0);
@@ -1177,7 +1280,7 @@ public class TestAssignmentManagerOnCluster {
     @Override
     public ServerName randomAssignment(HRegionInfo regionInfo,
         List<ServerName> servers) {
-      if (regionInfo.getEncodedName().equals(controledRegion)) {
+      if (regionInfo.equals(controledRegion)) {
         return null;
       }
       return super.randomAssignment(regionInfo, servers);
@@ -1195,8 +1298,26 @@ public class TestAssignmentManagerOnCluster {
           return null;
         }
       }
+      if (regions.get(0).equals(controledRegion)) {
+        Map<ServerName, List<HRegionInfo>> m = Maps.newHashMap();
+        m.put(LoadBalancer.BOGUS_SERVER_NAME, regions);
+        return m;
+      }
       return super.roundRobinAssignment(regions, servers);
     }
+
+    @Override
+    public Map<ServerName, List<HRegionInfo>> retainAssignment(
+        Map<HRegionInfo, ServerName> regions, List<ServerName> servers) {
+      for (HRegionInfo hri : regions.keySet()) {
+        if (hri.equals(controledRegion)) {
+          Map<ServerName, List<HRegionInfo>> m = Maps.newHashMap();
+          m.put(LoadBalancer.BOGUS_SERVER_NAME, Lists.newArrayList(regions.keySet()));
+          return m;
+        }
+      }
+      return super.retainAssignment(regions, servers);
+    }
   }
 
   public static class MyMaster extends HMaster {

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java
index 123e8b5..2949369 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java
@@ -434,6 +434,9 @@ public class TestCatalogJanitor {
         final long nonce) throws IOException {
       return -1;
     }
+    public LoadBalancer getLoadBalancer() {
+      return null;
+    }
 
     @Override
     public long truncateTable(

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestMasterStatusServlet.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestMasterStatusServlet.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestMasterStatusServlet.java
index f5f04e9..135aae5 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestMasterStatusServlet.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestMasterStatusServlet.java
@@ -135,7 +135,7 @@ public class TestMasterStatusServlet {
     setupMockTables();
 
     new MasterStatusTmpl()
-      .setMetaLocation(ServerName.valueOf("metaserver:123,12345"))
+      .setMetaLocation(ServerName.valueOf("metaserver,123,12345"))
       .render(new StringWriter(), master);
   }
 
@@ -144,16 +144,16 @@ public class TestMasterStatusServlet {
     setupMockTables();
 
     List<ServerName> servers = Lists.newArrayList(
-        ServerName.valueOf("rootserver:123,12345"),
-        ServerName.valueOf("metaserver:123,12345"));
+        ServerName.valueOf("rootserver,123,12345"),
+        ServerName.valueOf("metaserver,123,12345"));
     Set<ServerName> deadServers = new HashSet<ServerName>(
         Lists.newArrayList(
-            ServerName.valueOf("badserver:123,12345"),
-            ServerName.valueOf("uglyserver:123,12345"))
+            ServerName.valueOf("badserver,123,12345"),
+            ServerName.valueOf("uglyserver,123,12345"))
     );
 
     new MasterStatusTmpl()
-      .setMetaLocation(ServerName.valueOf("metaserver:123,12345"))
+      .setMetaLocation(ServerName.valueOf("metaserver,123,12345"))
       .setServers(servers)
       .setDeadServers(deadServers)
       .render(new StringWriter(), master);

http://git-wip-us.apache.org/repos/asf/hbase/blob/ca816f07/hbase-server/src/test/java/org/apache/hadoop/hbase/master/normalizer/TestSimpleRegionNormalizer.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/normalizer/TestSimpleRegionNormalizer.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/normalizer/TestSimpleRegionNormalizer.java
index 926c816..babf22d 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/normalizer/TestSimpleRegionNormalizer.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/normalizer/TestSimpleRegionNormalizer.java
@@ -261,7 +261,7 @@ public class TestSimpleRegionNormalizer {
     masterServices = Mockito.mock(MasterServices.class, RETURNS_DEEP_STUBS);
 
     // for simplicity all regions are assumed to be on one server; doesn't matter to us
-    ServerName sn = ServerName.valueOf("localhost", -1, 1L);
+    ServerName sn = ServerName.valueOf("localhost", 0, 1L);
     when(masterServices.getAssignmentManager().getRegionStates().
       getRegionsOfTable(any(TableName.class))).thenReturn(hris);
     when(masterServices.getAssignmentManager().getRegionStates().