You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@accumulo.apache.org by vi...@apache.org on 2013/02/04 22:11:31 UTC

svn commit: r1442362 [1/3] - in /accumulo/trunk/proxy/src: main/java/org/apache/accumulo/proxy/ main/java/org/apache/accumulo/proxy/thrift/ main/thrift/ test/java/org/apache/accumulo/proxy/

Author: vines
Date: Mon Feb  4 21:11:30 2013
New Revision: 1442362

URL: http://svn.apache.org/viewvc?rev=1442362&view=rev
Log:
ACCUMULO-996 - bringing back the login aspect for less expansive code changes in the future


Added:
    accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/thrift/UserPass.java
      - copied unchanged from r1442283, accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/thrift/UserPass.java
Modified:
    accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/ProxyServer.java
    accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/TestProxyClient.java
    accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/thrift/AccumuloProxy.java
    accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/thrift/ActiveScan.java
    accumulo/trunk/proxy/src/main/thrift/proxy.thrift
    accumulo/trunk/proxy/src/test/java/org/apache/accumulo/proxy/SimpleTest.java
    accumulo/trunk/proxy/src/test/java/org/apache/accumulo/proxy/TestProxyInstanceOperations.java
    accumulo/trunk/proxy/src/test/java/org/apache/accumulo/proxy/TestProxyReadWrite.java
    accumulo/trunk/proxy/src/test/java/org/apache/accumulo/proxy/TestProxySecurityOperations.java
    accumulo/trunk/proxy/src/test/java/org/apache/accumulo/proxy/TestProxyTableOperations.java

Modified: accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/ProxyServer.java
URL: http://svn.apache.org/viewvc/accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/ProxyServer.java?rev=1442362&r1=1442361&r2=1442362&view=diff
==============================================================================
--- accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/ProxyServer.java (original)
+++ accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/ProxyServer.java Mon Feb  4 21:11:30 2013
@@ -27,6 +27,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Map.Entry;
 import java.util.Properties;
+import java.util.Random;
 import java.util.Set;
 import java.util.SortedSet;
 import java.util.TreeSet;
@@ -63,6 +64,7 @@ import org.apache.accumulo.core.security
 import org.apache.accumulo.core.security.ColumnVisibility;
 import org.apache.accumulo.core.security.SystemPermission;
 import org.apache.accumulo.core.security.TablePermission;
+import org.apache.accumulo.core.security.thrift.Credentials;
 import org.apache.accumulo.core.util.ByteBufferUtil;
 import org.apache.accumulo.core.util.TextUtil;
 import org.apache.accumulo.proxy.thrift.AccumuloProxy;
@@ -73,7 +75,6 @@ import org.apache.accumulo.proxy.thrift.
 import org.apache.accumulo.proxy.thrift.KeyValue;
 import org.apache.accumulo.proxy.thrift.KeyValueAndPeek;
 import org.apache.accumulo.proxy.thrift.NoMoreEntriesException;
-import org.apache.accumulo.proxy.thrift.PrincipalToken;
 import org.apache.accumulo.proxy.thrift.ScanColumn;
 import org.apache.accumulo.proxy.thrift.ScanOptions;
 import org.apache.accumulo.proxy.thrift.ScanResult;
@@ -81,6 +82,7 @@ import org.apache.accumulo.proxy.thrift.
 import org.apache.accumulo.proxy.thrift.ScanType;
 import org.apache.accumulo.proxy.thrift.UnknownScanner;
 import org.apache.accumulo.proxy.thrift.UnknownWriter;
+import org.apache.accumulo.proxy.thrift.UserPass;
 import org.apache.accumulo.proxy.thrift.WriterOptions;
 import org.apache.hadoop.io.Text;
 import org.apache.log4j.Logger;
@@ -127,6 +129,8 @@ public class ProxyServer implements Accu
 
   protected Cache<UUID,ScannerPlusIterator> scannerCache;
   protected Cache<UUID,BatchWriter> writerCache;
+  protected Cache<ByteBuffer, Credentials> tokenCache;
+  private Random random = new Random();
   
   public ProxyServer(Properties props) {
     String useMock = props.getProperty("org.apache.accumulo.proxy.ProxyServer.useMockInstance");
@@ -139,10 +143,14 @@ public class ProxyServer implements Accu
     scannerCache = CacheBuilder.newBuilder().expireAfterAccess(10, TimeUnit.MINUTES).maximumSize(1000).removalListener(new CloseScanner()).build();
     
     writerCache = CacheBuilder.newBuilder().expireAfterAccess(10, TimeUnit.MINUTES).maximumSize(1000).removalListener(new CloseWriter()).build();
+    tokenCache = CacheBuilder.newBuilder().expireAfterAccess(10, TimeUnit.MINUTES).maximumSize(1000).build();
   }
   
-  protected Connector getConnector(PrincipalToken PrincipalToken) throws Exception {
-    Connector connector = instance.getConnector(PrincipalToken.getPrincipal(), PrincipalToken.bufferForToken());
+  protected Connector getConnector(ByteBuffer login) throws Exception {
+    Credentials user = tokenCache.getIfPresent(login);
+    if (user == null)
+      throw new org.apache.accumulo.proxy.thrift.AccumuloSecurityException("unknown user");
+    Connector connector = instance.getConnector(user);
     return connector;
   }
   
@@ -169,46 +177,41 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public boolean ping(PrincipalToken PrincipalToken) throws TException {
-    return true;
-  }
-  
-  @Override
-  public int addConstraint(PrincipalToken PrincipalToken, String tableName, String constraintClassName) throws TException {
+  public int addConstraint(ByteBuffer login, String tableName, String constraintClassName) throws TException {
     try {
-      return getConnector(PrincipalToken).tableOperations().addConstraint(tableName, constraintClassName);
+      return getConnector(login).tableOperations().addConstraint(tableName, constraintClassName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void addSplits(PrincipalToken PrincipalToken, String tableName, Set<ByteBuffer> splits) throws TException {
+  public void addSplits(ByteBuffer login, String tableName, Set<ByteBuffer> splits) throws TException {
     try {
       SortedSet<Text> sorted = new TreeSet<Text>();
       for (ByteBuffer split : splits) {
         sorted.add(ByteBufferUtil.toText(split));
       }
-      getConnector(PrincipalToken).tableOperations().addSplits(tableName, sorted);
+      getConnector(login).tableOperations().addSplits(tableName, sorted);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void clearLocatorCache(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public void clearLocatorCache(ByteBuffer login, String tableName) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().clearLocatorCache(tableName);
+      getConnector(login).tableOperations().clearLocatorCache(tableName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void compactTable(PrincipalToken PrincipalToken, String tableName, ByteBuffer start, ByteBuffer end, List<org.apache.accumulo.proxy.thrift.IteratorSetting> iterators, boolean flush,
+  public void compactTable(ByteBuffer login, String tableName, ByteBuffer start, ByteBuffer end, List<org.apache.accumulo.proxy.thrift.IteratorSetting> iterators, boolean flush,
       boolean wait) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().compact(tableName, ByteBufferUtil.toText(start), ByteBufferUtil.toText(end), getIteratorSettings(iterators),
+      getConnector(login).tableOperations().compact(tableName, ByteBufferUtil.toText(start), ByteBufferUtil.toText(end), getIteratorSettings(iterators),
           flush, wait);
     } catch (Exception e) {
       throw translateException(e);
@@ -216,10 +219,10 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void cancelCompaction(PrincipalToken PrincipalToken, String tableName) throws org.apache.accumulo.proxy.thrift.AccumuloSecurityException,
+  public void cancelCompaction(ByteBuffer login, String tableName) throws org.apache.accumulo.proxy.thrift.AccumuloSecurityException,
       org.apache.accumulo.proxy.thrift.TableNotFoundException, org.apache.accumulo.proxy.thrift.AccumuloException, TException {
     try {
-      getConnector(PrincipalToken).tableOperations().cancelCompaction(tableName);
+      getConnector(login).tableOperations().cancelCompaction(tableName);
     } catch (Exception e) {
       throw translateException(e);
     }
@@ -236,57 +239,57 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void createTable(PrincipalToken PrincipalToken, String tableName, boolean versioningIter, org.apache.accumulo.proxy.thrift.TimeType timeType) throws TException {
+  public void createTable(ByteBuffer login, String tableName, boolean versioningIter, org.apache.accumulo.proxy.thrift.TimeType timeType) throws TException {
     try {
       if (timeType == null)
         timeType = org.apache.accumulo.proxy.thrift.TimeType.MILLIS;
       
-      getConnector(PrincipalToken).tableOperations().create(tableName, versioningIter, TimeType.valueOf(timeType.toString()));
+      getConnector(login).tableOperations().create(tableName, versioningIter, TimeType.valueOf(timeType.toString()));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void deleteTable(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public void deleteTable(ByteBuffer login, String tableName) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().delete(tableName);
+      getConnector(login).tableOperations().delete(tableName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void deleteRows(PrincipalToken PrincipalToken, String tableName, ByteBuffer start, ByteBuffer end) throws TException {
+  public void deleteRows(ByteBuffer login, String tableName, ByteBuffer start, ByteBuffer end) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().deleteRows(tableName, ByteBufferUtil.toText(start), ByteBufferUtil.toText(end));
+      getConnector(login).tableOperations().deleteRows(tableName, ByteBufferUtil.toText(start), ByteBufferUtil.toText(end));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public boolean tableExists(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public boolean tableExists(ByteBuffer login, String tableName) throws TException {
     try {
-      return getConnector(PrincipalToken).tableOperations().exists(tableName);
+      return getConnector(login).tableOperations().exists(tableName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void flushTable(PrincipalToken PrincipalToken, String tableName, ByteBuffer startRow, ByteBuffer endRow, boolean wait) throws TException {
+  public void flushTable(ByteBuffer login, String tableName, ByteBuffer startRow, ByteBuffer endRow, boolean wait) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().flush(tableName, ByteBufferUtil.toText(startRow), ByteBufferUtil.toText(endRow), wait);
+      getConnector(login).tableOperations().flush(tableName, ByteBufferUtil.toText(startRow), ByteBufferUtil.toText(endRow), wait);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public Map<String,Set<String>> getLocalityGroups(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public Map<String,Set<String>> getLocalityGroups(ByteBuffer login, String tableName) throws TException {
     try {
-      Map<String,Set<Text>> groups = getConnector(PrincipalToken).tableOperations().getLocalityGroups(tableName);
+      Map<String,Set<Text>> groups = getConnector(login).tableOperations().getLocalityGroups(tableName);
       Map<String,Set<String>> ret = new HashMap<String,Set<String>>();
       for (String key : groups.keySet()) {
         ret.put(key, new HashSet<String>());
@@ -301,17 +304,18 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public ByteBuffer getMaxRow(PrincipalToken PrincipalToken, String tableName, Set<ByteBuffer> auths, ByteBuffer startRow, boolean startinclusive,
+  public ByteBuffer getMaxRow(ByteBuffer login, String tableName, Set<ByteBuffer> auths, ByteBuffer startRow, boolean startinclusive,
       ByteBuffer endRow, boolean endinclusive) throws TException {
     try {
-      Connector connector = getConnector(PrincipalToken);
+      Connector connector = getConnector(login);
       Text startText = ByteBufferUtil.toText(startRow);
       Text endText = ByteBufferUtil.toText(endRow);
       Authorizations auth;
       if (auths != null) {
         auth = getAuthorizations(auths);
       } else {
-        auth = connector.securityOperations().getUserAuthorizations(PrincipalToken.getPrincipal());
+        Credentials token = tokenCache.getIfPresent(login);
+        auth = connector.securityOperations().getUserAuthorizations(token.getPrincipal());
       }
       Text max = connector.tableOperations().getMaxRow(tableName, auth, startText, startinclusive, endText, endinclusive);
       return TextUtil.getByteBuffer(max);
@@ -321,11 +325,11 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public Map<String,String> getTableProperties(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public Map<String,String> getTableProperties(ByteBuffer login, String tableName) throws TException {
     try {
       Map<String,String> ret = new HashMap<String,String>();
       
-      for (Map.Entry<String,String> entry : getConnector(PrincipalToken).tableOperations().getProperties(tableName)) {
+      for (Map.Entry<String,String> entry : getConnector(login).tableOperations().getProperties(tableName)) {
         ret.put(entry.getKey(), entry.getValue());
       }
       return ret;
@@ -335,9 +339,9 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public List<ByteBuffer> getSplits(PrincipalToken PrincipalToken, String tableName, int maxSplits) throws TException {
+  public List<ByteBuffer> getSplits(ByteBuffer login, String tableName, int maxSplits) throws TException {
     try {
-      Collection<Text> splits = getConnector(PrincipalToken).tableOperations().getSplits(tableName, maxSplits);
+      Collection<Text> splits = getConnector(login).tableOperations().getSplits(tableName, maxSplits);
       List<ByteBuffer> ret = new ArrayList<ByteBuffer>();
       for (Text split : splits) {
         ret.add(TextUtil.getByteBuffer(split));
@@ -349,79 +353,79 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public Set<String> listTables(PrincipalToken PrincipalToken) throws TException {
+  public Set<String> listTables(ByteBuffer login) throws TException {
     try {
-      return getConnector(PrincipalToken).tableOperations().list();
+      return getConnector(login).tableOperations().list();
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public Map<String,Integer> listConstraints(PrincipalToken PrincipalToken, String arg2) throws TException {
+  public Map<String,Integer> listConstraints(ByteBuffer login, String arg2) throws TException {
     try {
-      return getConnector(PrincipalToken).tableOperations().listConstraints(arg2);
+      return getConnector(login).tableOperations().listConstraints(arg2);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void mergeTablets(PrincipalToken PrincipalToken, String tableName, ByteBuffer start, ByteBuffer end) throws TException {
+  public void mergeTablets(ByteBuffer login, String tableName, ByteBuffer start, ByteBuffer end) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().merge(tableName, ByteBufferUtil.toText(start), ByteBufferUtil.toText(end));
+      getConnector(login).tableOperations().merge(tableName, ByteBufferUtil.toText(start), ByteBufferUtil.toText(end));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void offlineTable(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public void offlineTable(ByteBuffer login, String tableName) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().offline(tableName);
+      getConnector(login).tableOperations().offline(tableName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void onlineTable(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public void onlineTable(ByteBuffer login, String tableName) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().online(tableName);
+      getConnector(login).tableOperations().online(tableName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void removeConstraint(PrincipalToken PrincipalToken, String tableName, int number) throws TException {
+  public void removeConstraint(ByteBuffer login, String tableName, int number) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().removeConstraint(tableName, number);
+      getConnector(login).tableOperations().removeConstraint(tableName, number);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void removeTableProperty(PrincipalToken PrincipalToken, String tableName, String property) throws TException {
+  public void removeTableProperty(ByteBuffer login, String tableName, String property) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().removeProperty(tableName, property);
+      getConnector(login).tableOperations().removeProperty(tableName, property);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void renameTable(PrincipalToken PrincipalToken, String oldTableName, String newTableName) throws TException {
+  public void renameTable(ByteBuffer login, String oldTableName, String newTableName) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().rename(oldTableName, newTableName);
+      getConnector(login).tableOperations().rename(oldTableName, newTableName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void setLocalityGroups(PrincipalToken PrincipalToken, String tableName, Map<String,Set<String>> groupStrings) throws TException {
+  public void setLocalityGroups(ByteBuffer login, String tableName, Map<String,Set<String>> groupStrings) throws TException {
     try {
       Map<String,Set<Text>> groups = new HashMap<String,Set<Text>>();
       for (String key : groupStrings.keySet()) {
@@ -430,66 +434,66 @@ public class ProxyServer implements Accu
           groups.get(key).add(new Text(val));
         }
       }
-      getConnector(PrincipalToken).tableOperations().setLocalityGroups(tableName, groups);
+      getConnector(login).tableOperations().setLocalityGroups(tableName, groups);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void setTableProperty(PrincipalToken PrincipalToken, String tableName, String property, String value) throws TException {
+  public void setTableProperty(ByteBuffer login, String tableName, String property, String value) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().setProperty(tableName, property, value);
+      getConnector(login).tableOperations().setProperty(tableName, property, value);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public Map<String,String> tableIdMap(PrincipalToken PrincipalToken) throws TException {
+  public Map<String,String> tableIdMap(ByteBuffer login) throws TException {
     try {
-      return getConnector(PrincipalToken).tableOperations().tableIdMap();
+      return getConnector(login).tableOperations().tableIdMap();
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public Map<String,String> getSiteConfiguration(PrincipalToken PrincipalToken) throws TException {
+  public Map<String,String> getSiteConfiguration(ByteBuffer login) throws TException {
     try {
-      return getConnector(PrincipalToken).instanceOperations().getSiteConfiguration();
+      return getConnector(login).instanceOperations().getSiteConfiguration();
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public Map<String,String> getSystemConfiguration(PrincipalToken PrincipalToken) throws TException {
+  public Map<String,String> getSystemConfiguration(ByteBuffer login) throws TException {
     try {
-      return getConnector(PrincipalToken).instanceOperations().getSystemConfiguration();
+      return getConnector(login).instanceOperations().getSystemConfiguration();
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public List<String> getTabletServers(PrincipalToken PrincipalToken) throws TException {
+  public List<String> getTabletServers(ByteBuffer login) throws TException {
     try {
-      return getConnector(PrincipalToken).instanceOperations().getTabletServers();
+      return getConnector(login).instanceOperations().getTabletServers();
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public List<org.apache.accumulo.proxy.thrift.ActiveScan> getActiveScans(PrincipalToken PrincipalToken, String tserver) throws TException {
+  public List<org.apache.accumulo.proxy.thrift.ActiveScan> getActiveScans(ByteBuffer login, String tserver) throws TException {
     List<org.apache.accumulo.proxy.thrift.ActiveScan> result = new ArrayList<org.apache.accumulo.proxy.thrift.ActiveScan>();
     try {
-      List<ActiveScan> activeScans = getConnector(PrincipalToken).instanceOperations().getActiveScans(tserver);
+      List<ActiveScan> activeScans = getConnector(login).instanceOperations().getActiveScans(tserver);
       for (ActiveScan scan : activeScans) {
         org.apache.accumulo.proxy.thrift.ActiveScan pscan = new org.apache.accumulo.proxy.thrift.ActiveScan();
         pscan.client = scan.getClient();
-        pscan.principal = scan.getUser();
+        pscan.user = scan.getUser();
         pscan.table = scan.getTable();
         pscan.age = scan.getAge();
         pscan.idleTime = scan.getIdleTime();
@@ -533,10 +537,10 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public List<org.apache.accumulo.proxy.thrift.ActiveCompaction> getActiveCompactions(PrincipalToken PrincipalToken, String tserver) throws TException {
+  public List<org.apache.accumulo.proxy.thrift.ActiveCompaction> getActiveCompactions(ByteBuffer login, String tserver) throws TException {
     try {
       List<org.apache.accumulo.proxy.thrift.ActiveCompaction> result = new ArrayList<org.apache.accumulo.proxy.thrift.ActiveCompaction>();
-      List<ActiveCompaction> active = getConnector(PrincipalToken).instanceOperations().getActiveCompactions(tserver);
+      List<ActiveCompaction> active = getConnector(login).instanceOperations().getActiveCompactions(tserver);
       for (ActiveCompaction comp : active) {
         org.apache.accumulo.proxy.thrift.ActiveCompaction pcomp = new org.apache.accumulo.proxy.thrift.ActiveCompaction();
         pcomp.age = comp.getAge();
@@ -569,148 +573,148 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void removeProperty(PrincipalToken PrincipalToken, String property) throws TException {
+  public void removeProperty(ByteBuffer login, String property) throws TException {
     try {
-      getConnector(PrincipalToken).instanceOperations().removeProperty(property);
+      getConnector(login).instanceOperations().removeProperty(property);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void setProperty(PrincipalToken PrincipalToken, String property, String value) throws TException {
+  public void setProperty(ByteBuffer login, String property, String value) throws TException {
     try {
-      getConnector(PrincipalToken).instanceOperations().setProperty(property, value);
+      getConnector(login).instanceOperations().setProperty(property, value);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public boolean testClassLoad(PrincipalToken PrincipalToken, String className, String asTypeName) throws TException {
+  public boolean testClassLoad(ByteBuffer login, String className, String asTypeName) throws TException {
     try {
-      return getConnector(PrincipalToken).instanceOperations().testClassLoad(className, asTypeName);
+      return getConnector(login).instanceOperations().testClassLoad(className, asTypeName);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public boolean authenticateUser(PrincipalToken PrincipalToken, String user, ByteBuffer password) throws TException {
+  public boolean authenticateUser(ByteBuffer login, String user, ByteBuffer password) throws TException {
     try {
-      return getConnector(PrincipalToken).securityOperations().authenticateUser(user, password.array());
+      return getConnector(login).securityOperations().authenticateUser(user, ByteBufferUtil.toBytes(password));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void changeUserAuthorizations(PrincipalToken PrincipalToken, String user, Set<ByteBuffer> authorizations) throws TException {
+  public void changeUserAuthorizations(ByteBuffer login, String user, Set<ByteBuffer> authorizations) throws TException {
     try {
       Set<String> auths = new HashSet<String>();
       for (ByteBuffer auth : authorizations) {
         auths.add(ByteBufferUtil.toString(auth));
       }
-      getConnector(PrincipalToken).securityOperations().changeUserAuthorizations(user, new Authorizations(auths.toArray(new String[0])));
+      getConnector(login).securityOperations().changeUserAuthorizations(user, new Authorizations(auths.toArray(new String[0])));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void changePrincipalTokenword(PrincipalToken PrincipalToken, String user, ByteBuffer password) throws TException {
+  public void changeUserPassword(ByteBuffer login, String user, ByteBuffer password) throws TException {
     try {
-      getConnector(PrincipalToken).securityOperations().changeUserPassword(user, password.array());
+      getConnector(login).securityOperations().changeUserPassword(user, ByteBufferUtil.toBytes(password));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void createUser(PrincipalToken PrincipalToken, String user, ByteBuffer password) throws TException {
+  public void createUser(ByteBuffer login, String user, ByteBuffer password) throws TException {
     try {
-      getConnector(PrincipalToken).securityOperations().createUser(user, password.array());
+      getConnector(login).securityOperations().createUser(user, ByteBufferUtil.toBytes(password));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void dropUser(PrincipalToken PrincipalToken, String user) throws TException {
+  public void dropUser(ByteBuffer login, String user) throws TException {
     try {
-      getConnector(PrincipalToken).securityOperations().dropUser(user);
+      getConnector(login).securityOperations().dropUser(user);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public List<ByteBuffer> getUserAuthorizations(PrincipalToken PrincipalToken, String user) throws TException {
+  public List<ByteBuffer> getUserAuthorizations(ByteBuffer login, String user) throws TException {
     try {
-      return getConnector(PrincipalToken).securityOperations().getUserAuthorizations(user).getAuthorizationsBB();
+      return getConnector(login).securityOperations().getUserAuthorizations(user).getAuthorizationsBB();
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void grantSystemPermission(PrincipalToken PrincipalToken, String user, org.apache.accumulo.proxy.thrift.SystemPermission perm) throws TException {
+  public void grantSystemPermission(ByteBuffer login, String user, org.apache.accumulo.proxy.thrift.SystemPermission perm) throws TException {
     try {
-      getConnector(PrincipalToken).securityOperations().grantSystemPermission(user, SystemPermission.getPermissionById((byte) perm.getValue()));
+      getConnector(login).securityOperations().grantSystemPermission(user, SystemPermission.getPermissionById((byte) perm.getValue()));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void grantTablePermission(PrincipalToken PrincipalToken, String user, String table, org.apache.accumulo.proxy.thrift.TablePermission perm) throws TException {
+  public void grantTablePermission(ByteBuffer login, String user, String table, org.apache.accumulo.proxy.thrift.TablePermission perm) throws TException {
     try {
-      getConnector(PrincipalToken).securityOperations().grantTablePermission(user, table, TablePermission.getPermissionById((byte) perm.getValue()));
+      getConnector(login).securityOperations().grantTablePermission(user, table, TablePermission.getPermissionById((byte) perm.getValue()));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public boolean hasSystemPermission(PrincipalToken PrincipalToken, String user, org.apache.accumulo.proxy.thrift.SystemPermission perm) throws TException {
+  public boolean hasSystemPermission(ByteBuffer login, String user, org.apache.accumulo.proxy.thrift.SystemPermission perm) throws TException {
     try {
-      return getConnector(PrincipalToken).securityOperations().hasSystemPermission(user, SystemPermission.getPermissionById((byte) perm.getValue()));
+      return getConnector(login).securityOperations().hasSystemPermission(user, SystemPermission.getPermissionById((byte) perm.getValue()));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public boolean hasTablePermission(PrincipalToken PrincipalToken, String user, String table, org.apache.accumulo.proxy.thrift.TablePermission perm) throws TException {
+  public boolean hasTablePermission(ByteBuffer login, String user, String table, org.apache.accumulo.proxy.thrift.TablePermission perm) throws TException {
     try {
-      return getConnector(PrincipalToken).securityOperations().hasTablePermission(user, table, TablePermission.getPermissionById((byte) perm.getValue()));
+      return getConnector(login).securityOperations().hasTablePermission(user, table, TablePermission.getPermissionById((byte) perm.getValue()));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public Set<String> listUsers(PrincipalToken PrincipalToken) throws TException {
+  public Set<String> listUsers(ByteBuffer login) throws TException {
     try {
-      return getConnector(PrincipalToken).securityOperations().listUsers();
+      return getConnector(login).securityOperations().listUsers();
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void revokeSystemPermission(PrincipalToken PrincipalToken, String user, org.apache.accumulo.proxy.thrift.SystemPermission perm) throws TException {
+  public void revokeSystemPermission(ByteBuffer login, String user, org.apache.accumulo.proxy.thrift.SystemPermission perm) throws TException {
     try {
-      getConnector(PrincipalToken).securityOperations().revokeSystemPermission(user, SystemPermission.getPermissionById((byte) perm.getValue()));
+      getConnector(login).securityOperations().revokeSystemPermission(user, SystemPermission.getPermissionById((byte) perm.getValue()));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void revokeTablePermission(PrincipalToken PrincipalToken, String user, String table, org.apache.accumulo.proxy.thrift.TablePermission perm) throws TException {
+  public void revokeTablePermission(ByteBuffer login, String user, String table, org.apache.accumulo.proxy.thrift.TablePermission perm) throws TException {
     try {
-      getConnector(PrincipalToken).securityOperations().revokeTablePermission(user, table, TablePermission.getPermissionById((byte) perm.getValue()));
+      getConnector(login).securityOperations().revokeTablePermission(user, table, TablePermission.getPermissionById((byte) perm.getValue()));
     } catch (Exception e) {
       throw translateException(e);
     }
@@ -725,16 +729,17 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public String createScanner(PrincipalToken PrincipalToken, String tableName, ScanOptions opts)
+  public String createScanner(ByteBuffer login, String tableName, ScanOptions opts)
       throws TException {
     try {
-      Connector connector = getConnector(PrincipalToken);
+      Connector connector = getConnector(login);
       
       Authorizations auth;
       if (opts != null && opts.isSetAuthorizations()) {
         auth = getAuthorizations(opts.authorizations);
       } else {
-        auth = connector.securityOperations().getUserAuthorizations(PrincipalToken.getPrincipal());
+        Credentials token = tokenCache.getIfPresent(login);
+        auth = connector.securityOperations().getUserAuthorizations(token.getPrincipal());
       }
       Scanner scanner = connector.createScanner(tableName, auth);
       
@@ -776,10 +781,10 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public String createBatchScanner(PrincipalToken PrincipalToken, String tableName, BatchScanOptions opts)
+  public String createBatchScanner(ByteBuffer login, String tableName, BatchScanOptions opts)
       throws TException {
     try {
-      Connector connector = getConnector(PrincipalToken);
+      Connector connector = getConnector(login);
       
             
       int threads = 10;
@@ -787,7 +792,8 @@ public class ProxyServer implements Accu
       if (opts != null && opts.isSetAuthorizations()) {
         auth = getAuthorizations(opts.authorizations);
       } else {
-        auth = connector.securityOperations().getUserAuthorizations(PrincipalToken.getPrincipal());
+        Credentials token = tokenCache.getIfPresent(login);
+        auth = connector.securityOperations().getUserAuthorizations(token.getPrincipal());
       }
       if (opts != null && opts.threads > 0)
         threads = opts.threads;
@@ -884,9 +890,9 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void updateAndFlush(PrincipalToken PrincipalToken, String tableName, Map<ByteBuffer,List<ColumnUpdate>> cells) throws TException {
+  public void updateAndFlush(ByteBuffer login, String tableName, Map<ByteBuffer,List<ColumnUpdate>> cells) throws TException {
     try {
-      BatchWriter writer = getWriter(PrincipalToken, tableName, null);
+      BatchWriter writer = getWriter(login, tableName, null);
       addCellsToWriter(cells, writer);
       writer.flush();
       writer.close();
@@ -934,9 +940,9 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public String createWriter(PrincipalToken PrincipalToken, String tableName, WriterOptions opts) throws TException {
+  public String createWriter(ByteBuffer login, String tableName, WriterOptions opts) throws TException {
     try {
-      BatchWriter writer = getWriter(PrincipalToken, tableName, opts);
+      BatchWriter writer = getWriter(login, tableName, opts);
       UUID uuid = UUID.randomUUID();
       writerCache.put(uuid, writer);
       return uuid.toString();
@@ -985,7 +991,7 @@ public class ProxyServer implements Accu
     }
   }
   
-  private BatchWriter getWriter(PrincipalToken PrincipalToken, String tableName, WriterOptions opts) throws Exception {
+  private BatchWriter getWriter(ByteBuffer login, String tableName, WriterOptions opts) throws Exception {
     BatchWriterConfig cfg = new BatchWriterConfig();
     if (opts != null) {
       if (opts.maxMemory != 0)
@@ -997,7 +1003,7 @@ public class ProxyServer implements Accu
       if (opts.latencyMs != 0)
         cfg.setMaxLatency(opts.latencyMs, TimeUnit.MILLISECONDS);
     }
-    return getConnector(PrincipalToken).createBatchWriter(tableName, cfg);
+    return getConnector(login).createBatchWriter(tableName, cfg);
   }
   
   private IteratorSetting getIteratorSetting(org.apache.accumulo.proxy.thrift.IteratorSetting setting) {
@@ -1025,56 +1031,56 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void attachIterator(PrincipalToken PrincipalToken, String tableName, org.apache.accumulo.proxy.thrift.IteratorSetting setting, Set<org.apache.accumulo.proxy.thrift.IteratorScope> scopes) throws TException {
+  public void attachIterator(ByteBuffer login, String tableName, org.apache.accumulo.proxy.thrift.IteratorSetting setting, Set<org.apache.accumulo.proxy.thrift.IteratorScope> scopes) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().attachIterator(tableName, getIteratorSetting(setting), getIteratorScopes(scopes));
+      getConnector(login).tableOperations().attachIterator(tableName, getIteratorSetting(setting), getIteratorScopes(scopes));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void checkIteratorConflicts(PrincipalToken PrincipalToken, String tableName, org.apache.accumulo.proxy.thrift.IteratorSetting setting, Set<org.apache.accumulo.proxy.thrift.IteratorScope> scopes)
+  public void checkIteratorConflicts(ByteBuffer login, String tableName, org.apache.accumulo.proxy.thrift.IteratorSetting setting, Set<org.apache.accumulo.proxy.thrift.IteratorScope> scopes)
       throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().checkIteratorConflicts(tableName, getIteratorSetting(setting), getIteratorScopes(scopes));
+      getConnector(login).tableOperations().checkIteratorConflicts(tableName, getIteratorSetting(setting), getIteratorScopes(scopes));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void cloneTable(PrincipalToken PrincipalToken, String tableName, String newTableName, boolean flush, Map<String,String> propertiesToSet,
+  public void cloneTable(ByteBuffer login, String tableName, String newTableName, boolean flush, Map<String,String> propertiesToSet,
       Set<String> propertiesToExclude) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().clone(tableName, newTableName, flush, propertiesToSet, propertiesToExclude);
+      getConnector(login).tableOperations().clone(tableName, newTableName, flush, propertiesToSet, propertiesToExclude);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void exportTable(PrincipalToken PrincipalToken, String tableName, String exportDir) throws TException {
+  public void exportTable(ByteBuffer login, String tableName, String exportDir) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().exportTable(tableName, exportDir);
+      getConnector(login).tableOperations().exportTable(tableName, exportDir);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public void importTable(PrincipalToken PrincipalToken, String tableName, String importDir) throws TException {
+  public void importTable(ByteBuffer login, String tableName, String importDir) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().importTable(tableName, importDir);
+      getConnector(login).tableOperations().importTable(tableName, importDir);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public org.apache.accumulo.proxy.thrift.IteratorSetting getIteratorSetting(PrincipalToken PrincipalToken, String tableName, String iteratorName, org.apache.accumulo.proxy.thrift.IteratorScope scope) throws TException {
+  public org.apache.accumulo.proxy.thrift.IteratorSetting getIteratorSetting(ByteBuffer login, String tableName, String iteratorName, org.apache.accumulo.proxy.thrift.IteratorScope scope) throws TException {
     try {
-      IteratorSetting is = getConnector(PrincipalToken).tableOperations().getIteratorSetting(tableName, iteratorName, getIteratorScope(scope));
+      IteratorSetting is = getConnector(login).tableOperations().getIteratorSetting(tableName, iteratorName, getIteratorScope(scope));
       return new org.apache.accumulo.proxy.thrift.IteratorSetting(is.getPriority(), is.getName(), is.getIteratorClass(), is.getOptions());
     } catch (Exception e) {
       throw translateException(e);
@@ -1082,9 +1088,9 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public Map<String,Set<org.apache.accumulo.proxy.thrift.IteratorScope>> listIterators(PrincipalToken PrincipalToken, String tableName) throws TException {
+  public Map<String,Set<org.apache.accumulo.proxy.thrift.IteratorScope>> listIterators(ByteBuffer login, String tableName) throws TException {
     try {
-      Map<String,EnumSet<IteratorScope>> iterMap = getConnector(PrincipalToken).tableOperations().listIterators(tableName);
+      Map<String,EnumSet<IteratorScope>> iterMap = getConnector(login).tableOperations().listIterators(tableName);
       Map<String,Set<org.apache.accumulo.proxy.thrift.IteratorScope>> result = new HashMap<String,Set<org.apache.accumulo.proxy.thrift.IteratorScope>>();
       for (Entry<String,EnumSet<IteratorScope>> entry : iterMap.entrySet()) {
         result.put(entry.getKey(), getProxyIteratorScopes(entry.getValue()));
@@ -1096,18 +1102,18 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void removeIterator(PrincipalToken PrincipalToken, String tableName, String iterName, Set<org.apache.accumulo.proxy.thrift.IteratorScope> scopes) throws TException {
+  public void removeIterator(ByteBuffer login, String tableName, String iterName, Set<org.apache.accumulo.proxy.thrift.IteratorScope> scopes) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().removeIterator(tableName, iterName, getIteratorScopes(scopes));
+      getConnector(login).tableOperations().removeIterator(tableName, iterName, getIteratorScopes(scopes));
     } catch (Exception e) {
       throw translateException(e);
     }
   }
   
   @Override
-  public Set<org.apache.accumulo.proxy.thrift.Range> splitRangeByTablets(PrincipalToken PrincipalToken, String tableName, org.apache.accumulo.proxy.thrift.Range range, int maxSplits) throws TException {
+  public Set<org.apache.accumulo.proxy.thrift.Range> splitRangeByTablets(ByteBuffer login, String tableName, org.apache.accumulo.proxy.thrift.Range range, int maxSplits) throws TException {
     try {
-      Set<Range> ranges = getConnector(PrincipalToken).tableOperations().splitRangeByTablets(tableName, getRange(range), maxSplits);
+      Set<Range> ranges = getConnector(login).tableOperations().splitRangeByTablets(tableName, getRange(range), maxSplits);
       Set<org.apache.accumulo.proxy.thrift.Range> result = new HashSet<org.apache.accumulo.proxy.thrift.Range>();
       for (Range r : ranges) {
         result.add(getRange(r));
@@ -1135,9 +1141,9 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void importDirectory(PrincipalToken PrincipalToken, String tableName, String importDir, String failureDir, boolean setTime) throws TException {
+  public void importDirectory(ByteBuffer login, String tableName, String importDir, String failureDir, boolean setTime) throws TException {
     try {
-      getConnector(PrincipalToken).tableOperations().importDirectory(tableName, importDir, failureDir, setTime);
+      getConnector(login).tableOperations().importDirectory(tableName, importDir, failureDir, setTime);
     } catch (Exception e) {
       throw translateException(e);
     }
@@ -1160,11 +1166,18 @@ public class ProxyServer implements Accu
   }
   
   @Override
-  public void pingTabletServer(PrincipalToken PrincipalToken, String tserver) throws TException {
+  public void pingTabletServer(ByteBuffer login, String tserver) throws TException {
     try {
-      getConnector(PrincipalToken).instanceOperations().ping(tserver);
+      getConnector(login).instanceOperations().ping(tserver);
     } catch (Exception e) {
       throw translateException(e);
     }
   }
+
+  @Override
+  public ByteBuffer login(UserPass login) throws TException {
+    ByteBuffer result = ByteBuffer.wrap(Long.toHexString(random.nextLong()).getBytes());
+    tokenCache.put(result, new Credentials(login.getUsername(), login.bufferForPassword(), instance.getInstanceID()));
+    return result;
+  }
 }

Modified: accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/TestProxyClient.java
URL: http://svn.apache.org/viewvc/accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/TestProxyClient.java?rev=1442362&r1=1442361&r2=1442362&view=diff
==============================================================================
--- accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/TestProxyClient.java (original)
+++ accumulo/trunk/proxy/src/main/java/org/apache/accumulo/proxy/TestProxyClient.java Mon Feb  4 21:11:30 2013
@@ -28,9 +28,9 @@ import org.apache.accumulo.core.iterator
 import org.apache.accumulo.proxy.thrift.AccumuloProxy;
 import org.apache.accumulo.proxy.thrift.ColumnUpdate;
 import org.apache.accumulo.proxy.thrift.Key;
-import org.apache.accumulo.proxy.thrift.PrincipalToken;
 import org.apache.accumulo.proxy.thrift.ScanResult;
 import org.apache.accumulo.proxy.thrift.TimeType;
+import org.apache.accumulo.proxy.thrift.UserPass;
 import org.apache.thrift.protocol.TCompactProtocol;
 import org.apache.thrift.protocol.TProtocol;
 import org.apache.thrift.protocol.TProtocolFactory;
@@ -64,29 +64,30 @@ public class TestProxyClient {
   public static void main(String[] args) throws Exception {
     
     TestProxyClient tpc = new TestProxyClient("localhost", 42424);
-    PrincipalToken principalToken = new PrincipalToken("root", ByteBuffer.wrap("secret".getBytes()));
+    UserPass userPass = new UserPass("root", ByteBuffer.wrap("secret".getBytes()));
     
     System.out.println("Logging in");
+    ByteBuffer login = tpc.proxy.login(userPass);
     
     System.out.println("Creating user: ");
-    if (!tpc.proxy().listUsers(principalToken).contains("testuser")) {
-      tpc.proxy().createUser(principalToken, "testuser", ByteBuffer.wrap("testpass".getBytes()));
+    if (!tpc.proxy().listUsers(login).contains("testuser")) {
+      tpc.proxy().createUser(login, "testuser", ByteBuffer.wrap("testpass".getBytes()));
     }
-    System.out.println("UserList: " + tpc.proxy().listUsers(principalToken));
+    System.out.println("UserList: " + tpc.proxy().listUsers(login));
     
-    System.out.println("Listing: " + tpc.proxy().listTables(principalToken));
+    System.out.println("Listing: " + tpc.proxy().listTables(login));
     
     System.out.println("Deleting: ");
     String testTable = "testtableOMGOMGOMG";
     
     System.out.println("Creating: ");
     
-    if (tpc.proxy().tableExists(principalToken, testTable))
-      tpc.proxy().deleteTable(principalToken, testTable);
+    if (tpc.proxy().tableExists(login, testTable))
+      tpc.proxy().deleteTable(login, testTable);
     
-    tpc.proxy().createTable(principalToken, testTable, true, TimeType.MILLIS);
+    tpc.proxy().createTable(login, testTable, true, TimeType.MILLIS);
     
-    System.out.println("Listing: " + tpc.proxy().listTables(principalToken));
+    System.out.println("Listing: " + tpc.proxy().listTables(login));
     
     System.out.println("Writing: ");
     Date start = new Date();
@@ -101,16 +102,16 @@ public class TestProxyClient {
       mutations.put(ByteBuffer.wrap(result.getBytes()), Collections.singletonList(update));
       
       if (i % 1000 == 0) {
-        tpc.proxy().updateAndFlush(principalToken, testTable, mutations);
+        tpc.proxy().updateAndFlush(login, testTable, mutations);
         mutations.clear();
       }
     }
-    tpc.proxy().updateAndFlush(principalToken, testTable, mutations);
+    tpc.proxy().updateAndFlush(login, testTable, mutations);
     Date end = new Date();
     System.out.println(" End of writing: " + (end.getTime() - start.getTime()));
     
-    tpc.proxy().deleteTable(principalToken, testTable);
-    tpc.proxy().createTable(principalToken, testTable, true, TimeType.MILLIS);
+    tpc.proxy().deleteTable(login, testTable);
+    tpc.proxy().createTable(login, testTable, true, TimeType.MILLIS);
     
     // Thread.sleep(1000);
     
@@ -118,7 +119,7 @@ public class TestProxyClient {
     start = new Date();
     then = new Date();
     mutations.clear();
-    String writer = tpc.proxy().createWriter(principalToken, testTable, null);
+    String writer = tpc.proxy().createWriter(login, testTable, null);
     for (int i = 0; i < maxInserts; i++) {
       String result = String.format(format, i);
       Key pkey = new Key();
@@ -145,7 +146,7 @@ public class TestProxyClient {
     IteratorSetting is = new IteratorSetting(50, regex, RegExFilter.class);
     RegExFilter.setRegexs(is, null, regex, null, null, false);
     
-    String cookie = tpc.proxy().createScanner(principalToken, testTable, null);
+    String cookie = tpc.proxy().createScanner(login, testTable, null);
     
     int i = 0;
     start = new Date();