You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by ns...@apache.org on 2011/10/11 04:14:52 UTC

svn commit: r1181509 - in /hbase/branches/0.89/src: main/java/org/apache/hadoop/hbase/regionserver/MemStore.java main/java/org/apache/hadoop/hbase/util/ManualEnvironmentEdge.java test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java

Author: nspiegelberg
Date: Tue Oct 11 02:14:51 2011
New Revision: 1181509

URL: http://svn.apache.org/viewvc?rev=1181509&view=rev
Log:
Importing HBASE-3235.

Added:
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/util/ManualEnvironmentEdge.java
Modified:
    hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStore.java
    hbase/branches/0.89/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java

Modified: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStore.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStore.java?rev=1181509&r1=1181508&r2=1181509&view=diff
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStore.java (original)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/regionserver/MemStore.java Tue Oct 11 02:14:51 2011
@@ -376,7 +376,11 @@ public class MemStore implements HeapSiz
       long addedSize = add(newKv);
 
       // now find and RM the old one(s) to prevent version explosion:
-      SortedSet<KeyValue> ss = kvset.tailSet(newKv);
+	  KeyValue firstKv = KeyValue.createFirstOnRow(
+			  newKv.getBuffer(), newKv.getRowOffset(), newKv.getRowLength(),
+			  newKv.getBuffer(), newKv.getFamilyOffset(), newKv.getFamilyLength(),
+			  newKv.getBuffer(), newKv.getQualifierOffset(), newKv.getQualifierLength());
+      SortedSet<KeyValue> ss = kvset.tailSet(firstKv);
       Iterator<KeyValue> it = ss.iterator();
       while ( it.hasNext() ) {
         KeyValue kv = it.next();
@@ -397,9 +401,7 @@ public class MemStore implements HeapSiz
             newKv.getBuffer(), newKv.getQualifierOffset(), newKv.getQualifierLength(),
             kv.getBuffer(), kv.getQualifierOffset(), kv.getQualifierLength())) {
 
-          // to be extra safe we only remove Puts that have a memstoreTS==0
-          if (kv.getType() == KeyValue.Type.Put.getCode() &&
-              kv.getMemstoreTS() == 0) {
+          if (kv.getType() == KeyValue.Type.Put.getCode()) {
             // false means there was a change, so give us the size.
             addedSize -= heapSizeChange(kv, true);
 

Added: hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/util/ManualEnvironmentEdge.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/util/ManualEnvironmentEdge.java?rev=1181509&view=auto
==============================================================================
--- hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/util/ManualEnvironmentEdge.java (added)
+++ hbase/branches/0.89/src/main/java/org/apache/hadoop/hbase/util/ManualEnvironmentEdge.java Tue Oct 11 02:14:51 2011
@@ -0,0 +1,39 @@
+/*
+ * Copyright 2010 The Apache Software Foundation
+ *
+ * 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.hadoop.hbase.util;
+
+/**
+ * An environment edge that uses a manually set value. This is useful for testing events that are supposed to
+ * happen in the same millisecond.
+ */
+public class ManualEnvironmentEdge implements EnvironmentEdge {
+
+  // Sometimes 0 ts might have a special value, so lets start with 1
+  protected long value = 1L;
+
+  public void setValue(long newValue) {
+    value = newValue;
+  }
+
+  @Override
+  public long currentTimeMillis() {
+    return this.value;
+  }
+}

Modified: hbase/branches/0.89/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java
URL: http://svn.apache.org/viewvc/hbase/branches/0.89/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java?rev=1181509&r1=1181508&r2=1181509&view=diff
==============================================================================
--- hbase/branches/0.89/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java (original)
+++ hbase/branches/0.89/src/test/java/org/apache/hadoop/hbase/regionserver/TestHRegion.java Tue Oct 11 02:14:51 2011
@@ -49,8 +49,11 @@ import org.apache.hadoop.hbase.filter.Si
 import org.apache.hadoop.hbase.regionserver.HRegion.RegionScanner;
 import org.apache.hadoop.hbase.regionserver.wal.HLog;
 import org.apache.hadoop.hbase.util.Bytes;
+import org.apache.hadoop.hbase.util.EnvironmentEdge;
+import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
 import org.apache.hadoop.hbase.util.EnvironmentEdgeManagerTestHelper;
 import org.apache.hadoop.hbase.util.IncrementingEnvironmentEdge;
+import org.apache.hadoop.hbase.util.ManualEnvironmentEdge;
 import org.apache.hadoop.hbase.util.Pair;
 import org.apache.hadoop.hbase.util.Threads;
 
@@ -1859,8 +1862,8 @@ public class TestHRegion extends HBaseTe
     assertEquals(value+amount, result);
 
     Store store = region.getStore(fam1);
-    // we will have the original Put, and also the ICV'ed Put as well.
-    assertEquals(2, store.memstore.kvset.size());
+    // ICV removes any extra values floating around in there.
+    assertEquals(1, store.memstore.kvset.size());
     assertTrue(store.memstore.snapshot.isEmpty());
 
     assertICV(row, fam1, qual1, value+amount);
@@ -2022,6 +2025,59 @@ public class TestHRegion extends HBaseTe
     assertICV(row, fam1, qual3, amount);
   }
 
+  /**
+   * Added for HBASE-3235.
+   *
+   * When the initial put and an ICV update were arriving with the same timestamp,
+   * the initial Put KV was being skipped during {@link MemStore#upsert(KeyValue)}
+   * causing the iteration for matching KVs, causing the update-in-place to not
+   * happen and the ICV put to effectively disappear.
+   * @throws IOException
+   */
+  public void testIncrementColumnValue_UpdatingInPlace_TimestampClobber() throws IOException {
+    initHRegion(tableName, getName(), fam1);
+
+    long value = 1L;
+    long amount = 3L;
+    long now = EnvironmentEdgeManager.currentTimeMillis();
+    ManualEnvironmentEdge mock = new ManualEnvironmentEdge();
+    mock.setValue(now);
+    EnvironmentEdgeManagerTestHelper.injectEdge(mock);
+
+    // verify we catch an ICV on a put with the same timestamp
+    Put put = new Put(row);
+    put.add(fam1, qual1, now, Bytes.toBytes(value));
+    region.put(put);
+
+    long result = region.incrementColumnValue(row, fam1, qual1, amount, true);
+
+    assertEquals(value+amount, result);
+
+    Store store = region.getStore(fam1);
+    // ICV should update the existing Put with the same timestamp
+    assertEquals(1, store.memstore.kvset.size());
+    assertTrue(store.memstore.snapshot.isEmpty());
+
+    assertICV(row, fam1, qual1, value+amount);
+
+    // verify we catch an ICV even when the put ts > now
+    put = new Put(row);
+    put.add(fam1, qual2, now+1, Bytes.toBytes(value));
+    region.put(put);
+
+    result = region.incrementColumnValue(row, fam1, qual2, amount, true);
+
+    assertEquals(value+amount, result);
+
+    store = region.getStore(fam1);
+    // ICV should update the existing Put with the same timestamp
+    assertEquals(2, store.memstore.kvset.size());
+    assertTrue(store.memstore.snapshot.isEmpty());
+
+    assertICV(row, fam1, qual2, value+amount);
+    EnvironmentEdgeManagerTestHelper.reset();
+  }
+
   private void assertICV(byte [] row,
                          byte [] familiy,
                          byte[] qualifier,