You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@accumulo.apache.org by kt...@apache.org on 2012/10/12 02:22:15 UTC

svn commit: r1397391 - in /accumulo/trunk: ./ core/ core/src/main/java/org/apache/accumulo/core/client/impl/ core/src/main/java/org/apache/accumulo/core/util/ core/src/test/java/org/apache/accumulo/core/client/impl/ fate/src/main/java/org/apache/accumu...

Author: kturner
Date: Fri Oct 12 00:22:14 2012
New Revision: 1397391

URL: http://svn.apache.org/viewvc?rev=1397391&view=rev
Log:
ACCUMULO-806 fixed bug where a tablet location could not be found when the metadata table had empty tablets (merged from 1.4)

Modified:
    accumulo/trunk/   (props changed)
    accumulo/trunk/core/   (props changed)
    accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/MetadataLocationObtainer.java
    accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocator.java
    accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocatorImpl.java
    accumulo/trunk/core/src/main/java/org/apache/accumulo/core/util/MetadataTable.java
    accumulo/trunk/core/src/test/java/org/apache/accumulo/core/client/impl/TabletLocatorImplTest.java
    accumulo/trunk/fate/src/main/java/org/apache/accumulo/fate/ZooStore.java   (props changed)
    accumulo/trunk/server/   (props changed)
    accumulo/trunk/src/   (props changed)

Propchange: accumulo/trunk/
------------------------------------------------------------------------------
  Merged /accumulo/branches/1.4/src:r1397383
  Merged /accumulo/branches/1.4:r1397383

Propchange: accumulo/trunk/core/
------------------------------------------------------------------------------
  Merged /accumulo/branches/1.4/core:r1397383
  Merged /accumulo/branches/1.4/src/core:r1397383

Modified: accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/MetadataLocationObtainer.java
URL: http://svn.apache.org/viewvc/accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/MetadataLocationObtainer.java?rev=1397391&r1=1397390&r2=1397391&view=diff
==============================================================================
--- accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/MetadataLocationObtainer.java (original)
+++ accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/MetadataLocationObtainer.java Fri Oct 12 00:22:14 2012
@@ -32,6 +32,7 @@ import org.apache.accumulo.core.client.A
 import org.apache.accumulo.core.client.AccumuloSecurityException;
 import org.apache.accumulo.core.client.Instance;
 import org.apache.accumulo.core.client.impl.TabletLocator.TabletLocation;
+import org.apache.accumulo.core.client.impl.TabletLocator.TabletLocations;
 import org.apache.accumulo.core.client.impl.TabletLocatorImpl.TabletLocationObtainer;
 import org.apache.accumulo.core.client.impl.TabletServerBatchReaderIterator.ResultReceiver;
 import org.apache.accumulo.core.data.Column;
@@ -44,6 +45,7 @@ import org.apache.accumulo.core.security
 import org.apache.accumulo.core.tabletserver.thrift.NotServingTabletException;
 import org.apache.accumulo.core.util.MetadataTable;
 import org.apache.accumulo.core.util.OpTimer;
+import org.apache.accumulo.core.util.Pair;
 import org.apache.accumulo.core.util.TextUtil;
 import org.apache.hadoop.io.Text;
 import org.apache.log4j.Level;
@@ -68,12 +70,12 @@ public class MetadataLocationObtainer im
   }
   
   @Override
-  public List<TabletLocation> lookupTablet(TabletLocation src, Text row, Text stopRow, TabletLocator parent) throws AccumuloSecurityException,
+  public TabletLocations lookupTablet(TabletLocation src, Text row, Text stopRow, TabletLocator parent) throws AccumuloSecurityException,
       AccumuloException {
-    
-    ArrayList<TabletLocation> list = new ArrayList<TabletLocation>();
-    
+
     try {
+      ArrayList<TabletLocation> list = new ArrayList<TabletLocation>();
+
       OpTimer opTimer = null;
       if (log.isTraceEnabled())
         opTimer = new OpTimer(log, Level.TRACE).start("Looking up in " + src.tablet_extent.getTableId() + " row=" + TextUtil.truncate(row) + "  extent="
@@ -98,12 +100,14 @@ public class MetadataLocationObtainer im
       
       // System.out.println("results "+results.keySet());
       
-      SortedMap<KeyExtent,Text> metadata = MetadataTable.getMetadataLocationEntries(results);
+      Pair<SortedMap<KeyExtent,Text>,List<KeyExtent>> metadata = MetadataTable.getMetadataLocationEntries(results);
       
-      for (Entry<KeyExtent,Text> entry : metadata.entrySet()) {
+      for (Entry<KeyExtent,Text> entry : metadata.getFirst().entrySet()) {
         list.add(new TabletLocation(entry.getKey(), entry.getValue().toString()));
       }
       
+      return new TabletLocations(list, metadata.getSecond());
+
     } catch (AccumuloServerException ase) {
       if (log.isTraceEnabled())
         log.trace(src.tablet_extent.getTableId() + " lookup failed, " + src.tablet_location + " server side exception");
@@ -118,7 +122,7 @@ public class MetadataLocationObtainer im
       parent.invalidateCache(src.tablet_location);
     }
     
-    return list;
+    return null;
   }
   
   @Override
@@ -161,7 +165,7 @@ public class MetadataLocationObtainer im
       throw e;
     }
     
-    SortedMap<KeyExtent,Text> metadata = MetadataTable.getMetadataLocationEntries(results);
+    SortedMap<KeyExtent,Text> metadata = MetadataTable.getMetadataLocationEntries(results).getFirst();
     
     for (Entry<KeyExtent,Text> entry : metadata.entrySet()) {
       list.add(new TabletLocation(entry.getKey(), entry.getValue().toString()));

Modified: accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocator.java
URL: http://svn.apache.org/viewvc/accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocator.java?rev=1397391&r1=1397390&r2=1397391&view=diff
==============================================================================
--- accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocator.java (original)
+++ accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocator.java Fri Oct 12 00:22:14 2012
@@ -126,6 +126,25 @@ public abstract class TabletLocator {
     return tl;
   }
   
+  public static class TabletLocations {
+    
+    private final List<TabletLocation> locations;
+    private final List<KeyExtent> locationless;
+    
+    public TabletLocations(List<TabletLocation> locations, List<KeyExtent> locationless) {
+      this.locations = locations;
+      this.locationless = locationless;
+    }
+    
+    public List<TabletLocation> getLocations() {
+      return locations;
+    }
+    
+    public List<KeyExtent> getLocationless() {
+      return locationless;
+    }
+  }
+
   public static class TabletLocation implements Comparable<TabletLocation> {
     private static WeakHashMap<String,WeakReference<String>> tabletLocs = new WeakHashMap<String,WeakReference<String>>();
     

Modified: accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocatorImpl.java
URL: http://svn.apache.org/viewvc/accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocatorImpl.java?rev=1397391&r1=1397390&r2=1397391&view=diff
==============================================================================
--- accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocatorImpl.java (original)
+++ accumulo/trunk/core/src/main/java/org/apache/accumulo/core/client/impl/TabletLocatorImpl.java Fri Oct 12 00:22:14 2012
@@ -91,7 +91,10 @@ public class TabletLocatorImpl extends T
   private Lock wLock = rwLock.writeLock();
   
   public static interface TabletLocationObtainer {
-    List<TabletLocation> lookupTablet(TabletLocation src, Text row, Text stopRow, TabletLocator parent) throws AccumuloSecurityException, AccumuloException;
+    /**
+     * @return null when unable to read information successfully
+     */
+    TabletLocations lookupTablet(TabletLocation src, Text row, Text stopRow, TabletLocator parent) throws AccumuloSecurityException, AccumuloException;
     
     List<TabletLocation> lookupTablets(String tserver, Map<KeyExtent,List<Range>> map, TabletLocator parent) throws AccumuloSecurityException,
         AccumuloException;
@@ -389,23 +392,31 @@ public class TabletLocatorImpl extends T
     TabletLocation ptl = parent.locateTablet(metadataRow, false, retry);
     
     if (ptl != null) {
-      List<TabletLocation> locations = locationObtainer.lookupTablet(ptl, metadataRow, lastTabletRow, parent);
-      if (locations.size() == 0 && !ptl.tablet_extent.isRootTablet()) {
-        // try the next tablet
+      TabletLocations locations = locationObtainer.lookupTablet(ptl, metadataRow, lastTabletRow, parent);
+      while (locations != null && locations.getLocations().isEmpty() && locations.getLocationless().isEmpty()
+ && !ptl.tablet_extent.isRootTablet()) {
+        // try the next tablet, the current tablet does not have any tablets that overlap the row
         Text er = ptl.tablet_extent.getEndRow();
         if (er != null && er.compareTo(lastTabletRow) < 0) {
           // System.out.println("er "+er+"  ltr "+lastTabletRow);
           ptl = parent.locateTablet(er, true, retry);
           if (ptl != null)
             locations = locationObtainer.lookupTablet(ptl, metadataRow, lastTabletRow, parent);
+          else
+            break;
+        } else {
+          break;
         }
       }
       
+      if (locations == null)
+        return;
+
       // cannot assume the list contains contiguous key extents... so it is probably
       // best to deal with each extent individually
       
       Text lastEndRow = null;
-      for (TabletLocation tabletLocation : locations) {
+      for (TabletLocation tabletLocation : locations.getLocations()) {
         
         KeyExtent ke = tabletLocation.tablet_extent;
         TabletLocation locToCache;

Modified: accumulo/trunk/core/src/main/java/org/apache/accumulo/core/util/MetadataTable.java
URL: http://svn.apache.org/viewvc/accumulo/trunk/core/src/main/java/org/apache/accumulo/core/util/MetadataTable.java?rev=1397391&r1=1397390&r2=1397391&view=diff
==============================================================================
--- accumulo/trunk/core/src/main/java/org/apache/accumulo/core/util/MetadataTable.java (original)
+++ accumulo/trunk/core/src/main/java/org/apache/accumulo/core/util/MetadataTable.java Fri Oct 12 00:22:14 2012
@@ -16,6 +16,7 @@
  */
 package org.apache.accumulo.core.util;
 
+import java.util.ArrayList;
 import java.util.HashSet;
 import java.util.Iterator;
 import java.util.List;
@@ -118,7 +119,7 @@ public class MetadataTable {
     }
   }
   
-  public static SortedMap<KeyExtent,Text> getMetadataLocationEntries(SortedMap<Key,Value> entries) {
+  public static Pair<SortedMap<KeyExtent,Text>,List<KeyExtent>> getMetadataLocationEntries(SortedMap<Key,Value> entries) {
     Key key;
     Value val;
     Text location = null;
@@ -126,6 +127,7 @@ public class MetadataTable {
     KeyExtent ke;
     
     SortedMap<KeyExtent,Text> results = new TreeMap<KeyExtent,Text>();
+    ArrayList<KeyExtent> locationless = new ArrayList<KeyExtent>();
     
     Text lastRowFromKey = new Text();
     
@@ -152,15 +154,19 @@ public class MetadataTable {
       else if (Constants.METADATA_PREV_ROW_COLUMN.equals(colf, colq))
         prevRow = new Value(val);
       
-      if (location != null && prevRow != null) {
+      if (prevRow != null) {
         ke = new KeyExtent(key.getRow(), prevRow);
-        results.put(ke, location);
-        
+        if (location != null)
+          results.put(ke, location);
+        else
+          locationless.add(ke);
+
         location = null;
         prevRow = null;
       }
     }
-    return results;
+    
+    return new Pair<SortedMap<KeyExtent,Text>,List<KeyExtent>>(results, locationless);
   }
   
   public static SortedMap<Text,SortedMap<ColumnFQ,Value>> getTabletEntries(Instance instance, KeyExtent ke, List<ColumnFQ> columns, AuthInfo credentials) {

Modified: accumulo/trunk/core/src/test/java/org/apache/accumulo/core/client/impl/TabletLocatorImplTest.java
URL: http://svn.apache.org/viewvc/accumulo/trunk/core/src/test/java/org/apache/accumulo/core/client/impl/TabletLocatorImplTest.java?rev=1397391&r1=1397390&r2=1397391&view=diff
==============================================================================
--- accumulo/trunk/core/src/test/java/org/apache/accumulo/core/client/impl/TabletLocatorImplTest.java (original)
+++ accumulo/trunk/core/src/test/java/org/apache/accumulo/core/client/impl/TabletLocatorImplTest.java Fri Oct 12 00:22:14 2012
@@ -37,6 +37,7 @@ import org.apache.accumulo.core.client.A
 import org.apache.accumulo.core.client.Connector;
 import org.apache.accumulo.core.client.Instance;
 import org.apache.accumulo.core.client.impl.TabletLocator.TabletLocation;
+import org.apache.accumulo.core.client.impl.TabletLocator.TabletLocations;
 import org.apache.accumulo.core.client.impl.TabletLocator.TabletServerMutations;
 import org.apache.accumulo.core.client.impl.TabletLocatorImpl.TabletLocationObtainer;
 import org.apache.accumulo.core.conf.AccumuloConfiguration;
@@ -48,6 +49,7 @@ import org.apache.accumulo.core.data.Ran
 import org.apache.accumulo.core.data.Value;
 import org.apache.accumulo.core.security.thrift.AuthInfo;
 import org.apache.accumulo.core.util.MetadataTable;
+import org.apache.accumulo.core.util.Pair;
 import org.apache.hadoop.io.Text;
 
 public class TabletLocatorImplTest extends TestCase {
@@ -461,7 +463,7 @@ public class TabletLocatorImplTest exten
     }
     
     @Override
-    public List<TabletLocation> lookupTablet(TabletLocation src, Text row, Text stopRow, TabletLocator parent) throws AccumuloSecurityException {
+    public TabletLocations lookupTablet(TabletLocation src, Text row, Text stopRow, TabletLocator parent) throws AccumuloSecurityException {
       
       // System.out.println("lookupTablet("+src+","+row+","+stopRow+","+ parent+")");
       // System.out.println(tservers);
@@ -472,14 +474,14 @@ public class TabletLocatorImplTest exten
       
       if (tablets == null) {
         parent.invalidateCache(src.tablet_location);
-        return list;
+        return null;
       }
       
       SortedMap<Key,Value> tabletData = tablets.get(src.tablet_extent);
       
       if (tabletData == null) {
         parent.invalidateCache(src.tablet_extent);
-        return list;
+        return null;
       }
       
       // the following clip is done on a tablet, do it here to see if it throws exceptions
@@ -490,13 +492,13 @@ public class TabletLocatorImplTest exten
       
       SortedMap<Key,Value> results = tabletData.tailMap(startKey).headMap(stopKey);
       
-      SortedMap<KeyExtent,Text> metadata = MetadataTable.getMetadataLocationEntries(results);
+      Pair<SortedMap<KeyExtent,Text>,List<KeyExtent>> metadata = MetadataTable.getMetadataLocationEntries(results);
       
-      for (Entry<KeyExtent,Text> entry : metadata.entrySet()) {
+      for (Entry<KeyExtent,Text> entry : metadata.getFirst().entrySet()) {
         list.add(new TabletLocation(entry.getKey(), entry.getValue().toString()));
       }
       
-      return list;
+      return new TabletLocations(list, metadata.getSecond());
     }
     
     @Override
@@ -545,7 +547,7 @@ public class TabletLocatorImplTest exten
       if (failures.size() > 0)
         parent.invalidateCache(failures);
       
-      SortedMap<KeyExtent,Text> metadata = MetadataTable.getMetadataLocationEntries(results);
+      SortedMap<KeyExtent,Text> metadata = MetadataTable.getMetadataLocationEntries(results).getFirst();
       
       for (Entry<KeyExtent,Text> entry : metadata.entrySet()) {
         list.add(new TabletLocation(entry.getKey(), entry.getValue().toString()));
@@ -557,6 +559,22 @@ public class TabletLocatorImplTest exten
     
   }
   
+  static void createEmptyTablet(TServers tservers, String server, KeyExtent tablet) {
+    Map<KeyExtent,SortedMap<Key,Value>> tablets = tservers.tservers.get(server);
+    if (tablets == null) {
+      tablets = new HashMap<KeyExtent,SortedMap<Key,Value>>();
+      tservers.tservers.put(server, tablets);
+    }
+    
+    SortedMap<Key,Value> tabletData = tablets.get(tablet);
+    if (tabletData == null) {
+      tabletData = new TreeMap<Key,Value>();
+      tablets.put(tablet, tabletData);
+    } else if (tabletData.size() > 0) {
+      throw new RuntimeException("Asked for empty tablet, but non empty tablet exists");
+    }
+  }
+
   static void setLocation(TServers tservers, String server, KeyExtent tablet, KeyExtent ke, String location) {
     Map<KeyExtent,SortedMap<Key,Value>> tablets = tservers.tservers.get(server);
     if (tablets == null) {
@@ -1185,4 +1203,40 @@ public class TabletLocatorImplTest exten
     assertNull(tab0TabletCache.locateTablet(new Text("row_0000000000"), false, false));
     
   }
+  
+  // this test reproduces a problem where empty metadata tablets, that were created by user tablets being merged away, caused locating tablets to fail
+  public void testBug3() throws Exception {
+    
+    KeyExtent mte1 = new KeyExtent(new Text(Constants.METADATA_TABLE_ID), new Text("1;c"), RTE.getEndRow());
+    KeyExtent mte2 = new KeyExtent(new Text(Constants.METADATA_TABLE_ID), new Text("1;f"), new Text("1;c"));
+    KeyExtent mte3 = new KeyExtent(new Text(Constants.METADATA_TABLE_ID), new Text("1;j"), new Text("1;f"));
+    KeyExtent mte4 = new KeyExtent(new Text(Constants.METADATA_TABLE_ID), new Text("1;r"), new Text("1;j"));
+    KeyExtent mte5 = new KeyExtent(new Text(Constants.METADATA_TABLE_ID), null, new Text("1;r"));
+    
+    KeyExtent ke1 = new KeyExtent(new Text("1"), null, null);
+    
+    TServers tservers = new TServers();
+    TestTabletLocationObtainer ttlo = new TestTabletLocationObtainer(tservers);
+    TestInstance testInstance = new TestInstance("instance1", "tserver1");
+    
+    RootTabletLocator rtl = new RootTabletLocator(testInstance);
+    
+    TabletLocatorImpl rootTabletCache = new TabletLocatorImpl(new Text(Constants.METADATA_TABLE_ID), rtl, ttlo);
+    TabletLocatorImpl tab0TabletCache = new TabletLocatorImpl(new Text("1"), rootTabletCache, ttlo);
+    
+    setLocation(tservers, "tserver1", RTE, mte1, "tserver2");
+    setLocation(tservers, "tserver1", RTE, mte2, "tserver3");
+    setLocation(tservers, "tserver1", RTE, mte3, "tserver4");
+    setLocation(tservers, "tserver1", RTE, mte4, "tserver5");
+    setLocation(tservers, "tserver1", RTE, mte5, "tserver6");
+    
+    createEmptyTablet(tservers, "tserver2", mte1);
+    createEmptyTablet(tservers, "tserver3", mte2);
+    createEmptyTablet(tservers, "tserver4", mte3);
+    createEmptyTablet(tservers, "tserver5", mte4);
+    setLocation(tservers, "tserver6", mte5, ke1, "tserver7");
+    
+    locateTabletTest(tab0TabletCache, "a", ke1, "tserver7");
+    
+  }
 }

Propchange: accumulo/trunk/fate/src/main/java/org/apache/accumulo/fate/ZooStore.java
------------------------------------------------------------------------------
  Merged /accumulo/branches/1.4/fate/src/main/java/org/apache/accumulo/fate/ZooStore.java:r1397383
  Merged /accumulo/branches/1.4/src/fate/src/main/java/org/apache/accumulo/fate/ZooStore.java:r1397383

Propchange: accumulo/trunk/server/
------------------------------------------------------------------------------
  Merged /accumulo/branches/1.4/server:r1397383
  Merged /accumulo/branches/1.4/src/server:r1397383

Propchange: accumulo/trunk/src/
------------------------------------------------------------------------------
  Merged /accumulo/branches/1.4/src:r1397383
  Merged /accumulo/branches/1.4/src/src:r1397383