Browse Source

HADOOP-2262 HADOOP-2261 fail fast on non-existing table, change abort to function after commit even if commit was successful

git-svn-id: https://svn.apache.org/repos/asf/lucene/hadoop/trunk@598555 13f79535-47bb-0310-9956-ffa450edef68
Jim Kellerman 17 năm trước cách đây
mục cha
commit
374a2ab960

+ 2 - 0
src/contrib/hbase/CHANGES.txt

@@ -51,6 +51,8 @@ Trunk (unreleased changes)
     HADOOP-2139 (phase 2) Make region server more event driven
     HADOOP-2289 Useless efforts of looking for the non-existant table in select
                 command.
+    HADOOP-2262 HADOOP-2261 fail fast on non-existing table, change abort to
+                function after commit even if commit was successful
 
 Release 0.15.1
 Branch 0.15

+ 4 - 1
src/contrib/hbase/src/java/org/apache/hadoop/hbase/HConnectionManager.java

@@ -738,7 +738,10 @@ public class HConnectionManager implements HConstants {
                 regionInfo, new HServerAddress(serverAddress)));
           }
         } catch (IOException e) {
-          if (tries == numRetries - 1) {                 // no retries left
+          if (e instanceof TableNotFoundException) {
+            throw e;                                    // don't retry
+          }
+          if (tries == numRetries - 1) {                // no retries left
             if (e instanceof RemoteException) {
               e = RemoteExceptionHandler.decodeRemoteException((RemoteException) e);
             }

+ 9 - 3
src/contrib/hbase/src/java/org/apache/hadoop/hbase/HTable.java

@@ -57,12 +57,17 @@ public class HTable implements HConstants {
   protected Random rand;
   protected volatile SortedMap<Text, HRegionLocation> tableServers;
   protected AtomicReference<BatchUpdate> batch;
+
+  protected volatile boolean tableDoesNotExist;
   
   // For row mutation operations
   
   protected volatile boolean closed;
 
   protected void checkClosed() {
+    if (tableDoesNotExist) {
+      throw new IllegalStateException("table does not exist: " + tableName);
+    }
     if (closed) {
       throw new IllegalStateException("table is closed");
     }
@@ -77,13 +82,15 @@ public class HTable implements HConstants {
    */
   public HTable(HBaseConfiguration conf, Text tableName) throws IOException {
     closed = true;
+    tableDoesNotExist = true;
     this.connection = HConnectionManager.getConnection(conf);
     this.tableName = tableName;
     this.pause = conf.getLong("hbase.client.pause", 10 * 1000);
     this.numRetries = conf.getInt("hbase.client.retries.number", 5);
     this.rand = new Random();
-    tableServers = connection.getTableServers(tableName);
     this.batch = new AtomicReference<BatchUpdate>();
+    tableServers = connection.getTableServers(tableName);
+    tableDoesNotExist = false;
     closed = false;
   }
 
@@ -685,8 +692,7 @@ public class HTable implements HConstants {
    */
   public synchronized void abort(long lockid) {
     checkClosed();
-    updateInProgress(true);
-    if (batch.get().getLockid() != lockid) {
+    if (batch.get() != null && batch.get().getLockid() != lockid) {
       throw new IllegalArgumentException("invalid lock id " + lockid);
     }
     batch.set(null);

+ 16 - 3
src/contrib/hbase/src/test/org/apache/hadoop/hbase/TestHTable.java

@@ -31,7 +31,8 @@ import org.apache.hadoop.io.Text;
 public class TestHTable extends HBaseClusterTestCase implements HConstants {
   private static final HColumnDescriptor column =
     new HColumnDescriptor(COLUMN_FAMILY.toString());
-  
+
+  private static final Text nosuchTable = new Text("nosuchTable");
   private static final Text tableAname = new Text("tableA");
   private static final Text tableBname = new Text("tableB");
   
@@ -42,6 +43,19 @@ public class TestHTable extends HBaseClusterTestCase implements HConstants {
    * @throws IOException
    */
   public void testHTable() throws IOException {
+    byte[] value = "value".getBytes(UTF8_ENCODING);
+    
+    try {
+      new HTable(conf, nosuchTable);
+      
+    } catch (TableNotFoundException e) {
+      // expected
+
+    } catch (IOException e) {
+      e.printStackTrace();
+      fail();
+    }
+    
     HTableDescriptor tableAdesc = new HTableDescriptor(tableAname.toString());
     tableAdesc.addFamily(column);
     
@@ -56,8 +70,6 @@ public class TestHTable extends HBaseClusterTestCase implements HConstants {
     
     // put some data into table A
     
-    byte[] value = "value".getBytes(UTF8_ENCODING);
-    
     HTable a = new HTable(conf, tableAname);
     long lockid = a.startUpdate(row);
     a.put(lockid, COLUMN_FAMILY, value);
@@ -82,6 +94,7 @@ public class TestHTable extends HBaseClusterTestCase implements HConstants {
           b.put(lockid, e.getKey(), e.getValue());
         }
         b.commit(lockid);
+        b.abort(lockid);
       }
     } finally {
       s.close();