HBASE-14273 Rename MVCC to MVCC: From MultiVersionConsistencyControl to MultiVersionConcurrencyControl (Lars Francke)

This commit is contained in:
stack 2015-08-24 08:47:26 -07:00
parent eb52529c02
commit 9334a47d45
7 changed files with 33 additions and 33 deletions

View File

@ -146,7 +146,7 @@ import org.apache.hadoop.hbase.protobuf.generated.WALProtos.FlushDescriptor.Stor
import org.apache.hadoop.hbase.protobuf.generated.WALProtos.RegionEventDescriptor; import org.apache.hadoop.hbase.protobuf.generated.WALProtos.RegionEventDescriptor;
import org.apache.hadoop.hbase.protobuf.generated.WALProtos.RegionEventDescriptor.EventType; import org.apache.hadoop.hbase.protobuf.generated.WALProtos.RegionEventDescriptor.EventType;
import org.apache.hadoop.hbase.protobuf.generated.WALProtos.StoreDescriptor; import org.apache.hadoop.hbase.protobuf.generated.WALProtos.StoreDescriptor;
import org.apache.hadoop.hbase.regionserver.MultiVersionConsistencyControl.WriteEntry; import org.apache.hadoop.hbase.regionserver.MultiVersionConcurrencyControl.WriteEntry;
import org.apache.hadoop.hbase.regionserver.ScannerContext.LimitScope; import org.apache.hadoop.hbase.regionserver.ScannerContext.LimitScope;
import org.apache.hadoop.hbase.regionserver.ScannerContext.NextState; import org.apache.hadoop.hbase.regionserver.ScannerContext.NextState;
import org.apache.hadoop.hbase.regionserver.compactions.CompactionContext; import org.apache.hadoop.hbase.regionserver.compactions.CompactionContext;
@ -584,8 +584,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
private boolean splitRequest; private boolean splitRequest;
private byte[] explicitSplitPoint = null; private byte[] explicitSplitPoint = null;
private final MultiVersionConsistencyControl mvcc = private final MultiVersionConcurrencyControl mvcc =
new MultiVersionConsistencyControl(); new MultiVersionConcurrencyControl();
// Coprocessor host // Coprocessor host
private RegionCoprocessorHost coprocessorHost; private RegionCoprocessorHost coprocessorHost;
@ -1252,7 +1252,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
} }
public MultiVersionConsistencyControl getMVCC() { public MultiVersionConcurrencyControl getMVCC() {
return mvcc; return mvcc;
} }
@ -2081,7 +2081,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
if (this.memstoreSize.get() <= 0) { if (this.memstoreSize.get() <= 0) {
// Take an update lock because am about to change the sequence id and we want the sequence id // Take an update lock because am about to change the sequence id and we want the sequence id
// to be at the border of the empty memstore. // to be at the border of the empty memstore.
MultiVersionConsistencyControl.WriteEntry w = null; MultiVersionConcurrencyControl.WriteEntry w = null;
this.updatesLock.writeLock().lock(); this.updatesLock.writeLock().lock();
try { try {
if (this.memstoreSize.get() <= 0) { if (this.memstoreSize.get() <= 0) {
@ -2137,7 +2137,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
// to do this for a moment. It is quick. We also set the memstore size to zero here before we // to do this for a moment. It is quick. We also set the memstore size to zero here before we
// allow updates again so its value will represent the size of the updates received // allow updates again so its value will represent the size of the updates received
// during flush // during flush
MultiVersionConsistencyControl.WriteEntry w = null; MultiVersionConcurrencyControl.WriteEntry w = null;
// We have to take an update lock during snapshot, or else a write could end up in both snapshot // We have to take an update lock during snapshot, or else a write could end up in both snapshot
// and memstore (makes it difficult to do atomic rows then) // and memstore (makes it difficult to do atomic rows then)
status.setStatus("Obtaining lock to block concurrent updates"); status.setStatus("Obtaining lock to block concurrent updates");
@ -2853,7 +2853,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
long currentNonceGroup = HConstants.NO_NONCE, currentNonce = HConstants.NO_NONCE; long currentNonceGroup = HConstants.NO_NONCE, currentNonce = HConstants.NO_NONCE;
WALEdit walEdit = new WALEdit(isInReplay); WALEdit walEdit = new WALEdit(isInReplay);
MultiVersionConsistencyControl.WriteEntry w = null; MultiVersionConcurrencyControl.WriteEntry w = null;
long txid = 0; long txid = 0;
boolean doRollBackMemstore = false; boolean doRollBackMemstore = false;
boolean locked = false; boolean locked = false;
@ -3000,7 +3000,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
if(isInReplay) { if(isInReplay) {
mvccNum = batchOp.getReplaySequenceId(); mvccNum = batchOp.getReplaySequenceId();
} else { } else {
mvccNum = MultiVersionConsistencyControl.getPreAssignedWriteNumber(this.sequenceId); mvccNum = MultiVersionConcurrencyControl.getPreAssignedWriteNumber(this.sequenceId);
} }
// //
// ------------------------------------ // ------------------------------------
@ -6635,7 +6635,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
return; return;
} }
MultiVersionConsistencyControl.WriteEntry writeEntry = null; MultiVersionConcurrencyControl.WriteEntry writeEntry = null;
boolean locked; boolean locked;
boolean walSyncSuccessful = false; boolean walSyncSuccessful = false;
List<RowLock> acquiredRowLocks; List<RowLock> acquiredRowLocks;
@ -6656,7 +6656,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
lock(this.updatesLock.readLock(), acquiredRowLocks.size() == 0 ? 1 : acquiredRowLocks.size()); lock(this.updatesLock.readLock(), acquiredRowLocks.size() == 0 ? 1 : acquiredRowLocks.size());
locked = true; locked = true;
// Get a mvcc write number // Get a mvcc write number
mvccNum = MultiVersionConsistencyControl.getPreAssignedWriteNumber(this.sequenceId); mvccNum = MultiVersionConcurrencyControl.getPreAssignedWriteNumber(this.sequenceId);
long now = EnvironmentEdgeManager.currentTime(); long now = EnvironmentEdgeManager.currentTime();
try { try {
@ -6853,7 +6853,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
} }
// now start my own transaction // now start my own transaction
mvccNum = MultiVersionConsistencyControl.getPreAssignedWriteNumber(this.sequenceId); mvccNum = MultiVersionConcurrencyControl.getPreAssignedWriteNumber(this.sequenceId);
w = mvcc.beginMemstoreInsertWithSeqNum(mvccNum); w = mvcc.beginMemstoreInsertWithSeqNum(mvccNum);
long now = EnvironmentEdgeManager.currentTime(); long now = EnvironmentEdgeManager.currentTime();
// Process each family // Process each family
@ -7106,7 +7106,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
} }
// now start my own transaction // now start my own transaction
mvccNum = MultiVersionConsistencyControl.getPreAssignedWriteNumber(this.sequenceId); mvccNum = MultiVersionConcurrencyControl.getPreAssignedWriteNumber(this.sequenceId);
w = mvcc.beginMemstoreInsertWithSeqNum(mvccNum); w = mvcc.beginMemstoreInsertWithSeqNum(mvccNum);
long now = EnvironmentEdgeManager.currentTime(); long now = EnvironmentEdgeManager.currentTime();
// Process each family // Process each family
@ -7332,7 +7332,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
WriteState.HEAP_SIZE + // writestate WriteState.HEAP_SIZE + // writestate
ClassSize.CONCURRENT_SKIPLISTMAP + ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY + // stores ClassSize.CONCURRENT_SKIPLISTMAP + ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY + // stores
(2 * ClassSize.REENTRANT_LOCK) + // lock, updatesLock (2 * ClassSize.REENTRANT_LOCK) + // lock, updatesLock
MultiVersionConsistencyControl.FIXED_SIZE // mvcc MultiVersionConcurrencyControl.FIXED_SIZE // mvcc
+ ClassSize.TREEMAP // maxSeqIdInStores + ClassSize.TREEMAP // maxSeqIdInStores
+ 2 * ClassSize.ATOMIC_INTEGER // majorInProgress, minorInProgress + 2 * ClassSize.ATOMIC_INTEGER // majorInProgress, minorInProgress
; ;

View File

@ -33,7 +33,7 @@ import org.apache.hadoop.hbase.util.ClassSize;
* the new writes for readers to read (thus forming atomic transactions). * the new writes for readers to read (thus forming atomic transactions).
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
public class MultiVersionConsistencyControl { public class MultiVersionConcurrencyControl {
private static final long NO_WRITE_NUMBER = 0; private static final long NO_WRITE_NUMBER = 0;
private volatile long memstoreRead = 0; private volatile long memstoreRead = 0;
private final Object readWaiters = new Object(); private final Object readWaiters = new Object();
@ -45,7 +45,7 @@ public class MultiVersionConsistencyControl {
/** /**
* Default constructor. Initializes the memstoreRead/Write points to 0. * Default constructor. Initializes the memstoreRead/Write points to 0.
*/ */
public MultiVersionConsistencyControl() { public MultiVersionConcurrencyControl() {
} }
/** /**

View File

@ -63,7 +63,7 @@ public interface RegionScanner extends InternalScanner, Shipper {
long getMaxResultSize(); long getMaxResultSize();
/** /**
* @return The Scanner's MVCC readPt see {@link MultiVersionConsistencyControl} * @return The Scanner's MVCC readPt see {@link MultiVersionConcurrencyControl}
*/ */
long getMvccReadPoint(); long getMvccReadPoint();
@ -94,7 +94,7 @@ public interface RegionScanner extends InternalScanner, Shipper {
* close a region operation, an synchronize on the scanner object. Example: <code> * close a region operation, an synchronize on the scanner object. Example: <code>
* HRegion region = ...; * HRegion region = ...;
* RegionScanner scanner = ... * RegionScanner scanner = ...
* MultiVersionConsistencyControl.setThreadReadPoint(scanner.getMvccReadPoint()); * MultiVersionConcurrencyControl.setThreadReadPoint(scanner.getMvccReadPoint());
* region.startRegionOperation(); * region.startRegionOperation();
* try { * try {
* synchronized(scanner) { * synchronized(scanner) {

View File

@ -3538,7 +3538,7 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
Scan scan = new Scan(get); Scan scan = new Scan(get);
InternalScanner scanner = (InternalScanner) store.getScanner(scan, InternalScanner scanner = (InternalScanner) store.getScanner(scan,
scan.getFamilyMap().get(store.getFamily().getName()), scan.getFamilyMap().get(store.getFamily().getName()),
// originally MultiVersionConsistencyControl.resetThreadReadPoint() was called to set // originally MultiVersionConcurrencyControl.resetThreadReadPoint() was called to set
// readpoint 0. // readpoint 0.
0); 0);

View File

@ -69,13 +69,13 @@ public class TestDefaultMemStore extends TestCase {
private static final int ROW_COUNT = 10; private static final int ROW_COUNT = 10;
private static final int QUALIFIER_COUNT = ROW_COUNT; private static final int QUALIFIER_COUNT = ROW_COUNT;
private static final byte [] FAMILY = Bytes.toBytes("column"); private static final byte [] FAMILY = Bytes.toBytes("column");
private MultiVersionConsistencyControl mvcc; private MultiVersionConcurrencyControl mvcc;
private AtomicLong startSeqNum = new AtomicLong(0); private AtomicLong startSeqNum = new AtomicLong(0);
@Override @Override
public void setUp() throws Exception { public void setUp() throws Exception {
super.setUp(); super.setUp();
this.mvcc = new MultiVersionConsistencyControl(); this.mvcc = new MultiVersionConcurrencyControl();
this.memstore = new DefaultMemStore(); this.memstore = new DefaultMemStore();
} }
@ -248,7 +248,7 @@ public class TestDefaultMemStore extends TestCase {
final byte[] q2 = Bytes.toBytes("q2"); final byte[] q2 = Bytes.toBytes("q2");
final byte[] v = Bytes.toBytes("value"); final byte[] v = Bytes.toBytes("value");
MultiVersionConsistencyControl.WriteEntry w = MultiVersionConcurrencyControl.WriteEntry w =
mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet()); mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet());
KeyValue kv1 = new KeyValue(row, f, q1, v); KeyValue kv1 = new KeyValue(row, f, q1, v);
@ -292,7 +292,7 @@ public class TestDefaultMemStore extends TestCase {
final byte[] v2 = Bytes.toBytes("value2"); final byte[] v2 = Bytes.toBytes("value2");
// INSERT 1: Write both columns val1 // INSERT 1: Write both columns val1
MultiVersionConsistencyControl.WriteEntry w = MultiVersionConcurrencyControl.WriteEntry w =
mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet()); mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet());
KeyValue kv11 = new KeyValue(row, f, q1, v1); KeyValue kv11 = new KeyValue(row, f, q1, v1);
@ -344,7 +344,7 @@ public class TestDefaultMemStore extends TestCase {
final byte[] q2 = Bytes.toBytes("q2"); final byte[] q2 = Bytes.toBytes("q2");
final byte[] v1 = Bytes.toBytes("value1"); final byte[] v1 = Bytes.toBytes("value1");
// INSERT 1: Write both columns val1 // INSERT 1: Write both columns val1
MultiVersionConsistencyControl.WriteEntry w = MultiVersionConcurrencyControl.WriteEntry w =
mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet()); mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet());
KeyValue kv11 = new KeyValue(row, f, q1, v1); KeyValue kv11 = new KeyValue(row, f, q1, v1);
@ -388,7 +388,7 @@ public class TestDefaultMemStore extends TestCase {
final byte[] f = Bytes.toBytes("family"); final byte[] f = Bytes.toBytes("family");
final byte[] q1 = Bytes.toBytes("q1"); final byte[] q1 = Bytes.toBytes("q1");
final MultiVersionConsistencyControl mvcc; final MultiVersionConcurrencyControl mvcc;
final MemStore memstore; final MemStore memstore;
final AtomicLong startSeqNum; final AtomicLong startSeqNum;
@ -397,7 +397,7 @@ public class TestDefaultMemStore extends TestCase {
public ReadOwnWritesTester(int id, public ReadOwnWritesTester(int id,
MemStore memstore, MemStore memstore,
MultiVersionConsistencyControl mvcc, MultiVersionConcurrencyControl mvcc,
AtomicReference<Throwable> caughtException, AtomicReference<Throwable> caughtException,
AtomicLong startSeqNum) AtomicLong startSeqNum)
{ {
@ -418,7 +418,7 @@ public class TestDefaultMemStore extends TestCase {
private void internalRun() throws IOException { private void internalRun() throws IOException {
for (long i = 0; i < NUM_TRIES && caughtException.get() == null; i++) { for (long i = 0; i < NUM_TRIES && caughtException.get() == null; i++) {
MultiVersionConsistencyControl.WriteEntry w = MultiVersionConcurrencyControl.WriteEntry w =
mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet()); mvcc.beginMemstoreInsertWithSeqNum(this.startSeqNum.incrementAndGet());
// Insert the sequence value (i) // Insert the sequence value (i)

View File

@ -27,17 +27,17 @@ import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicLong; import java.util.concurrent.atomic.AtomicLong;
/** /**
* This is a hammer test that verifies MultiVersionConsistencyControl in a * This is a hammer test that verifies MultiVersionConcurrencyControl in a
* multiple writer single reader scenario. * multiple writer single reader scenario.
*/ */
@Category({RegionServerTests.class, SmallTests.class}) @Category({RegionServerTests.class, SmallTests.class})
public class TestMultiVersionConsistencyControl extends TestCase { public class TestMultiVersionConcurrencyControl extends TestCase {
static class Writer implements Runnable { static class Writer implements Runnable {
final AtomicBoolean finished; final AtomicBoolean finished;
final MultiVersionConsistencyControl mvcc; final MultiVersionConcurrencyControl mvcc;
final AtomicBoolean status; final AtomicBoolean status;
Writer(AtomicBoolean finished, MultiVersionConsistencyControl mvcc, AtomicBoolean status) { Writer(AtomicBoolean finished, MultiVersionConcurrencyControl mvcc, AtomicBoolean status) {
this.finished = finished; this.finished = finished;
this.mvcc = mvcc; this.mvcc = mvcc;
this.status = status; this.status = status;
@ -49,7 +49,7 @@ public class TestMultiVersionConsistencyControl extends TestCase {
public void run() { public void run() {
AtomicLong startPoint = new AtomicLong(); AtomicLong startPoint = new AtomicLong();
while (!finished.get()) { while (!finished.get()) {
MultiVersionConsistencyControl.WriteEntry e = MultiVersionConcurrencyControl.WriteEntry e =
mvcc.beginMemstoreInsertWithSeqNum(startPoint.incrementAndGet()); mvcc.beginMemstoreInsertWithSeqNum(startPoint.incrementAndGet());
// System.out.println("Begin write: " + e.getWriteNumber()); // System.out.println("Begin write: " + e.getWriteNumber());
// 10 usec - 500usec (including 0) // 10 usec - 500usec (including 0)
@ -75,7 +75,7 @@ public class TestMultiVersionConsistencyControl extends TestCase {
} }
public void testParallelism() throws Exception { public void testParallelism() throws Exception {
final MultiVersionConsistencyControl mvcc = new MultiVersionConsistencyControl(); final MultiVersionConcurrencyControl mvcc = new MultiVersionConcurrencyControl();
final AtomicBoolean finished = new AtomicBoolean(false); final AtomicBoolean finished = new AtomicBoolean(false);

View File

@ -1495,7 +1495,7 @@ The minimum flush unit is per region, not at individual MemStore level.
* The `RegionScanner` object contains a list of `StoreScanner` objects, one per column family. * The `RegionScanner` object contains a list of `StoreScanner` objects, one per column family.
* Each `StoreScanner` object further contains a list of `StoreFileScanner` objects, corresponding to each StoreFile and HFile of the corresponding column family, and a list of `KeyValueScanner` objects for the MemStore. * Each `StoreScanner` object further contains a list of `StoreFileScanner` objects, corresponding to each StoreFile and HFile of the corresponding column family, and a list of `KeyValueScanner` objects for the MemStore.
* The two lists are merged into one, which is sorted in ascending order with the scan object for the MemStore at the end of the list. * The two lists are merged into one, which is sorted in ascending order with the scan object for the MemStore at the end of the list.
* When a `StoreFileScanner` object is constructed, it is associated with a `MultiVersionConsistencyControl` read point, which is the current `memstoreTS`, filtering out any new updates beyond the read point. * When a `StoreFileScanner` object is constructed, it is associated with a `MultiVersionConcurrencyControl` read point, which is the current `memstoreTS`, filtering out any new updates beyond the read point.
[[hfile]] [[hfile]]
==== StoreFile (HFile) ==== StoreFile (HFile)