You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@accumulo.apache.org by el...@apache.org on 2014/06/05 19:30:05 UTC

[1/6] git commit: ACCUMULO-2857 Return a more real tableID

Repository: accumulo
Updated Branches:
  refs/heads/1.5.2-SNAPSHOT 6dfcf8ff7 -> 9fcca2ede
  refs/heads/1.6.1-SNAPSHOT 05cf91849 -> 38ff00008
  refs/heads/master 47d593311 -> 2149b6430


ACCUMULO-2857 Return a more real tableID


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

Branch: refs/heads/1.5.2-SNAPSHOT
Commit: 9fcca2ede18a7b96bee4edb5ab105316ac383604
Parents: 6dfcf8f
Author: Josh Elser <el...@apache.org>
Authored: Thu Jun 5 12:53:26 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Jun 5 12:53:26 2014 -0400

----------------------------------------------------------------------
 .../accumulo/core/client/mock/MockAccumulo.java     |  4 +++-
 .../apache/accumulo/core/client/mock/MockTable.java |  8 +++++++-
 .../core/client/mock/MockTableOperations.java       |  4 ++--
 .../core/client/mock/MockTableOperationsTest.java   | 11 +++++++++++
 .../master/balancer/TableLoadBalancerTest.java      | 16 +++++++++-------
 5 files changed, 32 insertions(+), 11 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
index 5977d1d..272d1af 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
@@ -21,6 +21,7 @@ import java.util.EnumSet;
 import java.util.HashMap;
 import java.util.Map;
 import java.util.SortedSet;
+import java.util.concurrent.atomic.AtomicInteger;
 
 import org.apache.accumulo.core.Constants;
 import org.apache.accumulo.core.client.BatchScanner;
@@ -38,6 +39,7 @@ public class MockAccumulo {
   final Map<String,String> systemProperties = new HashMap<String,String>();
   Map<String,MockUser> users = new HashMap<String,MockUser>();
   final FileSystem fs;
+  final AtomicInteger tableIdCounter = new AtomicInteger(0);
   
   MockAccumulo(FileSystem fs) {
     this.fs = fs;
@@ -76,7 +78,7 @@ public class MockAccumulo {
   }
   
   public void createTable(String username, String tableName, boolean useVersions, TimeType timeType) {
-    MockTable t = new MockTable(useVersions, timeType);
+    MockTable t = new MockTable(useVersions, timeType, Integer.toString(tableIdCounter.incrementAndGet()));
     t.userPermissions.put(username, EnumSet.allOf(TablePermission.class));
     tables.put(tableName, t);
   }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
index 3dcab11..2e13d84 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
@@ -89,9 +89,11 @@ public class MockTable {
   private TimeType timeType;
   SortedSet<Text> splits = new ConcurrentSkipListSet<Text>();
   Map<String,Set<Text>> localityGroups = new TreeMap<String, Set<Text>>();
+  private String tableId;
   
-  MockTable(boolean limitVersion, TimeType timeType) {
+  MockTable(boolean limitVersion, TimeType timeType, String tableId) {
     this.timeType = timeType;
+    this.tableId = tableId;
     settings = IteratorUtil.generateInitialTableProperties(limitVersion);
     for (Entry<String,String> entry : AccumuloConfiguration.getDefaultConfiguration()) {
       String key = entry.getKey();
@@ -143,4 +145,8 @@ public class MockTable {
     if (reAdd)
       splits.add(start);
   }
+
+  public String getTableId() {
+    return this.tableId;
+  }
 }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
index 64f8225..5b15351 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
@@ -296,8 +296,8 @@ public class MockTableOperations extends TableOperationsHelper {
   @Override
   public Map<String,String> tableIdMap() {
     Map<String,String> result = new HashMap<String,String>();
-    for (String table : acu.tables.keySet()) {
-      result.put(table, table);
+    for (Entry<String,MockTable> entry : acu.tables.entrySet()) {
+      result.put(entry.getKey(), entry.getValue().getTableId());
     }
     return result;
   }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
----------------------------------------------------------------------
diff --git a/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java b/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
index ea916e7..cea66a4 100644
--- a/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
+++ b/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
@@ -333,4 +333,15 @@ public class MockTableOperationsTest {
 
   }
 
+
+  @Test
+  public void testTableIdMap() throws Exception {
+    Instance inst = new MockInstance("testTableIdMap");
+    Connector conn = inst.getConnector("root", new PasswordToken(""));
+    TableOperations tops = conn.tableOperations();
+    tops.create("foo");
+
+    // Should get a table ID, not the table name
+    Assert.assertNotEquals("foo", tops.tableIdMap().get("foo"));
+  }
 }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
----------------------------------------------------------------------
diff --git a/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java b/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
index 71ba50d..b7c6412 100644
--- a/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
+++ b/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
@@ -128,12 +128,14 @@ public class TableLoadBalancerTest {
   @Test
   public void test() throws Exception {
     Connector c = instance.getConnector("user", new PasswordToken("pass"));
-    c.tableOperations().create("t1");
-    c.tableOperations().create("t2");
-    c.tableOperations().create("t3");
+    TableOperations tops = c.tableOperations();
+    tops.create("t1");
+    tops.create("t2");
+    tops.create("t3");
+    String t1Id = tops.tableIdMap().get("t1"), t2Id = tops.tableIdMap().get("t2"), t3Id = tops.tableIdMap().get("t3");
     state = new TreeMap<TServerInstance,TabletServerStatus>();
     TServerInstance svr = mkts("10.0.0.1:1234", "0x01020304");
-    state.put(svr, status("t1", 10, "t2", 10, "t3", 10));
+    state.put(svr, status(t1Id, 10, t2Id, 10, t3Id, 10));
     
     Set<KeyExtent> migrations = Collections.emptySet();
     List<TabletMigration> migrationsOut = new ArrayList<TabletMigration>();
@@ -146,9 +148,9 @@ public class TableLoadBalancerTest {
     tls.balance(state, migrations, migrationsOut);
     int count = 0;
     Map<String,Integer> movedByTable = new HashMap<String,Integer>();
-    movedByTable.put("t1", new Integer(0));
-    movedByTable.put("t2", new Integer(0));
-    movedByTable.put("t3", new Integer(0));
+    movedByTable.put(t1Id, new Integer(0));
+    movedByTable.put(t2Id, new Integer(0));
+    movedByTable.put(t3Id, new Integer(0));
     for (TabletMigration migration : migrationsOut) {
       if (migration.oldServer.equals(svr))
         count++;


[3/6] git commit: ACCUMULO-2857 Return a more real tableID

Posted by el...@apache.org.
ACCUMULO-2857 Return a more real tableID


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

Branch: refs/heads/master
Commit: 9fcca2ede18a7b96bee4edb5ab105316ac383604
Parents: 6dfcf8f
Author: Josh Elser <el...@apache.org>
Authored: Thu Jun 5 12:53:26 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Jun 5 12:53:26 2014 -0400

----------------------------------------------------------------------
 .../accumulo/core/client/mock/MockAccumulo.java     |  4 +++-
 .../apache/accumulo/core/client/mock/MockTable.java |  8 +++++++-
 .../core/client/mock/MockTableOperations.java       |  4 ++--
 .../core/client/mock/MockTableOperationsTest.java   | 11 +++++++++++
 .../master/balancer/TableLoadBalancerTest.java      | 16 +++++++++-------
 5 files changed, 32 insertions(+), 11 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
index 5977d1d..272d1af 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
@@ -21,6 +21,7 @@ import java.util.EnumSet;
 import java.util.HashMap;
 import java.util.Map;
 import java.util.SortedSet;
+import java.util.concurrent.atomic.AtomicInteger;
 
 import org.apache.accumulo.core.Constants;
 import org.apache.accumulo.core.client.BatchScanner;
@@ -38,6 +39,7 @@ public class MockAccumulo {
   final Map<String,String> systemProperties = new HashMap<String,String>();
   Map<String,MockUser> users = new HashMap<String,MockUser>();
   final FileSystem fs;
+  final AtomicInteger tableIdCounter = new AtomicInteger(0);
   
   MockAccumulo(FileSystem fs) {
     this.fs = fs;
@@ -76,7 +78,7 @@ public class MockAccumulo {
   }
   
   public void createTable(String username, String tableName, boolean useVersions, TimeType timeType) {
-    MockTable t = new MockTable(useVersions, timeType);
+    MockTable t = new MockTable(useVersions, timeType, Integer.toString(tableIdCounter.incrementAndGet()));
     t.userPermissions.put(username, EnumSet.allOf(TablePermission.class));
     tables.put(tableName, t);
   }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
index 3dcab11..2e13d84 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
@@ -89,9 +89,11 @@ public class MockTable {
   private TimeType timeType;
   SortedSet<Text> splits = new ConcurrentSkipListSet<Text>();
   Map<String,Set<Text>> localityGroups = new TreeMap<String, Set<Text>>();
+  private String tableId;
   
-  MockTable(boolean limitVersion, TimeType timeType) {
+  MockTable(boolean limitVersion, TimeType timeType, String tableId) {
     this.timeType = timeType;
+    this.tableId = tableId;
     settings = IteratorUtil.generateInitialTableProperties(limitVersion);
     for (Entry<String,String> entry : AccumuloConfiguration.getDefaultConfiguration()) {
       String key = entry.getKey();
@@ -143,4 +145,8 @@ public class MockTable {
     if (reAdd)
       splits.add(start);
   }
+
+  public String getTableId() {
+    return this.tableId;
+  }
 }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
index 64f8225..5b15351 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
@@ -296,8 +296,8 @@ public class MockTableOperations extends TableOperationsHelper {
   @Override
   public Map<String,String> tableIdMap() {
     Map<String,String> result = new HashMap<String,String>();
-    for (String table : acu.tables.keySet()) {
-      result.put(table, table);
+    for (Entry<String,MockTable> entry : acu.tables.entrySet()) {
+      result.put(entry.getKey(), entry.getValue().getTableId());
     }
     return result;
   }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
----------------------------------------------------------------------
diff --git a/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java b/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
index ea916e7..cea66a4 100644
--- a/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
+++ b/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
@@ -333,4 +333,15 @@ public class MockTableOperationsTest {
 
   }
 
+
+  @Test
+  public void testTableIdMap() throws Exception {
+    Instance inst = new MockInstance("testTableIdMap");
+    Connector conn = inst.getConnector("root", new PasswordToken(""));
+    TableOperations tops = conn.tableOperations();
+    tops.create("foo");
+
+    // Should get a table ID, not the table name
+    Assert.assertNotEquals("foo", tops.tableIdMap().get("foo"));
+  }
 }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
----------------------------------------------------------------------
diff --git a/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java b/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
index 71ba50d..b7c6412 100644
--- a/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
+++ b/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
@@ -128,12 +128,14 @@ public class TableLoadBalancerTest {
   @Test
   public void test() throws Exception {
     Connector c = instance.getConnector("user", new PasswordToken("pass"));
-    c.tableOperations().create("t1");
-    c.tableOperations().create("t2");
-    c.tableOperations().create("t3");
+    TableOperations tops = c.tableOperations();
+    tops.create("t1");
+    tops.create("t2");
+    tops.create("t3");
+    String t1Id = tops.tableIdMap().get("t1"), t2Id = tops.tableIdMap().get("t2"), t3Id = tops.tableIdMap().get("t3");
     state = new TreeMap<TServerInstance,TabletServerStatus>();
     TServerInstance svr = mkts("10.0.0.1:1234", "0x01020304");
-    state.put(svr, status("t1", 10, "t2", 10, "t3", 10));
+    state.put(svr, status(t1Id, 10, t2Id, 10, t3Id, 10));
     
     Set<KeyExtent> migrations = Collections.emptySet();
     List<TabletMigration> migrationsOut = new ArrayList<TabletMigration>();
@@ -146,9 +148,9 @@ public class TableLoadBalancerTest {
     tls.balance(state, migrations, migrationsOut);
     int count = 0;
     Map<String,Integer> movedByTable = new HashMap<String,Integer>();
-    movedByTable.put("t1", new Integer(0));
-    movedByTable.put("t2", new Integer(0));
-    movedByTable.put("t3", new Integer(0));
+    movedByTable.put(t1Id, new Integer(0));
+    movedByTable.put(t2Id, new Integer(0));
+    movedByTable.put(t3Id, new Integer(0));
     for (TabletMigration migration : migrationsOut) {
       if (migration.oldServer.equals(svr))
         count++;


[5/6] git commit: Merge branch '1.5.2-SNAPSHOT' into 1.6.1-SNAPSHOT

Posted by el...@apache.org.
Merge branch '1.5.2-SNAPSHOT' into 1.6.1-SNAPSHOT

Conflicts:
	core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
	core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
	core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
	server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java


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

Branch: refs/heads/1.6.1-SNAPSHOT
Commit: 38ff0000879af97d24372cb004a81e10901e4276
Parents: 05cf918 9fcca2e
Author: Josh Elser <el...@apache.org>
Authored: Thu Jun 5 13:21:43 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Jun 5 13:21:43 2014 -0400

----------------------------------------------------------------------
 .../accumulo/core/client/mock/MockAccumulo.java      |  4 +++-
 .../apache/accumulo/core/client/mock/MockTable.java  | 12 +++++++++---
 .../core/client/mock/MockTableOperationsImpl.java    |  5 +++--
 .../core/client/mock/MockTableOperationsTest.java    | 11 +++++++++++
 .../master/balancer/TableLoadBalancerTest.java       | 15 +++++++++------
 5 files changed, 35 insertions(+), 12 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
----------------------------------------------------------------------
diff --cc core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
index 2c26ecc,272d1af..32dbb28
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
@@@ -21,17 -21,14 +21,18 @@@ import java.util.EnumSet
  import java.util.HashMap;
  import java.util.Map;
  import java.util.SortedSet;
+ import java.util.concurrent.atomic.AtomicInteger;
  
 -import org.apache.accumulo.core.Constants;
  import org.apache.accumulo.core.client.BatchScanner;
  import org.apache.accumulo.core.client.admin.TimeType;
 +import org.apache.accumulo.core.client.impl.Namespaces;
 +import org.apache.accumulo.core.client.impl.Tables;
  import org.apache.accumulo.core.client.security.tokens.PasswordToken;
  import org.apache.accumulo.core.data.Mutation;
 +import org.apache.accumulo.core.metadata.MetadataTable;
 +import org.apache.accumulo.core.metadata.RootTable;
  import org.apache.accumulo.core.security.Authorizations;
 +import org.apache.accumulo.core.security.NamespacePermission;
  import org.apache.accumulo.core.security.SystemPermission;
  import org.apache.accumulo.core.security.TablePermission;
  import org.apache.hadoop.fs.FileSystem;
@@@ -43,7 -39,8 +44,8 @@@ public class MockAccumulo 
    final Map<String,String> systemProperties = new HashMap<String,String>();
    Map<String,MockUser> users = new HashMap<String,MockUser>();
    final FileSystem fs;
+   final AtomicInteger tableIdCounter = new AtomicInteger(0);
 -  
 +
    MockAccumulo(FileSystem fs) {
      this.fs = fs;
    }
@@@ -82,30 -76,13 +84,30 @@@
    public BatchScanner createBatchScanner(String tableName, Authorizations authorizations) {
      return new MockBatchScanner(tables.get(tableName), authorizations);
    }
 -  
 +
    public void createTable(String username, String tableName, boolean useVersions, TimeType timeType) {
 -    MockTable t = new MockTable(useVersions, timeType, Integer.toString(tableIdCounter.incrementAndGet()));
 +    String namespace = Tables.qualify(tableName).getFirst();
 +
 +    if (!namespaceExists(namespace)) {
 +      return;
 +    }
 +
 +    MockNamespace n = namespaces.get(namespace);
-     MockTable t = new MockTable(n, useVersions, timeType);
++    MockTable t = new MockTable(n, useVersions, timeType, Integer.toString(tableIdCounter.incrementAndGet()));
      t.userPermissions.put(username, EnumSet.allOf(TablePermission.class));
 +    t.setNamespaceName(namespace);
 +    t.setNamespace(n);
      tables.put(tableName, t);
    }
 -  
 +
 +  public void createNamespace(String username, String namespace) {
 +    if (!namespaceExists(namespace)) {
 +      MockNamespace n = new MockNamespace();
 +      n.userPermissions.put(username, EnumSet.allOf(NamespacePermission.class));
 +      namespaces.put(namespace, n);
 +    }
 +  }
 +
    public void addSplits(String tableName, SortedSet<Text> partitionKeys) {
      tables.get(tableName).addSplits(partitionKeys);
    }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
----------------------------------------------------------------------
diff --cc core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
index cb50761,2e13d84..35cbdd2
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
@@@ -89,12 -88,12 +89,14 @@@ public class MockTable 
    Map<String,EnumSet<TablePermission>> userPermissions = new HashMap<String,EnumSet<TablePermission>>();
    private TimeType timeType;
    SortedSet<Text> splits = new ConcurrentSkipListSet<Text>();
 -  Map<String,Set<Text>> localityGroups = new TreeMap<String, Set<Text>>();
 +  Map<String,Set<Text>> localityGroups = new TreeMap<String,Set<Text>>();
 +  private MockNamespace namespace;
 +  private String namespaceName;
+   private String tableId;
    
-   MockTable(boolean limitVersion, TimeType timeType) {
+   MockTable(boolean limitVersion, TimeType timeType, String tableId) {
      this.timeType = timeType;
+     this.tableId = tableId;
      settings = IteratorUtil.generateInitialTableProperties(limitVersion);
      for (Entry<String,String> entry : AccumuloConfiguration.getDefaultConfiguration()) {
        String key = entry.getKey();
@@@ -103,18 -102,6 +105,18 @@@
      }
    }
    
-   MockTable(MockNamespace namespace, boolean limitVersion, TimeType timeType) {
-     this(limitVersion, timeType);
++  MockTable(MockNamespace namespace, boolean limitVersion, TimeType timeType, String tableId) {
++    this(limitVersion, timeType, tableId);
 +    Set<Entry<String,String>> set = namespace.settings.entrySet();
 +    Iterator<Entry<String,String>> entries = set.iterator();
 +    while (entries.hasNext()) {
 +      Entry<String,String> entry = entries.next();
 +      String key = entry.getKey();
 +      if (key.startsWith(Property.TABLE_PREFIX.getKey()))
 +        settings.put(key, entry.getValue());
 +    }
 +  }
 +  
    synchronized void addMutation(Mutation m) {
      if (m.size() == 0)
        throw new IllegalArgumentException("Can not add empty mutations");
@@@ -159,20 -145,8 +161,24 @@@
      if (reAdd)
        splits.add(start);
    }
 +  
 +  public void setNamespaceName(String n) {
 +    this.namespaceName = n;
 +  }
 +  
 +  public void setNamespace(MockNamespace n) {
 +    this.namespace = n;
 +  }
 +  
 +  public String getNamespaceName() {
 +    return this.namespaceName;
 +  }
 +  
 +  public MockNamespace getNamespace() {
 +    return this.namespace;
 +  }
+ 
+   public String getTableId() {
+     return this.tableId;
+   }
  }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
----------------------------------------------------------------------
diff --cc core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
index 8a8895f,0000000..de19137
mode 100644,000000..100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
@@@ -1,447 -1,0 +1,448 @@@
 +/*
 + * 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.accumulo.core.client.mock;
 +
 +import java.io.DataInputStream;
 +import java.io.IOException;
 +import java.util.ArrayList;
 +import java.util.Collection;
 +import java.util.Collections;
 +import java.util.HashMap;
 +import java.util.HashSet;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Set;
 +import java.util.SortedSet;
 +import java.util.TreeSet;
 +
 +import org.apache.accumulo.core.client.AccumuloException;
 +import org.apache.accumulo.core.client.AccumuloSecurityException;
 +import org.apache.accumulo.core.client.IteratorSetting;
 +import org.apache.accumulo.core.client.NamespaceNotFoundException;
 +import org.apache.accumulo.core.client.TableExistsException;
 +import org.apache.accumulo.core.client.TableNotFoundException;
 +import org.apache.accumulo.core.client.admin.DiskUsage;
 +import org.apache.accumulo.core.client.admin.FindMax;
 +import org.apache.accumulo.core.client.impl.TableOperationsHelper;
 +import org.apache.accumulo.core.client.admin.TimeType;
 +import org.apache.accumulo.core.client.impl.Tables;
 +import org.apache.accumulo.core.conf.AccumuloConfiguration;
 +import org.apache.accumulo.core.data.Key;
 +import org.apache.accumulo.core.data.Mutation;
 +import org.apache.accumulo.core.data.Range;
 +import org.apache.accumulo.core.data.Value;
 +import org.apache.accumulo.core.file.FileOperations;
 +import org.apache.accumulo.core.file.FileSKVIterator;
 +import org.apache.accumulo.core.metadata.MetadataTable;
 +import org.apache.accumulo.core.metadata.RootTable;
 +import org.apache.accumulo.core.security.Authorizations;
 +import org.apache.accumulo.core.security.ColumnVisibility;
 +import org.apache.accumulo.start.classloader.vfs.AccumuloVFSClassLoader;
 +import org.apache.commons.lang.NotImplementedException;
 +import org.apache.hadoop.fs.FSDataOutputStream;
 +import org.apache.hadoop.fs.FileStatus;
 +import org.apache.hadoop.fs.FileSystem;
 +import org.apache.hadoop.fs.Path;
 +import org.apache.hadoop.io.Text;
 +
 +class MockTableOperationsImpl extends TableOperationsHelper {
 +  private static final byte[] ZERO = {0};
 +  private final MockAccumulo acu;
 +  private final String username;
 +
 +  MockTableOperationsImpl(MockAccumulo acu, String username) {
 +    this.acu = acu;
 +    this.username = username;
 +  }
 +
 +  @Override
 +  public SortedSet<String> list() {
 +    return new TreeSet<String>(acu.tables.keySet());
 +  }
 +
 +  @Override
 +  public boolean exists(String tableName) {
 +    return acu.tables.containsKey(tableName);
 +  }
 +
 +  private boolean namespaceExists(String namespace) {
 +    return acu.namespaces.containsKey(namespace);
 +  }
 +
 +  @Override
 +  public void create(String tableName) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    create(tableName, true, TimeType.MILLIS);
 +  }
 +
 +  @Override
 +  public void create(String tableName, boolean versioningIter) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    create(tableName, versioningIter, TimeType.MILLIS);
 +  }
 +
 +  @Override
 +  public void create(String tableName, boolean versioningIter, TimeType timeType) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    String namespace = Tables.qualify(tableName).getFirst();
 +    if (!tableName.matches(Tables.VALID_NAME_REGEX)) {
 +      throw new IllegalArgumentException();
 +    }
 +    if (exists(tableName))
 +      throw new TableExistsException(tableName, tableName, "");
 +
 +    if (!namespaceExists(namespace)) {
 +      throw new IllegalArgumentException("Namespace (" + namespace + ") does not exist, create it first");
 +    }
 +    acu.createTable(username, tableName, versioningIter, timeType);
 +  }
 +
 +  @Override
 +  public void addSplits(String tableName, SortedSet<Text> partitionKeys) throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.addSplits(tableName, partitionKeys);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public Collection<Text> getSplits(String tableName) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public Collection<Text> getSplits(String tableName, int maxSplits) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Override
 +  public Collection<Text> listSplits(String tableName) throws TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return acu.getSplits(tableName);
 +  }
 +
 +  @Override
 +  public Collection<Text> listSplits(String tableName, int maxSplits) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Override
 +  public void delete(String tableName) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.tables.remove(tableName);
 +  }
 +
 +  @Override
 +  public void rename(String oldTableName, String newTableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException,
 +      TableExistsException {
 +    if (!exists(oldTableName))
 +      throw new TableNotFoundException(oldTableName, oldTableName, "");
 +    if (exists(newTableName))
 +      throw new TableExistsException(newTableName, newTableName, "");
 +    MockTable t = acu.tables.remove(oldTableName);
 +    String namespace = Tables.qualify(newTableName).getFirst();
 +    MockNamespace n = acu.namespaces.get(namespace);
 +    if (n == null) {
 +      n = new MockNamespace();
 +    }
 +    t.setNamespaceName(namespace);
 +    t.setNamespace(n);
 +    acu.namespaces.put(namespace, n);
 +    acu.tables.put(newTableName, t);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public void flush(String tableName) throws AccumuloException, AccumuloSecurityException {}
 +
 +  @Override
 +  public void setProperty(String tableName, String property, String value) throws AccumuloException, AccumuloSecurityException {
 +    acu.tables.get(tableName).settings.put(property, value);
 +  }
 +
 +  @Override
 +  public void removeProperty(String tableName, String property) throws AccumuloException, AccumuloSecurityException {
 +    acu.tables.get(tableName).settings.remove(property);
 +  }
 +
 +  @Override
 +  public Iterable<Entry<String,String>> getProperties(String tableName) throws TableNotFoundException {
 +    String namespace = Tables.qualify(tableName).getFirst();
 +    if (!exists(tableName)) {
 +      if (!namespaceExists(namespace))
 +        throw new TableNotFoundException(tableName, new NamespaceNotFoundException(null, namespace, null));
 +      throw new TableNotFoundException(null, tableName, null);
 +    }
 +
 +    Set<Entry<String,String>> props = new HashSet<Entry<String,String>>(acu.namespaces.get(namespace).settings.entrySet());
 +
 +    Set<Entry<String,String>> tableProps = acu.tables.get(tableName).settings.entrySet();
 +    for (Entry<String,String> e : tableProps) {
 +      if (props.contains(e)) {
 +        props.remove(e);
 +      }
 +      props.add(e);
 +    }
 +    return props;
 +  }
 +
 +  @Override
 +  public void setLocalityGroups(String tableName, Map<String,Set<Text>> groups) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.tables.get(tableName).setLocalityGroups(groups);
 +  }
 +
 +  @Override
 +  public Map<String,Set<Text>> getLocalityGroups(String tableName) throws AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return acu.tables.get(tableName).getLocalityGroups();
 +  }
 +
 +  @Override
 +  public Set<Range> splitRangeByTablets(String tableName, Range range, int maxSplits) throws AccumuloException, AccumuloSecurityException,
 +      TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return Collections.singleton(range);
 +  }
 +
 +  @Override
 +  public void importDirectory(String tableName, String dir, String failureDir, boolean setTime) throws IOException, AccumuloException,
 +      AccumuloSecurityException, TableNotFoundException {
 +    long time = System.currentTimeMillis();
 +    MockTable table = acu.tables.get(tableName);
 +    if (table == null) {
 +      throw new TableNotFoundException(null, tableName, "The table was not found");
 +    }
 +    Path importPath = new Path(dir);
 +    Path failurePath = new Path(failureDir);
 +
 +    FileSystem fs = acu.getFileSystem();
 +    /*
 +     * check preconditions
 +     */
 +    // directories are directories
 +    if (fs.isFile(importPath)) {
 +      throw new IOException("Import path must be a directory.");
 +    }
 +    if (fs.isFile(failurePath)) {
 +      throw new IOException("Failure path must be a directory.");
 +    }
 +    // failures are writable
 +    Path createPath = failurePath.suffix("/.createFile");
 +    FSDataOutputStream createStream = null;
 +    try {
 +      createStream = fs.create(createPath);
 +    } catch (IOException e) {
 +      throw new IOException("Error path is not writable.");
 +    } finally {
 +      if (createStream != null) {
 +        createStream.close();
 +      }
 +    }
 +    fs.delete(createPath, false);
 +    // failures are empty
 +    FileStatus[] failureChildStats = fs.listStatus(failurePath);
 +    if (failureChildStats.length > 0) {
 +      throw new IOException("Error path must be empty.");
 +    }
 +    /*
 +     * Begin the import - iterate the files in the path
 +     */
 +    for (FileStatus importStatus : fs.listStatus(importPath)) {
 +      try {
 +        FileSKVIterator importIterator = FileOperations.getInstance().openReader(importStatus.getPath().toString(), true, fs, fs.getConf(),
 +            AccumuloConfiguration.getDefaultConfiguration());
 +        while (importIterator.hasTop()) {
 +          Key key = importIterator.getTopKey();
 +          Value value = importIterator.getTopValue();
 +          if (setTime) {
 +            key.setTimestamp(time);
 +          }
 +          Mutation mutation = new Mutation(key.getRow());
 +          if (!key.isDeleted()) {
 +            mutation.put(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()), key.getTimestamp(),
 +                value);
 +          } else {
 +            mutation.putDelete(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()),
 +                key.getTimestamp());
 +          }
 +          table.addMutation(mutation);
 +          importIterator.next();
 +        }
 +      } catch (Exception e) {
 +        FSDataOutputStream failureWriter = null;
 +        DataInputStream failureReader = null;
 +        try {
 +          failureWriter = fs.create(failurePath.suffix("/" + importStatus.getPath().getName()));
 +          failureReader = fs.open(importStatus.getPath());
 +          int read = 0;
 +          byte[] buffer = new byte[1024];
 +          while (-1 != (read = failureReader.read(buffer))) {
 +            failureWriter.write(buffer, 0, read);
 +          }
 +        } finally {
 +          if (failureReader != null)
 +            failureReader.close();
 +          if (failureWriter != null)
 +            failureWriter.close();
 +        }
 +      }
 +      fs.delete(importStatus.getPath(), true);
 +    }
 +  }
 +
 +  @Override
 +  public void offline(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    offline(tableName, false);
 +  }
 +
 +  @Override
 +  public void offline(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void online(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    online(tableName, false);
 +  }
 +
 +  @Override
 +  public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void clearLocatorCache(String tableName) throws TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public Map<String,String> tableIdMap() {
 +    Map<String,String> result = new HashMap<String,String>();
-     for (String table : acu.tables.keySet()) {
++    for (Entry<String,MockTable> entry : acu.tables.entrySet()) {
++      String table = entry.getKey();
 +      if (RootTable.NAME.equals(table))
 +        result.put(table, RootTable.ID);
 +      else if (MetadataTable.NAME.equals(table))
 +        result.put(table, MetadataTable.ID);
 +      else
-         result.put(table, table);
++        result.put(table, entry.getValue().getTableId());
 +    }
 +    return result;
 +  }
 +
 +  @Override
 +  public List<DiskUsage> getDiskUsage(Set<String> tables) throws AccumuloException, AccumuloSecurityException {
 +
 +    List<DiskUsage> diskUsages = new ArrayList<DiskUsage>();
 +    diskUsages.add(new DiskUsage(new TreeSet<String>(tables), 0l));
 +
 +    return diskUsages;
 +  }
 +
 +  @Override
 +  public void merge(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.merge(tableName, start, end);
 +  }
 +
 +  @Override
 +  public void deleteRows(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    MockTable t = acu.tables.get(tableName);
 +    Text startText = start != null ? new Text(start) : new Text();
 +    Text endText = end != null ? new Text(end) : new Text(t.table.lastKey().getRow().getBytes());
 +    startText.append(ZERO, 0, 1);
 +    endText.append(ZERO, 0, 1);
 +    Set<Key> keep = new TreeSet<Key>(t.table.subMap(new Key(startText), new Key(endText)).keySet());
 +    t.table.keySet().removeAll(keep);
 +  }
 +
 +  @Override
 +  public void compact(String tableName, Text start, Text end, boolean flush, boolean wait) throws AccumuloSecurityException, TableNotFoundException,
 +      AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void compact(String tableName, Text start, Text end, List<IteratorSetting> iterators, boolean flush, boolean wait) throws AccumuloSecurityException,
 +      TableNotFoundException, AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void cancelCompaction(String tableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void clone(String srcTableName, String newTableName, boolean flush, Map<String,String> propertiesToSet, Set<String> propertiesToExclude)
 +      throws AccumuloException, AccumuloSecurityException, TableNotFoundException, TableExistsException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public void flush(String tableName, Text start, Text end, boolean wait) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public Text getMaxRow(String tableName, Authorizations auths, Text startRow, boolean startInclusive, Text endRow, boolean endInclusive)
 +      throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    MockTable table = acu.tables.get(tableName);
 +    if (table == null)
 +      throw new TableNotFoundException(tableName, tableName, "no such table");
 +
 +    return FindMax.findMax(new MockScanner(table, auths), startRow, startInclusive, endRow, endInclusive);
 +  }
 +
 +  @Override
 +  public void importTable(String tableName, String exportDir) throws TableExistsException, AccumuloException, AccumuloSecurityException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public void exportTable(String tableName, String exportDir) throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public boolean testClassLoad(String tableName, String className, String asTypeName) throws AccumuloException, AccumuloSecurityException,
 +      TableNotFoundException {
 +
 +    try {
 +      AccumuloVFSClassLoader.loadClass(className, Class.forName(asTypeName));
 +    } catch (ClassNotFoundException e) {
 +      e.printStackTrace();
 +      return false;
 +    }
 +    return true;
 +  }
 +}

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
----------------------------------------------------------------------
diff --cc server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
index 82e5885,0000000..3642bc3
mode 100644,000000..100644
--- a/server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
+++ b/server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
@@@ -1,165 -1,0 +1,168 @@@
 +/*
 + * 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.accumulo.server.master.balancer;
 +
 +import java.util.ArrayList;
 +import java.util.Collections;
 +import java.util.HashMap;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Set;
 +import java.util.SortedMap;
 +import java.util.TreeMap;
 +
 +import org.apache.accumulo.core.client.Connector;
 +import org.apache.accumulo.core.client.admin.TableOperations;
 +import org.apache.accumulo.core.client.impl.thrift.ThriftSecurityException;
 +import org.apache.accumulo.core.client.mock.MockInstance;
 +import org.apache.accumulo.core.client.security.tokens.PasswordToken;
 +import org.apache.accumulo.core.data.KeyExtent;
 +import org.apache.accumulo.core.master.thrift.TableInfo;
 +import org.apache.accumulo.core.master.thrift.TabletServerStatus;
 +import org.apache.accumulo.core.tabletserver.thrift.TabletStats;
 +import org.apache.accumulo.server.master.state.TServerInstance;
 +import org.apache.accumulo.server.master.state.TabletMigration;
 +import org.apache.hadoop.io.Text;
 +import org.apache.thrift.TException;
 +import org.junit.Assert;
 +import org.junit.Test;
 +
 +import com.google.common.net.HostAndPort;
 +
 +public class TableLoadBalancerTest {
 +  
 +  static private TServerInstance mkts(String address, String session) throws Exception {
 +    return new TServerInstance(HostAndPort.fromParts(address, 1234), session);
 +  }
 +  
 +  static private TabletServerStatus status(Object... config) {
 +    TabletServerStatus result = new TabletServerStatus();
 +    result.tableMap = new HashMap<String,TableInfo>();
 +    String tablename = null;
 +    for (Object c : config) {
 +      if (c instanceof String) {
 +        tablename = (String) c;
 +      } else {
 +        TableInfo info = new TableInfo();
 +        int count = (Integer) c;
 +        info.onlineTablets = count;
 +        info.tablets = count;
 +        result.tableMap.put(tablename, info);
 +      }
 +    }
 +    return result;
 +  }
 +  
 +  static MockInstance instance = new MockInstance("mockamatic");
 +  
 +  static SortedMap<TServerInstance,TabletServerStatus> state;
 +  
 +  static List<TabletStats> generateFakeTablets(TServerInstance tserver, String tableId) {
 +    List<TabletStats> result = new ArrayList<TabletStats>();
 +    TabletServerStatus tableInfo = state.get(tserver);
 +    // generate some fake tablets
 +    for (int i = 0; i < tableInfo.tableMap.get(tableId).onlineTablets; i++) {
 +      TabletStats stats = new TabletStats();
 +      stats.extent = new KeyExtent(new Text(tableId), new Text(tserver.host() + String.format("%03d", i + 1)), new Text(tserver.host()
 +          + String.format("%03d", i))).toThrift();
 +      result.add(stats);
 +    }
 +    return result;
 +  }
 +  
 +  static class DefaultLoadBalancer extends org.apache.accumulo.server.master.balancer.DefaultLoadBalancer {
 +    
 +    public DefaultLoadBalancer(String table) {
 +      super(table);
 +    }
 +    
 +    @Override
 +    public List<TabletStats> getOnlineTabletsForTable(TServerInstance tserver, String tableId) throws ThriftSecurityException, TException {
 +      return generateFakeTablets(tserver, tableId);
 +    }
 +  }
 +  
 +  // ugh... so wish I had provided mock objects to the LoadBalancer in the master
 +  static class TableLoadBalancer extends org.apache.accumulo.server.master.balancer.TableLoadBalancer {
 +    
 +    TableLoadBalancer() {
 +      super();
 +    }
 +    
 +    // need to use our mock instance
 +    @Override
 +    protected TableOperations getTableOperations() {
 +      try {
 +        return instance.getConnector("user", new PasswordToken("pass")).tableOperations();
 +      } catch (Exception e) {
 +        throw new RuntimeException(e);
 +      }
 +    }
 +    
 +    // use our new classname to test class loading
 +    @Override
 +    protected String getLoadBalancerClassNameForTable(String table) {
 +      return DefaultLoadBalancer.class.getName();
 +    }
 +    
 +    // we don't have real tablet servers to ask: invent some online tablets
 +    @Override
 +    public List<TabletStats> getOnlineTabletsForTable(TServerInstance tserver, String tableId) throws ThriftSecurityException, TException {
 +      return generateFakeTablets(tserver, tableId);
 +    }
 +  }
 +  
 +  @Test
 +  public void test() throws Exception {
 +    Connector c = instance.getConnector("user", new PasswordToken("pass"));
-     c.tableOperations().create("t1");
-     c.tableOperations().create("t2");
-     c.tableOperations().create("t3");
++    TableOperations tops = c.tableOperations();
++    tops.create("t1");
++    tops.create("t2");
++    tops.create("t3");
++    String t1Id = tops.tableIdMap().get("t1"), t2Id = tops.tableIdMap().get("t2"), t3Id = tops.tableIdMap().get("t3");
 +    state = new TreeMap<TServerInstance,TabletServerStatus>();
 +    TServerInstance svr = mkts("10.0.0.1", "0x01020304");
 +    state.put(svr, status("t1", 10, "t2", 10, "t3", 10));
++    state.put(svr, status(t1Id, 10, t2Id, 10, t3Id, 10));
 +    
 +    Set<KeyExtent> migrations = Collections.emptySet();
 +    List<TabletMigration> migrationsOut = new ArrayList<TabletMigration>();
 +    TableLoadBalancer tls = new TableLoadBalancer();
 +    tls.balance(state, migrations, migrationsOut);
 +    Assert.assertEquals(0, migrationsOut.size());
 +    
 +    state.put(mkts("10.0.0.2", "0x02030405"), status());
 +    tls = new TableLoadBalancer();
 +    tls.balance(state, migrations, migrationsOut);
 +    int count = 0;
 +    Map<String,Integer> movedByTable = new HashMap<String,Integer>();
-     movedByTable.put("t1", new Integer(0));
-     movedByTable.put("t2", new Integer(0));
-     movedByTable.put("t3", new Integer(0));
++    movedByTable.put(t1Id, new Integer(0));
++    movedByTable.put(t2Id, new Integer(0));
++    movedByTable.put(t3Id, new Integer(0));
 +    for (TabletMigration migration : migrationsOut) {
 +      if (migration.oldServer.equals(svr))
 +        count++;
 +      String key = migration.tablet.getTableId().toString();
 +      movedByTable.put(key, movedByTable.get(key) + 1);
 +    }
 +    Assert.assertEquals(15, count);
 +    for (Integer moved : movedByTable.values()) {
 +      Assert.assertEquals(5, moved.intValue());
 +    }
 +  }
 +  
 +}


[2/6] git commit: ACCUMULO-2857 Return a more real tableID

Posted by el...@apache.org.
ACCUMULO-2857 Return a more real tableID


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

Branch: refs/heads/1.6.1-SNAPSHOT
Commit: 9fcca2ede18a7b96bee4edb5ab105316ac383604
Parents: 6dfcf8f
Author: Josh Elser <el...@apache.org>
Authored: Thu Jun 5 12:53:26 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Jun 5 12:53:26 2014 -0400

----------------------------------------------------------------------
 .../accumulo/core/client/mock/MockAccumulo.java     |  4 +++-
 .../apache/accumulo/core/client/mock/MockTable.java |  8 +++++++-
 .../core/client/mock/MockTableOperations.java       |  4 ++--
 .../core/client/mock/MockTableOperationsTest.java   | 11 +++++++++++
 .../master/balancer/TableLoadBalancerTest.java      | 16 +++++++++-------
 5 files changed, 32 insertions(+), 11 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
index 5977d1d..272d1af 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
@@ -21,6 +21,7 @@ import java.util.EnumSet;
 import java.util.HashMap;
 import java.util.Map;
 import java.util.SortedSet;
+import java.util.concurrent.atomic.AtomicInteger;
 
 import org.apache.accumulo.core.Constants;
 import org.apache.accumulo.core.client.BatchScanner;
@@ -38,6 +39,7 @@ public class MockAccumulo {
   final Map<String,String> systemProperties = new HashMap<String,String>();
   Map<String,MockUser> users = new HashMap<String,MockUser>();
   final FileSystem fs;
+  final AtomicInteger tableIdCounter = new AtomicInteger(0);
   
   MockAccumulo(FileSystem fs) {
     this.fs = fs;
@@ -76,7 +78,7 @@ public class MockAccumulo {
   }
   
   public void createTable(String username, String tableName, boolean useVersions, TimeType timeType) {
-    MockTable t = new MockTable(useVersions, timeType);
+    MockTable t = new MockTable(useVersions, timeType, Integer.toString(tableIdCounter.incrementAndGet()));
     t.userPermissions.put(username, EnumSet.allOf(TablePermission.class));
     tables.put(tableName, t);
   }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
index 3dcab11..2e13d84 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
@@ -89,9 +89,11 @@ public class MockTable {
   private TimeType timeType;
   SortedSet<Text> splits = new ConcurrentSkipListSet<Text>();
   Map<String,Set<Text>> localityGroups = new TreeMap<String, Set<Text>>();
+  private String tableId;
   
-  MockTable(boolean limitVersion, TimeType timeType) {
+  MockTable(boolean limitVersion, TimeType timeType, String tableId) {
     this.timeType = timeType;
+    this.tableId = tableId;
     settings = IteratorUtil.generateInitialTableProperties(limitVersion);
     for (Entry<String,String> entry : AccumuloConfiguration.getDefaultConfiguration()) {
       String key = entry.getKey();
@@ -143,4 +145,8 @@ public class MockTable {
     if (reAdd)
       splits.add(start);
   }
+
+  public String getTableId() {
+    return this.tableId;
+  }
 }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
----------------------------------------------------------------------
diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
index 64f8225..5b15351 100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
@@ -296,8 +296,8 @@ public class MockTableOperations extends TableOperationsHelper {
   @Override
   public Map<String,String> tableIdMap() {
     Map<String,String> result = new HashMap<String,String>();
-    for (String table : acu.tables.keySet()) {
-      result.put(table, table);
+    for (Entry<String,MockTable> entry : acu.tables.entrySet()) {
+      result.put(entry.getKey(), entry.getValue().getTableId());
     }
     return result;
   }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
----------------------------------------------------------------------
diff --git a/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java b/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
index ea916e7..cea66a4 100644
--- a/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
+++ b/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
@@ -333,4 +333,15 @@ public class MockTableOperationsTest {
 
   }
 
+
+  @Test
+  public void testTableIdMap() throws Exception {
+    Instance inst = new MockInstance("testTableIdMap");
+    Connector conn = inst.getConnector("root", new PasswordToken(""));
+    TableOperations tops = conn.tableOperations();
+    tops.create("foo");
+
+    // Should get a table ID, not the table name
+    Assert.assertNotEquals("foo", tops.tableIdMap().get("foo"));
+  }
 }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/9fcca2ed/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
----------------------------------------------------------------------
diff --git a/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java b/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
index 71ba50d..b7c6412 100644
--- a/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
+++ b/server/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
@@ -128,12 +128,14 @@ public class TableLoadBalancerTest {
   @Test
   public void test() throws Exception {
     Connector c = instance.getConnector("user", new PasswordToken("pass"));
-    c.tableOperations().create("t1");
-    c.tableOperations().create("t2");
-    c.tableOperations().create("t3");
+    TableOperations tops = c.tableOperations();
+    tops.create("t1");
+    tops.create("t2");
+    tops.create("t3");
+    String t1Id = tops.tableIdMap().get("t1"), t2Id = tops.tableIdMap().get("t2"), t3Id = tops.tableIdMap().get("t3");
     state = new TreeMap<TServerInstance,TabletServerStatus>();
     TServerInstance svr = mkts("10.0.0.1:1234", "0x01020304");
-    state.put(svr, status("t1", 10, "t2", 10, "t3", 10));
+    state.put(svr, status(t1Id, 10, t2Id, 10, t3Id, 10));
     
     Set<KeyExtent> migrations = Collections.emptySet();
     List<TabletMigration> migrationsOut = new ArrayList<TabletMigration>();
@@ -146,9 +148,9 @@ public class TableLoadBalancerTest {
     tls.balance(state, migrations, migrationsOut);
     int count = 0;
     Map<String,Integer> movedByTable = new HashMap<String,Integer>();
-    movedByTable.put("t1", new Integer(0));
-    movedByTable.put("t2", new Integer(0));
-    movedByTable.put("t3", new Integer(0));
+    movedByTable.put(t1Id, new Integer(0));
+    movedByTable.put(t2Id, new Integer(0));
+    movedByTable.put(t3Id, new Integer(0));
     for (TabletMigration migration : migrationsOut) {
       if (migration.oldServer.equals(svr))
         count++;


[4/6] git commit: Merge branch '1.5.2-SNAPSHOT' into 1.6.1-SNAPSHOT

Posted by el...@apache.org.
Merge branch '1.5.2-SNAPSHOT' into 1.6.1-SNAPSHOT

Conflicts:
	core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
	core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
	core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
	server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java


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

Branch: refs/heads/master
Commit: 38ff0000879af97d24372cb004a81e10901e4276
Parents: 05cf918 9fcca2e
Author: Josh Elser <el...@apache.org>
Authored: Thu Jun 5 13:21:43 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Jun 5 13:21:43 2014 -0400

----------------------------------------------------------------------
 .../accumulo/core/client/mock/MockAccumulo.java      |  4 +++-
 .../apache/accumulo/core/client/mock/MockTable.java  | 12 +++++++++---
 .../core/client/mock/MockTableOperationsImpl.java    |  5 +++--
 .../core/client/mock/MockTableOperationsTest.java    | 11 +++++++++++
 .../master/balancer/TableLoadBalancerTest.java       | 15 +++++++++------
 5 files changed, 35 insertions(+), 12 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
----------------------------------------------------------------------
diff --cc core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
index 2c26ecc,272d1af..32dbb28
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockAccumulo.java
@@@ -21,17 -21,14 +21,18 @@@ import java.util.EnumSet
  import java.util.HashMap;
  import java.util.Map;
  import java.util.SortedSet;
+ import java.util.concurrent.atomic.AtomicInteger;
  
 -import org.apache.accumulo.core.Constants;
  import org.apache.accumulo.core.client.BatchScanner;
  import org.apache.accumulo.core.client.admin.TimeType;
 +import org.apache.accumulo.core.client.impl.Namespaces;
 +import org.apache.accumulo.core.client.impl.Tables;
  import org.apache.accumulo.core.client.security.tokens.PasswordToken;
  import org.apache.accumulo.core.data.Mutation;
 +import org.apache.accumulo.core.metadata.MetadataTable;
 +import org.apache.accumulo.core.metadata.RootTable;
  import org.apache.accumulo.core.security.Authorizations;
 +import org.apache.accumulo.core.security.NamespacePermission;
  import org.apache.accumulo.core.security.SystemPermission;
  import org.apache.accumulo.core.security.TablePermission;
  import org.apache.hadoop.fs.FileSystem;
@@@ -43,7 -39,8 +44,8 @@@ public class MockAccumulo 
    final Map<String,String> systemProperties = new HashMap<String,String>();
    Map<String,MockUser> users = new HashMap<String,MockUser>();
    final FileSystem fs;
+   final AtomicInteger tableIdCounter = new AtomicInteger(0);
 -  
 +
    MockAccumulo(FileSystem fs) {
      this.fs = fs;
    }
@@@ -82,30 -76,13 +84,30 @@@
    public BatchScanner createBatchScanner(String tableName, Authorizations authorizations) {
      return new MockBatchScanner(tables.get(tableName), authorizations);
    }
 -  
 +
    public void createTable(String username, String tableName, boolean useVersions, TimeType timeType) {
 -    MockTable t = new MockTable(useVersions, timeType, Integer.toString(tableIdCounter.incrementAndGet()));
 +    String namespace = Tables.qualify(tableName).getFirst();
 +
 +    if (!namespaceExists(namespace)) {
 +      return;
 +    }
 +
 +    MockNamespace n = namespaces.get(namespace);
-     MockTable t = new MockTable(n, useVersions, timeType);
++    MockTable t = new MockTable(n, useVersions, timeType, Integer.toString(tableIdCounter.incrementAndGet()));
      t.userPermissions.put(username, EnumSet.allOf(TablePermission.class));
 +    t.setNamespaceName(namespace);
 +    t.setNamespace(n);
      tables.put(tableName, t);
    }
 -  
 +
 +  public void createNamespace(String username, String namespace) {
 +    if (!namespaceExists(namespace)) {
 +      MockNamespace n = new MockNamespace();
 +      n.userPermissions.put(username, EnumSet.allOf(NamespacePermission.class));
 +      namespaces.put(namespace, n);
 +    }
 +  }
 +
    public void addSplits(String tableName, SortedSet<Text> partitionKeys) {
      tables.get(tableName).addSplits(partitionKeys);
    }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
----------------------------------------------------------------------
diff --cc core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
index cb50761,2e13d84..35cbdd2
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTable.java
@@@ -89,12 -88,12 +89,14 @@@ public class MockTable 
    Map<String,EnumSet<TablePermission>> userPermissions = new HashMap<String,EnumSet<TablePermission>>();
    private TimeType timeType;
    SortedSet<Text> splits = new ConcurrentSkipListSet<Text>();
 -  Map<String,Set<Text>> localityGroups = new TreeMap<String, Set<Text>>();
 +  Map<String,Set<Text>> localityGroups = new TreeMap<String,Set<Text>>();
 +  private MockNamespace namespace;
 +  private String namespaceName;
+   private String tableId;
    
-   MockTable(boolean limitVersion, TimeType timeType) {
+   MockTable(boolean limitVersion, TimeType timeType, String tableId) {
      this.timeType = timeType;
+     this.tableId = tableId;
      settings = IteratorUtil.generateInitialTableProperties(limitVersion);
      for (Entry<String,String> entry : AccumuloConfiguration.getDefaultConfiguration()) {
        String key = entry.getKey();
@@@ -103,18 -102,6 +105,18 @@@
      }
    }
    
-   MockTable(MockNamespace namespace, boolean limitVersion, TimeType timeType) {
-     this(limitVersion, timeType);
++  MockTable(MockNamespace namespace, boolean limitVersion, TimeType timeType, String tableId) {
++    this(limitVersion, timeType, tableId);
 +    Set<Entry<String,String>> set = namespace.settings.entrySet();
 +    Iterator<Entry<String,String>> entries = set.iterator();
 +    while (entries.hasNext()) {
 +      Entry<String,String> entry = entries.next();
 +      String key = entry.getKey();
 +      if (key.startsWith(Property.TABLE_PREFIX.getKey()))
 +        settings.put(key, entry.getValue());
 +    }
 +  }
 +  
    synchronized void addMutation(Mutation m) {
      if (m.size() == 0)
        throw new IllegalArgumentException("Can not add empty mutations");
@@@ -159,20 -145,8 +161,24 @@@
      if (reAdd)
        splits.add(start);
    }
 +  
 +  public void setNamespaceName(String n) {
 +    this.namespaceName = n;
 +  }
 +  
 +  public void setNamespace(MockNamespace n) {
 +    this.namespace = n;
 +  }
 +  
 +  public String getNamespaceName() {
 +    return this.namespaceName;
 +  }
 +  
 +  public MockNamespace getNamespace() {
 +    return this.namespace;
 +  }
+ 
+   public String getTableId() {
+     return this.tableId;
+   }
  }

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
----------------------------------------------------------------------
diff --cc core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
index 8a8895f,0000000..de19137
mode 100644,000000..100644
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java
@@@ -1,447 -1,0 +1,448 @@@
 +/*
 + * 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.accumulo.core.client.mock;
 +
 +import java.io.DataInputStream;
 +import java.io.IOException;
 +import java.util.ArrayList;
 +import java.util.Collection;
 +import java.util.Collections;
 +import java.util.HashMap;
 +import java.util.HashSet;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Set;
 +import java.util.SortedSet;
 +import java.util.TreeSet;
 +
 +import org.apache.accumulo.core.client.AccumuloException;
 +import org.apache.accumulo.core.client.AccumuloSecurityException;
 +import org.apache.accumulo.core.client.IteratorSetting;
 +import org.apache.accumulo.core.client.NamespaceNotFoundException;
 +import org.apache.accumulo.core.client.TableExistsException;
 +import org.apache.accumulo.core.client.TableNotFoundException;
 +import org.apache.accumulo.core.client.admin.DiskUsage;
 +import org.apache.accumulo.core.client.admin.FindMax;
 +import org.apache.accumulo.core.client.impl.TableOperationsHelper;
 +import org.apache.accumulo.core.client.admin.TimeType;
 +import org.apache.accumulo.core.client.impl.Tables;
 +import org.apache.accumulo.core.conf.AccumuloConfiguration;
 +import org.apache.accumulo.core.data.Key;
 +import org.apache.accumulo.core.data.Mutation;
 +import org.apache.accumulo.core.data.Range;
 +import org.apache.accumulo.core.data.Value;
 +import org.apache.accumulo.core.file.FileOperations;
 +import org.apache.accumulo.core.file.FileSKVIterator;
 +import org.apache.accumulo.core.metadata.MetadataTable;
 +import org.apache.accumulo.core.metadata.RootTable;
 +import org.apache.accumulo.core.security.Authorizations;
 +import org.apache.accumulo.core.security.ColumnVisibility;
 +import org.apache.accumulo.start.classloader.vfs.AccumuloVFSClassLoader;
 +import org.apache.commons.lang.NotImplementedException;
 +import org.apache.hadoop.fs.FSDataOutputStream;
 +import org.apache.hadoop.fs.FileStatus;
 +import org.apache.hadoop.fs.FileSystem;
 +import org.apache.hadoop.fs.Path;
 +import org.apache.hadoop.io.Text;
 +
 +class MockTableOperationsImpl extends TableOperationsHelper {
 +  private static final byte[] ZERO = {0};
 +  private final MockAccumulo acu;
 +  private final String username;
 +
 +  MockTableOperationsImpl(MockAccumulo acu, String username) {
 +    this.acu = acu;
 +    this.username = username;
 +  }
 +
 +  @Override
 +  public SortedSet<String> list() {
 +    return new TreeSet<String>(acu.tables.keySet());
 +  }
 +
 +  @Override
 +  public boolean exists(String tableName) {
 +    return acu.tables.containsKey(tableName);
 +  }
 +
 +  private boolean namespaceExists(String namespace) {
 +    return acu.namespaces.containsKey(namespace);
 +  }
 +
 +  @Override
 +  public void create(String tableName) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    create(tableName, true, TimeType.MILLIS);
 +  }
 +
 +  @Override
 +  public void create(String tableName, boolean versioningIter) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    create(tableName, versioningIter, TimeType.MILLIS);
 +  }
 +
 +  @Override
 +  public void create(String tableName, boolean versioningIter, TimeType timeType) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    String namespace = Tables.qualify(tableName).getFirst();
 +    if (!tableName.matches(Tables.VALID_NAME_REGEX)) {
 +      throw new IllegalArgumentException();
 +    }
 +    if (exists(tableName))
 +      throw new TableExistsException(tableName, tableName, "");
 +
 +    if (!namespaceExists(namespace)) {
 +      throw new IllegalArgumentException("Namespace (" + namespace + ") does not exist, create it first");
 +    }
 +    acu.createTable(username, tableName, versioningIter, timeType);
 +  }
 +
 +  @Override
 +  public void addSplits(String tableName, SortedSet<Text> partitionKeys) throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.addSplits(tableName, partitionKeys);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public Collection<Text> getSplits(String tableName) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public Collection<Text> getSplits(String tableName, int maxSplits) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Override
 +  public Collection<Text> listSplits(String tableName) throws TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return acu.getSplits(tableName);
 +  }
 +
 +  @Override
 +  public Collection<Text> listSplits(String tableName, int maxSplits) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Override
 +  public void delete(String tableName) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.tables.remove(tableName);
 +  }
 +
 +  @Override
 +  public void rename(String oldTableName, String newTableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException,
 +      TableExistsException {
 +    if (!exists(oldTableName))
 +      throw new TableNotFoundException(oldTableName, oldTableName, "");
 +    if (exists(newTableName))
 +      throw new TableExistsException(newTableName, newTableName, "");
 +    MockTable t = acu.tables.remove(oldTableName);
 +    String namespace = Tables.qualify(newTableName).getFirst();
 +    MockNamespace n = acu.namespaces.get(namespace);
 +    if (n == null) {
 +      n = new MockNamespace();
 +    }
 +    t.setNamespaceName(namespace);
 +    t.setNamespace(n);
 +    acu.namespaces.put(namespace, n);
 +    acu.tables.put(newTableName, t);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public void flush(String tableName) throws AccumuloException, AccumuloSecurityException {}
 +
 +  @Override
 +  public void setProperty(String tableName, String property, String value) throws AccumuloException, AccumuloSecurityException {
 +    acu.tables.get(tableName).settings.put(property, value);
 +  }
 +
 +  @Override
 +  public void removeProperty(String tableName, String property) throws AccumuloException, AccumuloSecurityException {
 +    acu.tables.get(tableName).settings.remove(property);
 +  }
 +
 +  @Override
 +  public Iterable<Entry<String,String>> getProperties(String tableName) throws TableNotFoundException {
 +    String namespace = Tables.qualify(tableName).getFirst();
 +    if (!exists(tableName)) {
 +      if (!namespaceExists(namespace))
 +        throw new TableNotFoundException(tableName, new NamespaceNotFoundException(null, namespace, null));
 +      throw new TableNotFoundException(null, tableName, null);
 +    }
 +
 +    Set<Entry<String,String>> props = new HashSet<Entry<String,String>>(acu.namespaces.get(namespace).settings.entrySet());
 +
 +    Set<Entry<String,String>> tableProps = acu.tables.get(tableName).settings.entrySet();
 +    for (Entry<String,String> e : tableProps) {
 +      if (props.contains(e)) {
 +        props.remove(e);
 +      }
 +      props.add(e);
 +    }
 +    return props;
 +  }
 +
 +  @Override
 +  public void setLocalityGroups(String tableName, Map<String,Set<Text>> groups) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.tables.get(tableName).setLocalityGroups(groups);
 +  }
 +
 +  @Override
 +  public Map<String,Set<Text>> getLocalityGroups(String tableName) throws AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return acu.tables.get(tableName).getLocalityGroups();
 +  }
 +
 +  @Override
 +  public Set<Range> splitRangeByTablets(String tableName, Range range, int maxSplits) throws AccumuloException, AccumuloSecurityException,
 +      TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return Collections.singleton(range);
 +  }
 +
 +  @Override
 +  public void importDirectory(String tableName, String dir, String failureDir, boolean setTime) throws IOException, AccumuloException,
 +      AccumuloSecurityException, TableNotFoundException {
 +    long time = System.currentTimeMillis();
 +    MockTable table = acu.tables.get(tableName);
 +    if (table == null) {
 +      throw new TableNotFoundException(null, tableName, "The table was not found");
 +    }
 +    Path importPath = new Path(dir);
 +    Path failurePath = new Path(failureDir);
 +
 +    FileSystem fs = acu.getFileSystem();
 +    /*
 +     * check preconditions
 +     */
 +    // directories are directories
 +    if (fs.isFile(importPath)) {
 +      throw new IOException("Import path must be a directory.");
 +    }
 +    if (fs.isFile(failurePath)) {
 +      throw new IOException("Failure path must be a directory.");
 +    }
 +    // failures are writable
 +    Path createPath = failurePath.suffix("/.createFile");
 +    FSDataOutputStream createStream = null;
 +    try {
 +      createStream = fs.create(createPath);
 +    } catch (IOException e) {
 +      throw new IOException("Error path is not writable.");
 +    } finally {
 +      if (createStream != null) {
 +        createStream.close();
 +      }
 +    }
 +    fs.delete(createPath, false);
 +    // failures are empty
 +    FileStatus[] failureChildStats = fs.listStatus(failurePath);
 +    if (failureChildStats.length > 0) {
 +      throw new IOException("Error path must be empty.");
 +    }
 +    /*
 +     * Begin the import - iterate the files in the path
 +     */
 +    for (FileStatus importStatus : fs.listStatus(importPath)) {
 +      try {
 +        FileSKVIterator importIterator = FileOperations.getInstance().openReader(importStatus.getPath().toString(), true, fs, fs.getConf(),
 +            AccumuloConfiguration.getDefaultConfiguration());
 +        while (importIterator.hasTop()) {
 +          Key key = importIterator.getTopKey();
 +          Value value = importIterator.getTopValue();
 +          if (setTime) {
 +            key.setTimestamp(time);
 +          }
 +          Mutation mutation = new Mutation(key.getRow());
 +          if (!key.isDeleted()) {
 +            mutation.put(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()), key.getTimestamp(),
 +                value);
 +          } else {
 +            mutation.putDelete(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()),
 +                key.getTimestamp());
 +          }
 +          table.addMutation(mutation);
 +          importIterator.next();
 +        }
 +      } catch (Exception e) {
 +        FSDataOutputStream failureWriter = null;
 +        DataInputStream failureReader = null;
 +        try {
 +          failureWriter = fs.create(failurePath.suffix("/" + importStatus.getPath().getName()));
 +          failureReader = fs.open(importStatus.getPath());
 +          int read = 0;
 +          byte[] buffer = new byte[1024];
 +          while (-1 != (read = failureReader.read(buffer))) {
 +            failureWriter.write(buffer, 0, read);
 +          }
 +        } finally {
 +          if (failureReader != null)
 +            failureReader.close();
 +          if (failureWriter != null)
 +            failureWriter.close();
 +        }
 +      }
 +      fs.delete(importStatus.getPath(), true);
 +    }
 +  }
 +
 +  @Override
 +  public void offline(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    offline(tableName, false);
 +  }
 +
 +  @Override
 +  public void offline(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void online(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    online(tableName, false);
 +  }
 +
 +  @Override
 +  public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void clearLocatorCache(String tableName) throws TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public Map<String,String> tableIdMap() {
 +    Map<String,String> result = new HashMap<String,String>();
-     for (String table : acu.tables.keySet()) {
++    for (Entry<String,MockTable> entry : acu.tables.entrySet()) {
++      String table = entry.getKey();
 +      if (RootTable.NAME.equals(table))
 +        result.put(table, RootTable.ID);
 +      else if (MetadataTable.NAME.equals(table))
 +        result.put(table, MetadataTable.ID);
 +      else
-         result.put(table, table);
++        result.put(table, entry.getValue().getTableId());
 +    }
 +    return result;
 +  }
 +
 +  @Override
 +  public List<DiskUsage> getDiskUsage(Set<String> tables) throws AccumuloException, AccumuloSecurityException {
 +
 +    List<DiskUsage> diskUsages = new ArrayList<DiskUsage>();
 +    diskUsages.add(new DiskUsage(new TreeSet<String>(tables), 0l));
 +
 +    return diskUsages;
 +  }
 +
 +  @Override
 +  public void merge(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.merge(tableName, start, end);
 +  }
 +
 +  @Override
 +  public void deleteRows(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    MockTable t = acu.tables.get(tableName);
 +    Text startText = start != null ? new Text(start) : new Text();
 +    Text endText = end != null ? new Text(end) : new Text(t.table.lastKey().getRow().getBytes());
 +    startText.append(ZERO, 0, 1);
 +    endText.append(ZERO, 0, 1);
 +    Set<Key> keep = new TreeSet<Key>(t.table.subMap(new Key(startText), new Key(endText)).keySet());
 +    t.table.keySet().removeAll(keep);
 +  }
 +
 +  @Override
 +  public void compact(String tableName, Text start, Text end, boolean flush, boolean wait) throws AccumuloSecurityException, TableNotFoundException,
 +      AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void compact(String tableName, Text start, Text end, List<IteratorSetting> iterators, boolean flush, boolean wait) throws AccumuloSecurityException,
 +      TableNotFoundException, AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void cancelCompaction(String tableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void clone(String srcTableName, String newTableName, boolean flush, Map<String,String> propertiesToSet, Set<String> propertiesToExclude)
 +      throws AccumuloException, AccumuloSecurityException, TableNotFoundException, TableExistsException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public void flush(String tableName, Text start, Text end, boolean wait) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public Text getMaxRow(String tableName, Authorizations auths, Text startRow, boolean startInclusive, Text endRow, boolean endInclusive)
 +      throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    MockTable table = acu.tables.get(tableName);
 +    if (table == null)
 +      throw new TableNotFoundException(tableName, tableName, "no such table");
 +
 +    return FindMax.findMax(new MockScanner(table, auths), startRow, startInclusive, endRow, endInclusive);
 +  }
 +
 +  @Override
 +  public void importTable(String tableName, String exportDir) throws TableExistsException, AccumuloException, AccumuloSecurityException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public void exportTable(String tableName, String exportDir) throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public boolean testClassLoad(String tableName, String className, String asTypeName) throws AccumuloException, AccumuloSecurityException,
 +      TableNotFoundException {
 +
 +    try {
 +      AccumuloVFSClassLoader.loadClass(className, Class.forName(asTypeName));
 +    } catch (ClassNotFoundException e) {
 +      e.printStackTrace();
 +      return false;
 +    }
 +    return true;
 +  }
 +}

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/core/src/test/java/org/apache/accumulo/core/client/mock/MockTableOperationsTest.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/accumulo/blob/38ff0000/server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
----------------------------------------------------------------------
diff --cc server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
index 82e5885,0000000..3642bc3
mode 100644,000000..100644
--- a/server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
+++ b/server/base/src/test/java/org/apache/accumulo/server/master/balancer/TableLoadBalancerTest.java
@@@ -1,165 -1,0 +1,168 @@@
 +/*
 + * 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.accumulo.server.master.balancer;
 +
 +import java.util.ArrayList;
 +import java.util.Collections;
 +import java.util.HashMap;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Set;
 +import java.util.SortedMap;
 +import java.util.TreeMap;
 +
 +import org.apache.accumulo.core.client.Connector;
 +import org.apache.accumulo.core.client.admin.TableOperations;
 +import org.apache.accumulo.core.client.impl.thrift.ThriftSecurityException;
 +import org.apache.accumulo.core.client.mock.MockInstance;
 +import org.apache.accumulo.core.client.security.tokens.PasswordToken;
 +import org.apache.accumulo.core.data.KeyExtent;
 +import org.apache.accumulo.core.master.thrift.TableInfo;
 +import org.apache.accumulo.core.master.thrift.TabletServerStatus;
 +import org.apache.accumulo.core.tabletserver.thrift.TabletStats;
 +import org.apache.accumulo.server.master.state.TServerInstance;
 +import org.apache.accumulo.server.master.state.TabletMigration;
 +import org.apache.hadoop.io.Text;
 +import org.apache.thrift.TException;
 +import org.junit.Assert;
 +import org.junit.Test;
 +
 +import com.google.common.net.HostAndPort;
 +
 +public class TableLoadBalancerTest {
 +  
 +  static private TServerInstance mkts(String address, String session) throws Exception {
 +    return new TServerInstance(HostAndPort.fromParts(address, 1234), session);
 +  }
 +  
 +  static private TabletServerStatus status(Object... config) {
 +    TabletServerStatus result = new TabletServerStatus();
 +    result.tableMap = new HashMap<String,TableInfo>();
 +    String tablename = null;
 +    for (Object c : config) {
 +      if (c instanceof String) {
 +        tablename = (String) c;
 +      } else {
 +        TableInfo info = new TableInfo();
 +        int count = (Integer) c;
 +        info.onlineTablets = count;
 +        info.tablets = count;
 +        result.tableMap.put(tablename, info);
 +      }
 +    }
 +    return result;
 +  }
 +  
 +  static MockInstance instance = new MockInstance("mockamatic");
 +  
 +  static SortedMap<TServerInstance,TabletServerStatus> state;
 +  
 +  static List<TabletStats> generateFakeTablets(TServerInstance tserver, String tableId) {
 +    List<TabletStats> result = new ArrayList<TabletStats>();
 +    TabletServerStatus tableInfo = state.get(tserver);
 +    // generate some fake tablets
 +    for (int i = 0; i < tableInfo.tableMap.get(tableId).onlineTablets; i++) {
 +      TabletStats stats = new TabletStats();
 +      stats.extent = new KeyExtent(new Text(tableId), new Text(tserver.host() + String.format("%03d", i + 1)), new Text(tserver.host()
 +          + String.format("%03d", i))).toThrift();
 +      result.add(stats);
 +    }
 +    return result;
 +  }
 +  
 +  static class DefaultLoadBalancer extends org.apache.accumulo.server.master.balancer.DefaultLoadBalancer {
 +    
 +    public DefaultLoadBalancer(String table) {
 +      super(table);
 +    }
 +    
 +    @Override
 +    public List<TabletStats> getOnlineTabletsForTable(TServerInstance tserver, String tableId) throws ThriftSecurityException, TException {
 +      return generateFakeTablets(tserver, tableId);
 +    }
 +  }
 +  
 +  // ugh... so wish I had provided mock objects to the LoadBalancer in the master
 +  static class TableLoadBalancer extends org.apache.accumulo.server.master.balancer.TableLoadBalancer {
 +    
 +    TableLoadBalancer() {
 +      super();
 +    }
 +    
 +    // need to use our mock instance
 +    @Override
 +    protected TableOperations getTableOperations() {
 +      try {
 +        return instance.getConnector("user", new PasswordToken("pass")).tableOperations();
 +      } catch (Exception e) {
 +        throw new RuntimeException(e);
 +      }
 +    }
 +    
 +    // use our new classname to test class loading
 +    @Override
 +    protected String getLoadBalancerClassNameForTable(String table) {
 +      return DefaultLoadBalancer.class.getName();
 +    }
 +    
 +    // we don't have real tablet servers to ask: invent some online tablets
 +    @Override
 +    public List<TabletStats> getOnlineTabletsForTable(TServerInstance tserver, String tableId) throws ThriftSecurityException, TException {
 +      return generateFakeTablets(tserver, tableId);
 +    }
 +  }
 +  
 +  @Test
 +  public void test() throws Exception {
 +    Connector c = instance.getConnector("user", new PasswordToken("pass"));
-     c.tableOperations().create("t1");
-     c.tableOperations().create("t2");
-     c.tableOperations().create("t3");
++    TableOperations tops = c.tableOperations();
++    tops.create("t1");
++    tops.create("t2");
++    tops.create("t3");
++    String t1Id = tops.tableIdMap().get("t1"), t2Id = tops.tableIdMap().get("t2"), t3Id = tops.tableIdMap().get("t3");
 +    state = new TreeMap<TServerInstance,TabletServerStatus>();
 +    TServerInstance svr = mkts("10.0.0.1", "0x01020304");
 +    state.put(svr, status("t1", 10, "t2", 10, "t3", 10));
++    state.put(svr, status(t1Id, 10, t2Id, 10, t3Id, 10));
 +    
 +    Set<KeyExtent> migrations = Collections.emptySet();
 +    List<TabletMigration> migrationsOut = new ArrayList<TabletMigration>();
 +    TableLoadBalancer tls = new TableLoadBalancer();
 +    tls.balance(state, migrations, migrationsOut);
 +    Assert.assertEquals(0, migrationsOut.size());
 +    
 +    state.put(mkts("10.0.0.2", "0x02030405"), status());
 +    tls = new TableLoadBalancer();
 +    tls.balance(state, migrations, migrationsOut);
 +    int count = 0;
 +    Map<String,Integer> movedByTable = new HashMap<String,Integer>();
-     movedByTable.put("t1", new Integer(0));
-     movedByTable.put("t2", new Integer(0));
-     movedByTable.put("t3", new Integer(0));
++    movedByTable.put(t1Id, new Integer(0));
++    movedByTable.put(t2Id, new Integer(0));
++    movedByTable.put(t3Id, new Integer(0));
 +    for (TabletMigration migration : migrationsOut) {
 +      if (migration.oldServer.equals(svr))
 +        count++;
 +      String key = migration.tablet.getTableId().toString();
 +      movedByTable.put(key, movedByTable.get(key) + 1);
 +    }
 +    Assert.assertEquals(15, count);
 +    for (Integer moved : movedByTable.values()) {
 +      Assert.assertEquals(5, moved.intValue());
 +    }
 +  }
 +  
 +}


[6/6] git commit: Merge branch '1.6.1-SNAPSHOT'

Posted by el...@apache.org.
Merge branch '1.6.1-SNAPSHOT'

Conflicts:
	core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperationsImpl.java


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

Branch: refs/heads/master
Commit: 2149b6430d8e15d4c11b5b310332365b8855874f
Parents: 47d5933 38ff000
Author: Josh Elser <el...@apache.org>
Authored: Thu Jun 5 13:29:47 2014 -0400
Committer: Josh Elser <el...@apache.org>
Committed: Thu Jun 5 13:29:47 2014 -0400

----------------------------------------------------------------------
 .../accumulo/core/client/mock/MockAccumulo.java      |  4 +++-
 .../apache/accumulo/core/client/mock/MockTable.java  | 12 +++++++++---
 .../core/client/mock/MockTableOperations.java        |  5 +++--
 .../core/client/mock/MockTableOperationsTest.java    | 11 +++++++++++
 .../master/balancer/TableLoadBalancerTest.java       | 15 +++++++++------
 5 files changed, 35 insertions(+), 12 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/accumulo/blob/2149b643/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
----------------------------------------------------------------------
diff --cc core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
index a44a027,ee8880c..b1bba45
--- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
+++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java
@@@ -16,432 -16,13 +16,433 @@@
   */
  package org.apache.accumulo.core.client.mock;
  
 -/**
 - * @deprecated since 1.6.0; not intended for public api and you should not use it.
 - */
 -@Deprecated
 -public class MockTableOperations extends MockTableOperationsImpl {
 +import java.io.DataInputStream;
 +import java.io.IOException;
 +import java.util.ArrayList;
 +import java.util.Collection;
 +import java.util.Collections;
 +import java.util.HashMap;
 +import java.util.HashSet;
 +import java.util.List;
 +import java.util.Map;
 +import java.util.Map.Entry;
 +import java.util.Set;
 +import java.util.SortedSet;
 +import java.util.TreeSet;
 +
 +import org.apache.accumulo.core.client.AccumuloException;
 +import org.apache.accumulo.core.client.AccumuloSecurityException;
 +import org.apache.accumulo.core.client.IteratorSetting;
 +import org.apache.accumulo.core.client.NamespaceNotFoundException;
 +import org.apache.accumulo.core.client.TableExistsException;
 +import org.apache.accumulo.core.client.TableNotFoundException;
 +import org.apache.accumulo.core.client.admin.DiskUsage;
 +import org.apache.accumulo.core.client.admin.FindMax;
 +import org.apache.accumulo.core.client.impl.TableOperationsHelper;
 +import org.apache.accumulo.core.client.admin.TimeType;
 +import org.apache.accumulo.core.client.impl.Tables;
 +import org.apache.accumulo.core.conf.AccumuloConfiguration;
 +import org.apache.accumulo.core.data.Key;
 +import org.apache.accumulo.core.data.Mutation;
 +import org.apache.accumulo.core.data.Range;
 +import org.apache.accumulo.core.data.Value;
 +import org.apache.accumulo.core.file.FileOperations;
 +import org.apache.accumulo.core.file.FileSKVIterator;
 +import org.apache.accumulo.core.metadata.MetadataTable;
 +import org.apache.accumulo.core.metadata.RootTable;
 +import org.apache.accumulo.core.security.Authorizations;
 +import org.apache.accumulo.core.security.ColumnVisibility;
 +import org.apache.accumulo.start.classloader.vfs.AccumuloVFSClassLoader;
 +import org.apache.commons.lang.NotImplementedException;
 +import org.apache.hadoop.fs.FSDataOutputStream;
 +import org.apache.hadoop.fs.FileStatus;
 +import org.apache.hadoop.fs.FileSystem;
 +import org.apache.hadoop.fs.Path;
 +import org.apache.hadoop.io.Text;
 +
 +class MockTableOperations extends TableOperationsHelper {
 +  private static final byte[] ZERO = {0};
 +  private final MockAccumulo acu;
 +  private final String username;
  
    MockTableOperations(MockAccumulo acu, String username) {
 -    super(acu,username);
 +    this.acu = acu;
 +    this.username = username;
 +  }
 +
 +  @Override
 +  public SortedSet<String> list() {
 +    return new TreeSet<String>(acu.tables.keySet());
 +  }
 +
 +  @Override
 +  public boolean exists(String tableName) {
 +    return acu.tables.containsKey(tableName);
 +  }
 +
 +  private boolean namespaceExists(String namespace) {
 +    return acu.namespaces.containsKey(namespace);
 +  }
 +
 +  @Override
 +  public void create(String tableName) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    create(tableName, true, TimeType.MILLIS);
 +  }
 +
 +  @Override
 +  public void create(String tableName, boolean versioningIter) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    create(tableName, versioningIter, TimeType.MILLIS);
 +  }
 +
 +  @Override
 +  public void create(String tableName, boolean versioningIter, TimeType timeType) throws AccumuloException, AccumuloSecurityException, TableExistsException {
 +    String namespace = Tables.qualify(tableName).getFirst();
 +    if (!tableName.matches(Tables.VALID_NAME_REGEX)) {
 +      throw new IllegalArgumentException();
 +    }
 +    if (exists(tableName))
 +      throw new TableExistsException(tableName, tableName, "");
 +
 +    if (!namespaceExists(namespace)) {
 +      throw new IllegalArgumentException("Namespace (" + namespace + ") does not exist, create it first");
 +    }
 +    acu.createTable(username, tableName, versioningIter, timeType);
 +  }
 +
 +  @Override
 +  public void addSplits(String tableName, SortedSet<Text> partitionKeys) throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.addSplits(tableName, partitionKeys);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public Collection<Text> getSplits(String tableName) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public Collection<Text> getSplits(String tableName, int maxSplits) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Override
 +  public Collection<Text> listSplits(String tableName) throws TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return acu.getSplits(tableName);
 +  }
 +
 +  @Override
 +  public Collection<Text> listSplits(String tableName, int maxSplits) throws TableNotFoundException {
 +    return listSplits(tableName);
 +  }
 +
 +  @Override
 +  public void delete(String tableName) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.tables.remove(tableName);
 +  }
 +
 +  @Override
 +  public void rename(String oldTableName, String newTableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException,
 +      TableExistsException {
 +    if (!exists(oldTableName))
 +      throw new TableNotFoundException(oldTableName, oldTableName, "");
 +    if (exists(newTableName))
 +      throw new TableExistsException(newTableName, newTableName, "");
 +    MockTable t = acu.tables.remove(oldTableName);
 +    String namespace = Tables.qualify(newTableName).getFirst();
 +    MockNamespace n = acu.namespaces.get(namespace);
 +    if (n == null) {
 +      n = new MockNamespace();
 +    }
 +    t.setNamespaceName(namespace);
 +    t.setNamespace(n);
 +    acu.namespaces.put(namespace, n);
 +    acu.tables.put(newTableName, t);
 +  }
 +
 +  @Deprecated
 +  @Override
 +  public void flush(String tableName) throws AccumuloException, AccumuloSecurityException {}
 +
 +  @Override
 +  public void setProperty(String tableName, String property, String value) throws AccumuloException, AccumuloSecurityException {
 +    acu.tables.get(tableName).settings.put(property, value);
 +  }
 +
 +  @Override
 +  public void removeProperty(String tableName, String property) throws AccumuloException, AccumuloSecurityException {
 +    acu.tables.get(tableName).settings.remove(property);
 +  }
 +
 +  @Override
 +  public Iterable<Entry<String,String>> getProperties(String tableName) throws TableNotFoundException {
 +    String namespace = Tables.qualify(tableName).getFirst();
 +    if (!exists(tableName)) {
 +      if (!namespaceExists(namespace))
 +        throw new TableNotFoundException(tableName, new NamespaceNotFoundException(null, namespace, null));
 +      throw new TableNotFoundException(null, tableName, null);
 +    }
 +
 +    Set<Entry<String,String>> props = new HashSet<Entry<String,String>>(acu.namespaces.get(namespace).settings.entrySet());
 +
 +    Set<Entry<String,String>> tableProps = acu.tables.get(tableName).settings.entrySet();
 +    for (Entry<String,String> e : tableProps) {
 +      if (props.contains(e)) {
 +        props.remove(e);
 +      }
 +      props.add(e);
 +    }
 +    return props;
 +  }
 +
 +  @Override
 +  public void setLocalityGroups(String tableName, Map<String,Set<Text>> groups) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.tables.get(tableName).setLocalityGroups(groups);
 +  }
 +
 +  @Override
 +  public Map<String,Set<Text>> getLocalityGroups(String tableName) throws AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return acu.tables.get(tableName).getLocalityGroups();
 +  }
 +
 +  @Override
 +  public Set<Range> splitRangeByTablets(String tableName, Range range, int maxSplits) throws AccumuloException, AccumuloSecurityException,
 +      TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    return Collections.singleton(range);
 +  }
 +
 +  @Override
 +  public void importDirectory(String tableName, String dir, String failureDir, boolean setTime) throws IOException, AccumuloException,
 +      AccumuloSecurityException, TableNotFoundException {
 +    long time = System.currentTimeMillis();
 +    MockTable table = acu.tables.get(tableName);
 +    if (table == null) {
 +      throw new TableNotFoundException(null, tableName, "The table was not found");
 +    }
 +    Path importPath = new Path(dir);
 +    Path failurePath = new Path(failureDir);
 +
 +    FileSystem fs = acu.getFileSystem();
 +    /*
 +     * check preconditions
 +     */
 +    // directories are directories
 +    if (fs.isFile(importPath)) {
 +      throw new IOException("Import path must be a directory.");
 +    }
 +    if (fs.isFile(failurePath)) {
 +      throw new IOException("Failure path must be a directory.");
 +    }
 +    // failures are writable
 +    Path createPath = failurePath.suffix("/.createFile");
 +    FSDataOutputStream createStream = null;
 +    try {
 +      createStream = fs.create(createPath);
 +    } catch (IOException e) {
 +      throw new IOException("Error path is not writable.");
 +    } finally {
 +      if (createStream != null) {
 +        createStream.close();
 +      }
 +    }
 +    fs.delete(createPath, false);
 +    // failures are empty
 +    FileStatus[] failureChildStats = fs.listStatus(failurePath);
 +    if (failureChildStats.length > 0) {
 +      throw new IOException("Error path must be empty.");
 +    }
 +    /*
 +     * Begin the import - iterate the files in the path
 +     */
 +    for (FileStatus importStatus : fs.listStatus(importPath)) {
 +      try {
 +        FileSKVIterator importIterator = FileOperations.getInstance().openReader(importStatus.getPath().toString(), true, fs, fs.getConf(),
 +            AccumuloConfiguration.getDefaultConfiguration());
 +        while (importIterator.hasTop()) {
 +          Key key = importIterator.getTopKey();
 +          Value value = importIterator.getTopValue();
 +          if (setTime) {
 +            key.setTimestamp(time);
 +          }
 +          Mutation mutation = new Mutation(key.getRow());
 +          if (!key.isDeleted()) {
 +            mutation.put(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()), key.getTimestamp(),
 +                value);
 +          } else {
 +            mutation.putDelete(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()),
 +                key.getTimestamp());
 +          }
 +          table.addMutation(mutation);
 +          importIterator.next();
 +        }
 +      } catch (Exception e) {
 +        FSDataOutputStream failureWriter = null;
 +        DataInputStream failureReader = null;
 +        try {
 +          failureWriter = fs.create(failurePath.suffix("/" + importStatus.getPath().getName()));
 +          failureReader = fs.open(importStatus.getPath());
 +          int read = 0;
 +          byte[] buffer = new byte[1024];
 +          while (-1 != (read = failureReader.read(buffer))) {
 +            failureWriter.write(buffer, 0, read);
 +          }
 +        } finally {
 +          if (failureReader != null)
 +            failureReader.close();
 +          if (failureWriter != null)
 +            failureWriter.close();
 +        }
 +      }
 +      fs.delete(importStatus.getPath(), true);
 +    }
 +  }
 +
 +  @Override
 +  public void offline(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    offline(tableName, false);
 +  }
 +
 +  @Override
 +  public void offline(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void online(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    online(tableName, false);
 +  }
 +
 +  @Override
 +  public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void clearLocatorCache(String tableName) throws TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public Map<String,String> tableIdMap() {
 +    Map<String,String> result = new HashMap<String,String>();
-     for (String table : acu.tables.keySet()) {
++    for (Entry<String,MockTable> entry : acu.tables.entrySet()) {
++      String table = entry.getKey();
 +      if (RootTable.NAME.equals(table))
 +        result.put(table, RootTable.ID);
 +      else if (MetadataTable.NAME.equals(table))
 +        result.put(table, MetadataTable.ID);
 +      else
-         result.put(table, table);
++        result.put(table, entry.getValue().getTableId());
 +    }
 +    return result;
 +  }
 +
 +  @Override
 +  public List<DiskUsage> getDiskUsage(Set<String> tables) throws AccumuloException, AccumuloSecurityException {
 +
 +    List<DiskUsage> diskUsages = new ArrayList<DiskUsage>();
 +    diskUsages.add(new DiskUsage(new TreeSet<String>(tables), 0l));
 +
 +    return diskUsages;
 +  }
 +
 +  @Override
 +  public void merge(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    acu.merge(tableName, start, end);
 +  }
 +
 +  @Override
 +  public void deleteRows(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +    MockTable t = acu.tables.get(tableName);
 +    Text startText = start != null ? new Text(start) : new Text();
 +    Text endText = end != null ? new Text(end) : new Text(t.table.lastKey().getRow().getBytes());
 +    startText.append(ZERO, 0, 1);
 +    endText.append(ZERO, 0, 1);
 +    Set<Key> keep = new TreeSet<Key>(t.table.subMap(new Key(startText), new Key(endText)).keySet());
 +    t.table.keySet().removeAll(keep);
 +  }
 +
 +  @Override
 +  public void compact(String tableName, Text start, Text end, boolean flush, boolean wait) throws AccumuloSecurityException, TableNotFoundException,
 +      AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void compact(String tableName, Text start, Text end, List<IteratorSetting> iterators, boolean flush, boolean wait) throws AccumuloSecurityException,
 +      TableNotFoundException, AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void cancelCompaction(String tableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public void clone(String srcTableName, String newTableName, boolean flush, Map<String,String> propertiesToSet, Set<String> propertiesToExclude)
 +      throws AccumuloException, AccumuloSecurityException, TableNotFoundException, TableExistsException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public void flush(String tableName, Text start, Text end, boolean wait) throws AccumuloException, AccumuloSecurityException, TableNotFoundException {
 +    if (!exists(tableName))
 +      throw new TableNotFoundException(tableName, tableName, "");
 +  }
 +
 +  @Override
 +  public Text getMaxRow(String tableName, Authorizations auths, Text startRow, boolean startInclusive, Text endRow, boolean endInclusive)
 +      throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    MockTable table = acu.tables.get(tableName);
 +    if (table == null)
 +      throw new TableNotFoundException(tableName, tableName, "no such table");
 +
 +    return FindMax.findMax(new MockScanner(table, auths), startRow, startInclusive, endRow, endInclusive);
 +  }
 +
 +  @Override
 +  public void importTable(String tableName, String exportDir) throws TableExistsException, AccumuloException, AccumuloSecurityException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public void exportTable(String tableName, String exportDir) throws TableNotFoundException, AccumuloException, AccumuloSecurityException {
 +    throw new NotImplementedException();
 +  }
 +
 +  @Override
 +  public boolean testClassLoad(String tableName, String className, String asTypeName) throws AccumuloException, AccumuloSecurityException,
 +      TableNotFoundException {
 +
 +    try {
 +      AccumuloVFSClassLoader.loadClass(className, Class.forName(asTypeName));
 +    } catch (ClassNotFoundException e) {
 +      e.printStackTrace();
 +      return false;
 +    }
 +    return true;
    }
  }