You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by te...@apache.org on 2011/12/09 07:01:58 UTC

svn commit: r1212252 - in /hbase/branches/0.92: ./ src/main/java/org/apache/hadoop/hbase/client/coprocessor/ src/main/java/org/apache/hadoop/hbase/regionserver/ src/test/java/org/apache/hadoop/hbase/coprocessor/

Author: tedyu
Date: Fri Dec  9 06:01:58 2011
New Revision: 1212252

URL: http://svn.apache.org/viewvc?rev=1212252&view=rev
Log:
HBASE-4946  HTable.coprocessorExec (and possibly coprocessorProxy) does not work with
               dynamically loaded coprocessors (Andrei Dragomir)

Modified:
    hbase/branches/0.92/CHANGES.txt
    hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/client/coprocessor/Exec.java
    hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java
    hbase/branches/0.92/src/test/java/org/apache/hadoop/hbase/coprocessor/TestCoprocessorEndpoint.java

Modified: hbase/branches/0.92/CHANGES.txt
URL: http://svn.apache.org/viewvc/hbase/branches/0.92/CHANGES.txt?rev=1212252&r1=1212251&r2=1212252&view=diff
==============================================================================
--- hbase/branches/0.92/CHANGES.txt (original)
+++ hbase/branches/0.92/CHANGES.txt Fri Dec  9 06:01:58 2011
@@ -467,6 +467,8 @@ Release 0.92.0 - Unreleased
                internal invocations of other hbase scripts
    HBASE-4980  Null pointer exception in HBaseClient receiveResponse
                (Shrijeet Paliwal)
+   HBASE-4946  HTable.coprocessorExec (and possibly coprocessorProxy) does not work with
+               dynamically loaded coprocessors (Andrei Dragomir)
 
   TESTS
    HBASE-4492  TestRollingRestart fails intermittently

Modified: hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/client/coprocessor/Exec.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/client/coprocessor/Exec.java?rev=1212252&r1=1212251&r2=1212252&view=diff
==============================================================================
--- hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/client/coprocessor/Exec.java (original)
+++ hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/client/coprocessor/Exec.java Fri Dec  9 06:01:58 2011
@@ -56,6 +56,7 @@ public class Exec extends Invocation imp
   /** Row key used as a reference for any region lookups */
   private byte[] referenceRow;
   private Class<? extends CoprocessorProtocol> protocol;
+  private String protocolName;
 
   public Exec() {
   }
@@ -68,6 +69,11 @@ public class Exec extends Invocation imp
     this.conf = configuration;
     this.referenceRow = row;
     this.protocol = protocol;
+    this.protocolName = protocol.getName();
+  }
+
+  public String getProtocolName() {
+    return protocolName;
   }
 
   public Class<? extends CoprocessorProtocol> getProtocol() {
@@ -117,12 +123,6 @@ public class Exec extends Invocation imp
     }
     // fields for Exec
     referenceRow = Bytes.readByteArray(in);
-    String protocolName = in.readUTF();
-    try {
-      protocol = (Class<CoprocessorProtocol>)conf.getClassByName(protocolName);
-    }
-    catch (ClassNotFoundException cnfe) {
-      throw new IOException("Protocol class "+protocolName+" not found", cnfe);
-    }
+    protocolName = in.readUTF();
   }
 }

Modified: hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java?rev=1212252&r1=1212251&r2=1212252&view=diff
==============================================================================
--- hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java (original)
+++ hbase/branches/0.92/src/main/java/org/apache/hadoop/hbase/regionserver/HRegion.java Fri Dec  9 06:01:58 2011
@@ -49,6 +49,7 @@ import java.util.concurrent.atomic.Atomi
 import java.util.concurrent.atomic.AtomicLong;
 import java.util.concurrent.locks.ReentrantReadWriteLock;
 
+import com.google.common.collect.Maps;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
@@ -178,6 +179,9 @@ public class HRegion implements HeapSize
   // Registered region protocol handlers
   private ClassToInstanceMap<CoprocessorProtocol>
       protocolHandlers = MutableClassToInstanceMap.create();
+  
+  private Map<String, Class<? extends CoprocessorProtocol>>
+      protocolHandlerNames = Maps.newHashMap();
 
   //These variable are just used for getting data out of the region, to test on
   //client side
@@ -3980,6 +3984,7 @@ public class HRegion implements HeapSize
     }
 
     protocolHandlers.putInstance(protocol, handler);
+    protocolHandlerNames.put(protocol.getName(), protocol);
     if (LOG.isDebugEnabled()) {
       LOG.debug("Registered protocol handler: region="+
           Bytes.toStringBinary(getRegionName())+" protocol="+protocol.getName());
@@ -4005,6 +4010,19 @@ public class HRegion implements HeapSize
   public ExecResult exec(Exec call)
       throws IOException {
     Class<? extends CoprocessorProtocol> protocol = call.getProtocol();
+    if (protocol == null) {
+      String protocolName = call.getProtocolName();
+      if (LOG.isDebugEnabled()) {
+        LOG.debug("Received dynamic protocol exec call with protocolName " + protocolName);
+      }
+      // detect the actual protocol class
+      protocol  = protocolHandlerNames.get(protocolName);
+      if (protocol == null) {
+        throw new HBaseRPC.UnknownProtocolException(protocol,
+            "No matching handler for protocol "+protocolName+
+            " in region "+Bytes.toStringBinary(getRegionName()));
+      }
+    }
     if (!protocolHandlers.containsKey(protocol)) {
       throw new HBaseRPC.UnknownProtocolException(protocol,
           "No matching handler for protocol "+protocol.getName()+

Modified: hbase/branches/0.92/src/test/java/org/apache/hadoop/hbase/coprocessor/TestCoprocessorEndpoint.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.92/src/test/java/org/apache/hadoop/hbase/coprocessor/TestCoprocessorEndpoint.java?rev=1212252&r1=1212251&r2=1212252&view=diff
==============================================================================
--- hbase/branches/0.92/src/test/java/org/apache/hadoop/hbase/coprocessor/TestCoprocessorEndpoint.java (original)
+++ hbase/branches/0.92/src/test/java/org/apache/hadoop/hbase/coprocessor/TestCoprocessorEndpoint.java Fri Dec  9 06:01:58 2011
@@ -33,8 +33,13 @@ import org.apache.hadoop.hbase.client.HT
 import org.apache.hadoop.hbase.client.Put;
 import org.apache.hadoop.hbase.client.Scan;
 import org.apache.hadoop.hbase.client.coprocessor.Batch;
+import org.apache.hadoop.hbase.client.coprocessor.Exec;
+import org.apache.hadoop.hbase.io.HbaseObjectWritable;
 import org.apache.hadoop.hbase.util.Bytes;
+import org.apache.hadoop.io.DataInputBuffer;
+import org.apache.hadoop.io.DataOutputBuffer;
 import org.apache.hadoop.io.Text;
+import org.apache.hadoop.io.Writable;
 import org.junit.AfterClass;
 import org.junit.BeforeClass;
 import org.junit.Test;
@@ -48,6 +53,9 @@ public class TestCoprocessorEndpoint {
   private static final byte[] TEST_FAMILY = Bytes.toBytes("TestFamily");
   private static final byte[] TEST_QUALIFIER = Bytes.toBytes("TestQualifier");
   private static byte[] ROW = Bytes.toBytes("testRow");
+  
+  private static final String protocolName =  "org.apache.hadoop.hbase.CustomProtocol";
+  private static final String methodName = "myFunc";
 
   private static final int ROWSIZE = 20;
   private static final int rowSeperator1 = 5;
@@ -171,6 +179,28 @@ public class TestCoprocessorEndpoint {
     assertEquals("Invalid result", sumResult, expectedResult);
   }
 
+  @Test
+  public void testExecDeserialization() throws IOException {
+    DataOutputBuffer dob = new DataOutputBuffer();
+    dob.writeUTF(methodName);
+    dob.writeInt(1);
+    Scan scan = new Scan();
+    HbaseObjectWritable.writeObject(dob, scan, Scan.class, new Configuration());
+    dob.writeUTF("org.apache.hadoop.hbase.client.Scan");
+    Bytes.writeByteArray(dob, new byte[]{'a'});
+    // this is the dynamic protocol name
+    dob.writeUTF(protocolName);
+
+    DataInputBuffer dib = new DataInputBuffer();
+    dib.reset(dob.getData(), dob.getLength());
+
+    Exec after = new Exec();
+    after.readFields(dib);
+    // no error thrown
+    assertEquals(after.getProtocolName(), protocolName);
+    assertEquals(after.getMethodName(), methodName);
+  }
+
   private static byte[][] makeN(byte[] base, int n) {
     byte[][] ret = new byte[n][];
     for (int i = 0; i < n; i++) {