HBASE-12050 Avoid KeyValueUtil#ensureKeyValue from DefaultMemStore.
This commit is contained in:
parent
3f98c02cd3
commit
a2e05b9f8f
|
@ -134,6 +134,8 @@ public class KeyValueUtil {
|
|||
/**************** copy key and value *********************/
|
||||
|
||||
public static int appendToByteArray(final Cell cell, final byte[] output, final int offset) {
|
||||
// TODO when cell instance of KV we can bypass all steps and just do backing single array
|
||||
// copy(?)
|
||||
int pos = offset;
|
||||
pos = Bytes.putInt(output, pos, keyLength(cell));
|
||||
pos = Bytes.putInt(output, pos, cell.getValueLength());
|
||||
|
|
|
@ -107,8 +107,8 @@ public class ClassSize {
|
|||
/** Overhead for TimeRangeTracker */
|
||||
public static final int TIMERANGE_TRACKER;
|
||||
|
||||
/** Overhead for KeyValueSkipListSet */
|
||||
public static final int KEYVALUE_SKIPLIST_SET;
|
||||
/** Overhead for CellSkipListSet */
|
||||
public static final int CELL_SKIPLIST_SET;
|
||||
|
||||
/* Are we running on jdk7? */
|
||||
private static final boolean JDK7;
|
||||
|
@ -192,7 +192,7 @@ public class ClassSize {
|
|||
|
||||
TIMERANGE_TRACKER = align(ClassSize.OBJECT + Bytes.SIZEOF_LONG * 2);
|
||||
|
||||
KEYVALUE_SKIPLIST_SET = align(OBJECT + REFERENCE);
|
||||
CELL_SKIPLIST_SET = align(OBJECT + REFERENCE);
|
||||
}
|
||||
|
||||
/**
|
||||
|
|
|
@ -18,9 +18,6 @@
|
|||
*/
|
||||
package org.apache.hadoop.hbase.regionserver;
|
||||
|
||||
import org.apache.hadoop.hbase.classification.InterfaceAudience;
|
||||
import org.apache.hadoop.hbase.KeyValue;
|
||||
|
||||
import java.util.Collection;
|
||||
import java.util.Comparator;
|
||||
import java.util.Iterator;
|
||||
|
@ -29,8 +26,12 @@ import java.util.SortedSet;
|
|||
import java.util.concurrent.ConcurrentNavigableMap;
|
||||
import java.util.concurrent.ConcurrentSkipListMap;
|
||||
|
||||
import org.apache.hadoop.hbase.Cell;
|
||||
import org.apache.hadoop.hbase.KeyValue;
|
||||
import org.apache.hadoop.hbase.classification.InterfaceAudience;
|
||||
|
||||
/**
|
||||
* A {@link java.util.Set} of {@link KeyValue}s implemented on top of a
|
||||
* A {@link java.util.Set} of {@link Cell}s implemented on top of a
|
||||
* {@link java.util.concurrent.ConcurrentSkipListMap}. Works like a
|
||||
* {@link java.util.concurrent.ConcurrentSkipListSet} in all but one regard:
|
||||
* An add will overwrite if already an entry for the added key. In other words,
|
||||
|
@ -44,96 +45,96 @@ import java.util.concurrent.ConcurrentSkipListMap;
|
|||
* get and set and won't throw ConcurrentModificationException when iterating.
|
||||
*/
|
||||
@InterfaceAudience.Private
|
||||
public class KeyValueSkipListSet implements NavigableSet<KeyValue> {
|
||||
private final ConcurrentNavigableMap<KeyValue, KeyValue> delegatee;
|
||||
public class CellSkipListSet implements NavigableSet<Cell> {
|
||||
private final ConcurrentNavigableMap<Cell, Cell> delegatee;
|
||||
|
||||
KeyValueSkipListSet(final KeyValue.KVComparator c) {
|
||||
this.delegatee = new ConcurrentSkipListMap<KeyValue, KeyValue>(c);
|
||||
CellSkipListSet(final KeyValue.KVComparator c) {
|
||||
this.delegatee = new ConcurrentSkipListMap<Cell, Cell>(c);
|
||||
}
|
||||
|
||||
KeyValueSkipListSet(final ConcurrentNavigableMap<KeyValue, KeyValue> m) {
|
||||
CellSkipListSet(final ConcurrentNavigableMap<Cell, Cell> m) {
|
||||
this.delegatee = m;
|
||||
}
|
||||
|
||||
public KeyValue ceiling(KeyValue e) {
|
||||
public Cell ceiling(Cell e) {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public Iterator<KeyValue> descendingIterator() {
|
||||
public Iterator<Cell> descendingIterator() {
|
||||
return this.delegatee.descendingMap().values().iterator();
|
||||
}
|
||||
|
||||
public NavigableSet<KeyValue> descendingSet() {
|
||||
public NavigableSet<Cell> descendingSet() {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public KeyValue floor(KeyValue e) {
|
||||
public Cell floor(Cell e) {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public SortedSet<KeyValue> headSet(final KeyValue toElement) {
|
||||
public SortedSet<Cell> headSet(final Cell toElement) {
|
||||
return headSet(toElement, false);
|
||||
}
|
||||
|
||||
public NavigableSet<KeyValue> headSet(final KeyValue toElement,
|
||||
public NavigableSet<Cell> headSet(final Cell toElement,
|
||||
boolean inclusive) {
|
||||
return new KeyValueSkipListSet(this.delegatee.headMap(toElement, inclusive));
|
||||
return new CellSkipListSet(this.delegatee.headMap(toElement, inclusive));
|
||||
}
|
||||
|
||||
public KeyValue higher(KeyValue e) {
|
||||
public Cell higher(Cell e) {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public Iterator<KeyValue> iterator() {
|
||||
public Iterator<Cell> iterator() {
|
||||
return this.delegatee.values().iterator();
|
||||
}
|
||||
|
||||
public KeyValue lower(KeyValue e) {
|
||||
public Cell lower(Cell e) {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public KeyValue pollFirst() {
|
||||
public Cell pollFirst() {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public KeyValue pollLast() {
|
||||
public Cell pollLast() {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public SortedSet<KeyValue> subSet(KeyValue fromElement, KeyValue toElement) {
|
||||
public SortedSet<Cell> subSet(Cell fromElement, Cell toElement) {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public NavigableSet<KeyValue> subSet(KeyValue fromElement,
|
||||
boolean fromInclusive, KeyValue toElement, boolean toInclusive) {
|
||||
public NavigableSet<Cell> subSet(Cell fromElement,
|
||||
boolean fromInclusive, Cell toElement, boolean toInclusive) {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public SortedSet<KeyValue> tailSet(KeyValue fromElement) {
|
||||
public SortedSet<Cell> tailSet(Cell fromElement) {
|
||||
return tailSet(fromElement, true);
|
||||
}
|
||||
|
||||
public NavigableSet<KeyValue> tailSet(KeyValue fromElement, boolean inclusive) {
|
||||
return new KeyValueSkipListSet(this.delegatee.tailMap(fromElement, inclusive));
|
||||
public NavigableSet<Cell> tailSet(Cell fromElement, boolean inclusive) {
|
||||
return new CellSkipListSet(this.delegatee.tailMap(fromElement, inclusive));
|
||||
}
|
||||
|
||||
public Comparator<? super KeyValue> comparator() {
|
||||
public Comparator<? super Cell> comparator() {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public KeyValue first() {
|
||||
public Cell first() {
|
||||
return this.delegatee.get(this.delegatee.firstKey());
|
||||
}
|
||||
|
||||
public KeyValue last() {
|
||||
public Cell last() {
|
||||
return this.delegatee.get(this.delegatee.lastKey());
|
||||
}
|
||||
|
||||
public boolean add(KeyValue e) {
|
||||
public boolean add(Cell e) {
|
||||
return this.delegatee.put(e, e) == null;
|
||||
}
|
||||
|
||||
public boolean addAll(Collection<? extends KeyValue> c) {
|
||||
public boolean addAll(Collection<? extends Cell> c) {
|
||||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
|
@ -166,7 +167,7 @@ public class KeyValueSkipListSet implements NavigableSet<KeyValue> {
|
|||
throw new UnsupportedOperationException("Not implemented");
|
||||
}
|
||||
|
||||
public KeyValue get(KeyValue kv) {
|
||||
public Cell get(Cell kv) {
|
||||
return this.delegatee.get(kv);
|
||||
}
|
||||
|
|
@ -73,15 +73,15 @@ public class DefaultMemStore implements MemStore {
|
|||
|
||||
private Configuration conf;
|
||||
|
||||
// MemStore. Use a KeyValueSkipListSet rather than SkipListSet because of the
|
||||
// MemStore. Use a CellSkipListSet rather than SkipListSet because of the
|
||||
// better semantics. The Map will overwrite if passed a key it already had
|
||||
// whereas the Set will not add new KV if key is same though value might be
|
||||
// whereas the Set will not add new Cell if key is same though value might be
|
||||
// different. Value is not important -- just make sure always same
|
||||
// reference passed.
|
||||
volatile KeyValueSkipListSet kvset;
|
||||
volatile CellSkipListSet cellSet;
|
||||
|
||||
// Snapshot of memstore. Made for flusher.
|
||||
volatile KeyValueSkipListSet snapshot;
|
||||
volatile CellSkipListSet snapshot;
|
||||
|
||||
final KeyValue.KVComparator comparator;
|
||||
|
||||
|
@ -114,8 +114,8 @@ public class DefaultMemStore implements MemStore {
|
|||
final KeyValue.KVComparator c) {
|
||||
this.conf = conf;
|
||||
this.comparator = c;
|
||||
this.kvset = new KeyValueSkipListSet(c);
|
||||
this.snapshot = new KeyValueSkipListSet(c);
|
||||
this.cellSet = new CellSkipListSet(c);
|
||||
this.snapshot = new CellSkipListSet(c);
|
||||
timeRangeTracker = new TimeRangeTracker();
|
||||
snapshotTimeRangeTracker = new TimeRangeTracker();
|
||||
this.size = new AtomicLong(DEEP_OVERHEAD);
|
||||
|
@ -130,11 +130,11 @@ public class DefaultMemStore implements MemStore {
|
|||
}
|
||||
|
||||
void dump() {
|
||||
for (KeyValue kv: this.kvset) {
|
||||
LOG.info(kv);
|
||||
for (Cell cell: this.cellSet) {
|
||||
LOG.info(cell);
|
||||
}
|
||||
for (KeyValue kv: this.snapshot) {
|
||||
LOG.info(kv);
|
||||
for (Cell cell: this.snapshot) {
|
||||
LOG.info(cell);
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -152,9 +152,9 @@ public class DefaultMemStore implements MemStore {
|
|||
} else {
|
||||
this.snapshotId = EnvironmentEdgeManager.currentTime();
|
||||
this.snapshotSize = keySize();
|
||||
if (!this.kvset.isEmpty()) {
|
||||
this.snapshot = this.kvset;
|
||||
this.kvset = new KeyValueSkipListSet(this.comparator);
|
||||
if (!this.cellSet.isEmpty()) {
|
||||
this.snapshot = this.cellSet;
|
||||
this.cellSet = new CellSkipListSet(this.comparator);
|
||||
this.snapshotTimeRangeTracker = this.timeRangeTracker;
|
||||
this.timeRangeTracker = new TimeRangeTracker();
|
||||
// Reset heap to not include any keys
|
||||
|
@ -189,7 +189,7 @@ public class DefaultMemStore implements MemStore {
|
|||
// OK. Passed in snapshot is same as current snapshot. If not-empty,
|
||||
// create a new snapshot and let the old one go.
|
||||
if (!this.snapshot.isEmpty()) {
|
||||
this.snapshot = new KeyValueSkipListSet(this.comparator);
|
||||
this.snapshot = new CellSkipListSet(this.comparator);
|
||||
this.snapshotTimeRangeTracker = new TimeRangeTracker();
|
||||
}
|
||||
this.snapshotSize = 0;
|
||||
|
@ -216,7 +216,7 @@ public class DefaultMemStore implements MemStore {
|
|||
*/
|
||||
@Override
|
||||
public Pair<Long, Cell> add(Cell cell) {
|
||||
KeyValue toAdd = maybeCloneWithAllocator(KeyValueUtil.ensureKeyValue(cell));
|
||||
Cell toAdd = maybeCloneWithAllocator(cell);
|
||||
return new Pair<Long, Cell>(internalAdd(toAdd), toAdd);
|
||||
}
|
||||
|
||||
|
@ -225,14 +225,14 @@ public class DefaultMemStore implements MemStore {
|
|||
return timeOfOldestEdit;
|
||||
}
|
||||
|
||||
private boolean addToKVSet(KeyValue e) {
|
||||
boolean b = this.kvset.add(e);
|
||||
private boolean addToCellSet(Cell e) {
|
||||
boolean b = this.cellSet.add(e);
|
||||
setOldestEditTimeToNow();
|
||||
return b;
|
||||
}
|
||||
|
||||
private boolean removeFromKVSet(KeyValue e) {
|
||||
boolean b = this.kvset.remove(e);
|
||||
private boolean removeFromCellSet(Cell e) {
|
||||
boolean b = this.cellSet.remove(e);
|
||||
setOldestEditTimeToNow();
|
||||
return b;
|
||||
}
|
||||
|
@ -244,39 +244,39 @@ public class DefaultMemStore implements MemStore {
|
|||
}
|
||||
|
||||
/**
|
||||
* Internal version of add() that doesn't clone KVs with the
|
||||
* Internal version of add() that doesn't clone Cells with the
|
||||
* allocator, and doesn't take the lock.
|
||||
*
|
||||
* Callers should ensure they already have the read lock taken
|
||||
*/
|
||||
private long internalAdd(final KeyValue toAdd) {
|
||||
long s = heapSizeChange(toAdd, addToKVSet(toAdd));
|
||||
private long internalAdd(final Cell toAdd) {
|
||||
long s = heapSizeChange(toAdd, addToCellSet(toAdd));
|
||||
timeRangeTracker.includeTimestamp(toAdd);
|
||||
this.size.addAndGet(s);
|
||||
return s;
|
||||
}
|
||||
|
||||
private KeyValue maybeCloneWithAllocator(KeyValue kv) {
|
||||
private Cell maybeCloneWithAllocator(Cell cell) {
|
||||
if (allocator == null) {
|
||||
return kv;
|
||||
return cell;
|
||||
}
|
||||
|
||||
int len = kv.getLength();
|
||||
int len = KeyValueUtil.length(cell);
|
||||
ByteRange alloc = allocator.allocateBytes(len);
|
||||
if (alloc == null) {
|
||||
// The allocation was too large, allocator decided
|
||||
// not to do anything with it.
|
||||
return kv;
|
||||
return cell;
|
||||
}
|
||||
assert alloc.getBytes() != null;
|
||||
alloc.put(0, kv.getBuffer(), kv.getOffset(), len);
|
||||
KeyValueUtil.appendToByteArray(cell, alloc.getBytes(), alloc.getOffset());
|
||||
KeyValue newKv = new KeyValue(alloc.getBytes(), alloc.getOffset(), len);
|
||||
newKv.setSequenceId(kv.getMvccVersion());
|
||||
newKv.setSequenceId(cell.getSequenceId());
|
||||
return newKv;
|
||||
}
|
||||
|
||||
/**
|
||||
* Remove n key from the memstore. Only kvs that have the same key and the
|
||||
* Remove n key from the memstore. Only cells that have the same key and the
|
||||
* same memstoreTS are removed. It is ok to not update timeRangeTracker
|
||||
* in this call. It is possible that we can optimize this method by using
|
||||
* tailMap/iterator, but since this method is called rarely (only for
|
||||
|
@ -290,18 +290,17 @@ public class DefaultMemStore implements MemStore {
|
|||
// not the snapshot. The flush of this snapshot to disk has not
|
||||
// yet started because Store.flush() waits for all rwcc transactions to
|
||||
// commit before starting the flush to disk.
|
||||
KeyValue kv = KeyValueUtil.ensureKeyValue(cell);
|
||||
KeyValue found = this.snapshot.get(kv);
|
||||
if (found != null && found.getMvccVersion() == kv.getMvccVersion()) {
|
||||
this.snapshot.remove(kv);
|
||||
long sz = heapSizeChange(kv, true);
|
||||
Cell found = this.snapshot.get(cell);
|
||||
if (found != null && found.getSequenceId() == cell.getSequenceId()) {
|
||||
this.snapshot.remove(cell);
|
||||
long sz = heapSizeChange(cell, true);
|
||||
this.snapshotSize -= sz;
|
||||
}
|
||||
// If the key is in the memstore, delete it. Update this.size.
|
||||
found = this.kvset.get(kv);
|
||||
if (found != null && found.getMvccVersion() == kv.getMvccVersion()) {
|
||||
removeFromKVSet(kv);
|
||||
long s = heapSizeChange(kv, true);
|
||||
found = this.cellSet.get(cell);
|
||||
if (found != null && found.getSequenceId() == cell.getSequenceId()) {
|
||||
removeFromCellSet(cell);
|
||||
long s = heapSizeChange(cell, true);
|
||||
this.size.addAndGet(-s);
|
||||
}
|
||||
}
|
||||
|
@ -314,20 +313,20 @@ public class DefaultMemStore implements MemStore {
|
|||
@Override
|
||||
public long delete(Cell deleteCell) {
|
||||
long s = 0;
|
||||
KeyValue toAdd = maybeCloneWithAllocator(KeyValueUtil.ensureKeyValue(deleteCell));
|
||||
s += heapSizeChange(toAdd, addToKVSet(toAdd));
|
||||
Cell toAdd = maybeCloneWithAllocator(deleteCell);
|
||||
s += heapSizeChange(toAdd, addToCellSet(toAdd));
|
||||
timeRangeTracker.includeTimestamp(toAdd);
|
||||
this.size.addAndGet(s);
|
||||
return s;
|
||||
}
|
||||
|
||||
/**
|
||||
* @param kv Find the row that comes after this one. If null, we return the
|
||||
* @param cell Find the row that comes after this one. If null, we return the
|
||||
* first.
|
||||
* @return Next row or null if none found.
|
||||
*/
|
||||
KeyValue getNextRow(final KeyValue kv) {
|
||||
return getLowest(getNextRow(kv, this.kvset), getNextRow(kv, this.snapshot));
|
||||
Cell getNextRow(final Cell cell) {
|
||||
return getLowest(getNextRow(cell, this.cellSet), getNextRow(cell, this.snapshot));
|
||||
}
|
||||
|
||||
/*
|
||||
|
@ -335,7 +334,7 @@ public class DefaultMemStore implements MemStore {
|
|||
* @param b
|
||||
* @return Return lowest of a or b or null if both a and b are null
|
||||
*/
|
||||
private KeyValue getLowest(final KeyValue a, final KeyValue b) {
|
||||
private Cell getLowest(final Cell a, final Cell b) {
|
||||
if (a == null) {
|
||||
return b;
|
||||
}
|
||||
|
@ -351,17 +350,17 @@ public class DefaultMemStore implements MemStore {
|
|||
* @return Next row or null if none found. If one found, will be a new
|
||||
* KeyValue -- can be destroyed by subsequent calls to this method.
|
||||
*/
|
||||
private KeyValue getNextRow(final KeyValue key,
|
||||
final NavigableSet<KeyValue> set) {
|
||||
KeyValue result = null;
|
||||
SortedSet<KeyValue> tail = key == null? set: set.tailSet(key);
|
||||
private Cell getNextRow(final Cell key,
|
||||
final NavigableSet<Cell> set) {
|
||||
Cell result = null;
|
||||
SortedSet<Cell> tail = key == null? set: set.tailSet(key);
|
||||
// Iterate until we fall into the next row; i.e. move off current row
|
||||
for (KeyValue kv: tail) {
|
||||
if (comparator.compareRows(kv, key) <= 0)
|
||||
for (Cell cell: tail) {
|
||||
if (comparator.compareRows(cell, key) <= 0)
|
||||
continue;
|
||||
// Note: Not suppressing deletes or expired cells. Needs to be handled
|
||||
// by higher up functions.
|
||||
result = kv;
|
||||
result = cell;
|
||||
break;
|
||||
}
|
||||
return result;
|
||||
|
@ -372,7 +371,7 @@ public class DefaultMemStore implements MemStore {
|
|||
*/
|
||||
@Override
|
||||
public void getRowKeyAtOrBefore(final GetClosestRowBeforeTracker state) {
|
||||
getRowKeyAtOrBefore(kvset, state);
|
||||
getRowKeyAtOrBefore(cellSet, state);
|
||||
getRowKeyAtOrBefore(snapshot, state);
|
||||
}
|
||||
|
||||
|
@ -380,7 +379,7 @@ public class DefaultMemStore implements MemStore {
|
|||
* @param set
|
||||
* @param state Accumulates deletes and candidates.
|
||||
*/
|
||||
private void getRowKeyAtOrBefore(final NavigableSet<KeyValue> set,
|
||||
private void getRowKeyAtOrBefore(final NavigableSet<Cell> set,
|
||||
final GetClosestRowBeforeTracker state) {
|
||||
if (set.isEmpty()) {
|
||||
return;
|
||||
|
@ -401,13 +400,13 @@ public class DefaultMemStore implements MemStore {
|
|||
* @param state
|
||||
* @return True if we found a candidate walking this row.
|
||||
*/
|
||||
private boolean walkForwardInSingleRow(final SortedSet<KeyValue> set,
|
||||
final KeyValue firstOnRow, final GetClosestRowBeforeTracker state) {
|
||||
private boolean walkForwardInSingleRow(final SortedSet<Cell> set,
|
||||
final Cell firstOnRow, final GetClosestRowBeforeTracker state) {
|
||||
boolean foundCandidate = false;
|
||||
SortedSet<KeyValue> tail = set.tailSet(firstOnRow);
|
||||
SortedSet<Cell> tail = set.tailSet(firstOnRow);
|
||||
if (tail.isEmpty()) return foundCandidate;
|
||||
for (Iterator<KeyValue> i = tail.iterator(); i.hasNext();) {
|
||||
KeyValue kv = i.next();
|
||||
for (Iterator<Cell> i = tail.iterator(); i.hasNext();) {
|
||||
Cell kv = i.next();
|
||||
// Did we go beyond the target row? If so break.
|
||||
if (state.isTooFar(kv, firstOnRow)) break;
|
||||
if (state.isExpired(kv)) {
|
||||
|
@ -429,17 +428,17 @@ public class DefaultMemStore implements MemStore {
|
|||
* @param set
|
||||
* @param state
|
||||
*/
|
||||
private void getRowKeyBefore(NavigableSet<KeyValue> set,
|
||||
private void getRowKeyBefore(NavigableSet<Cell> set,
|
||||
final GetClosestRowBeforeTracker state) {
|
||||
KeyValue firstOnRow = state.getTargetKey();
|
||||
Cell firstOnRow = state.getTargetKey();
|
||||
for (Member p = memberOfPreviousRow(set, state, firstOnRow);
|
||||
p != null; p = memberOfPreviousRow(p.set, state, firstOnRow)) {
|
||||
// Make sure we don't fall out of our table.
|
||||
if (!state.isTargetTable(p.kv)) break;
|
||||
if (!state.isTargetTable(p.cell)) break;
|
||||
// Stop looking if we've exited the better candidate range.
|
||||
if (!state.isBetterCandidate(p.kv)) break;
|
||||
if (!state.isBetterCandidate(p.cell)) break;
|
||||
// Make into firstOnRow
|
||||
firstOnRow = new KeyValue(p.kv.getRowArray(), p.kv.getRowOffset(), p.kv.getRowLength(),
|
||||
firstOnRow = new KeyValue(p.cell.getRowArray(), p.cell.getRowOffset(), p.cell.getRowLength(),
|
||||
HConstants.LATEST_TIMESTAMP);
|
||||
// If we find something, break;
|
||||
if (walkForwardInSingleRow(p.set, firstOnRow, state)) break;
|
||||
|
@ -468,15 +467,14 @@ public class DefaultMemStore implements MemStore {
|
|||
byte[] qualifier,
|
||||
long newValue,
|
||||
long now) {
|
||||
KeyValue firstKv = KeyValueUtil.createFirstOnRow(
|
||||
row, family, qualifier);
|
||||
// Is there a KeyValue in 'snapshot' with the same TS? If so, upgrade the timestamp a bit.
|
||||
SortedSet<KeyValue> snSs = snapshot.tailSet(firstKv);
|
||||
Cell firstCell = KeyValueUtil.createFirstOnRow(row, family, qualifier);
|
||||
// Is there a Cell in 'snapshot' with the same TS? If so, upgrade the timestamp a bit.
|
||||
SortedSet<Cell> snSs = snapshot.tailSet(firstCell);
|
||||
if (!snSs.isEmpty()) {
|
||||
KeyValue snKv = snSs.first();
|
||||
// is there a matching KV in the snapshot?
|
||||
if (CellUtil.matchingRow(snKv, firstKv) && CellUtil.matchingQualifier(snKv, firstKv)) {
|
||||
if (snKv.getTimestamp() == now) {
|
||||
Cell snc = snSs.first();
|
||||
// is there a matching Cell in the snapshot?
|
||||
if (CellUtil.matchingRow(snc, firstCell) && CellUtil.matchingQualifier(snc, firstCell)) {
|
||||
if (snc.getTimestamp() == now) {
|
||||
// poop,
|
||||
now += 1;
|
||||
}
|
||||
|
@ -486,24 +484,25 @@ public class DefaultMemStore implements MemStore {
|
|||
// logic here: the new ts MUST be at least 'now'. But it could be larger if necessary.
|
||||
// But the timestamp should also be max(now, mostRecentTsInMemstore)
|
||||
|
||||
// so we cant add the new KV w/o knowing what's there already, but we also
|
||||
// want to take this chance to delete some kvs. So two loops (sad)
|
||||
// so we cant add the new Cell w/o knowing what's there already, but we also
|
||||
// want to take this chance to delete some cells. So two loops (sad)
|
||||
|
||||
SortedSet<KeyValue> ss = kvset.tailSet(firstKv);
|
||||
for (KeyValue kv : ss) {
|
||||
SortedSet<Cell> ss = cellSet.tailSet(firstCell);
|
||||
for (Cell cell : ss) {
|
||||
// if this isnt the row we are interested in, then bail:
|
||||
if (!CellUtil.matchingColumn(kv, family, qualifier) || !CellUtil.matchingRow(kv, firstKv)) {
|
||||
if (!CellUtil.matchingColumn(cell, family, qualifier)
|
||||
|| !CellUtil.matchingRow(cell, firstCell)) {
|
||||
break; // rows dont match, bail.
|
||||
}
|
||||
|
||||
// if the qualifier matches and it's a put, just RM it out of the kvset.
|
||||
if (kv.getTypeByte() == KeyValue.Type.Put.getCode() &&
|
||||
kv.getTimestamp() > now && CellUtil.matchingQualifier(firstKv, kv)) {
|
||||
now = kv.getTimestamp();
|
||||
// if the qualifier matches and it's a put, just RM it out of the cellSet.
|
||||
if (cell.getTypeByte() == KeyValue.Type.Put.getCode() &&
|
||||
cell.getTimestamp() > now && CellUtil.matchingQualifier(firstCell, cell)) {
|
||||
now = cell.getTimestamp();
|
||||
}
|
||||
}
|
||||
|
||||
// create or update (upsert) a new KeyValue with
|
||||
// create or update (upsert) a new Cell with
|
||||
// 'now' and a 0 memstoreTS == immediately visible
|
||||
List<Cell> cells = new ArrayList<Cell>(1);
|
||||
cells.add(new KeyValue(row, family, qualifier, now, Bytes.toBytes(newValue)));
|
||||
|
@ -552,37 +551,36 @@ public class DefaultMemStore implements MemStore {
|
|||
* @return change in size of MemStore
|
||||
*/
|
||||
private long upsert(Cell cell, long readpoint) {
|
||||
// Add the KeyValue to the MemStore
|
||||
// Add the Cell to the MemStore
|
||||
// Use the internalAdd method here since we (a) already have a lock
|
||||
// and (b) cannot safely use the MSLAB here without potentially
|
||||
// hitting OOME - see TestMemStore.testUpsertMSLAB for a
|
||||
// test that triggers the pathological case if we don't avoid MSLAB
|
||||
// here.
|
||||
KeyValue kv = KeyValueUtil.ensureKeyValue(cell);
|
||||
long addedSize = internalAdd(kv);
|
||||
long addedSize = internalAdd(cell);
|
||||
|
||||
// Get the KeyValues for the row/family/qualifier regardless of timestamp.
|
||||
// Get the Cells for the row/family/qualifier regardless of timestamp.
|
||||
// For this case we want to clean up any other puts
|
||||
KeyValue firstKv = KeyValueUtil.createFirstOnRow(
|
||||
kv.getRowArray(), kv.getRowOffset(), kv.getRowLength(),
|
||||
kv.getFamilyArray(), kv.getFamilyOffset(), kv.getFamilyLength(),
|
||||
kv.getQualifierArray(), kv.getQualifierOffset(), kv.getQualifierLength());
|
||||
SortedSet<KeyValue> ss = kvset.tailSet(firstKv);
|
||||
Iterator<KeyValue> it = ss.iterator();
|
||||
Cell firstCell = KeyValueUtil.createFirstOnRow(
|
||||
cell.getRowArray(), cell.getRowOffset(), cell.getRowLength(),
|
||||
cell.getFamilyArray(), cell.getFamilyOffset(), cell.getFamilyLength(),
|
||||
cell.getQualifierArray(), cell.getQualifierOffset(), cell.getQualifierLength());
|
||||
SortedSet<Cell> ss = cellSet.tailSet(firstCell);
|
||||
Iterator<Cell> it = ss.iterator();
|
||||
// versions visible to oldest scanner
|
||||
int versionsVisible = 0;
|
||||
while ( it.hasNext() ) {
|
||||
KeyValue cur = it.next();
|
||||
Cell cur = it.next();
|
||||
|
||||
if (kv == cur) {
|
||||
if (cell == cur) {
|
||||
// ignore the one just put in
|
||||
continue;
|
||||
}
|
||||
// check that this is the row and column we are interested in, otherwise bail
|
||||
if (CellUtil.matchingRow(kv, cur) && CellUtil.matchingQualifier(kv, cur)) {
|
||||
if (CellUtil.matchingRow(cell, cur) && CellUtil.matchingQualifier(cell, cur)) {
|
||||
// only remove Puts that concurrent scanners cannot possibly see
|
||||
if (cur.getTypeByte() == KeyValue.Type.Put.getCode() &&
|
||||
cur.getMvccVersion() <= readpoint) {
|
||||
cur.getSequenceId() <= readpoint) {
|
||||
if (versionsVisible > 1) {
|
||||
// if we get here we have seen at least one version visible to the oldest scanner,
|
||||
// which means we can prove that no scanner will see this version
|
||||
|
@ -607,13 +605,13 @@ public class DefaultMemStore implements MemStore {
|
|||
|
||||
/*
|
||||
* Immutable data structure to hold member found in set and the set it was
|
||||
* found in. Include set because it is carrying context.
|
||||
* found in. Include set because it is carrying context.
|
||||
*/
|
||||
private static class Member {
|
||||
final KeyValue kv;
|
||||
final NavigableSet<KeyValue> set;
|
||||
Member(final NavigableSet<KeyValue> s, final KeyValue kv) {
|
||||
this.kv = kv;
|
||||
final Cell cell;
|
||||
final NavigableSet<Cell> set;
|
||||
Member(final NavigableSet<Cell> s, final Cell kv) {
|
||||
this.cell = kv;
|
||||
this.set = s;
|
||||
}
|
||||
}
|
||||
|
@ -626,12 +624,12 @@ public class DefaultMemStore implements MemStore {
|
|||
* member in.
|
||||
* @return Null or member of row previous to <code>firstOnRow</code>
|
||||
*/
|
||||
private Member memberOfPreviousRow(NavigableSet<KeyValue> set,
|
||||
final GetClosestRowBeforeTracker state, final KeyValue firstOnRow) {
|
||||
NavigableSet<KeyValue> head = set.headSet(firstOnRow, false);
|
||||
private Member memberOfPreviousRow(NavigableSet<Cell> set,
|
||||
final GetClosestRowBeforeTracker state, final Cell firstOnRow) {
|
||||
NavigableSet<Cell> head = set.headSet(firstOnRow, false);
|
||||
if (head.isEmpty()) return null;
|
||||
for (Iterator<KeyValue> i = head.descendingIterator(); i.hasNext();) {
|
||||
KeyValue found = i.next();
|
||||
for (Iterator<Cell> i = head.descendingIterator(); i.hasNext();) {
|
||||
Cell found = i.next();
|
||||
if (state.isExpired(found)) {
|
||||
i.remove();
|
||||
continue;
|
||||
|
@ -669,32 +667,32 @@ public class DefaultMemStore implements MemStore {
|
|||
* This behaves as if it were a real scanner but does not maintain position.
|
||||
*/
|
||||
protected class MemStoreScanner extends NonLazyKeyValueScanner {
|
||||
// Next row information for either kvset or snapshot
|
||||
private KeyValue kvsetNextRow = null;
|
||||
private KeyValue snapshotNextRow = null;
|
||||
// Next row information for either cellSet or snapshot
|
||||
private Cell cellSetNextRow = null;
|
||||
private Cell snapshotNextRow = null;
|
||||
|
||||
// last iterated KVs for kvset and snapshot (to restore iterator state after reseek)
|
||||
private KeyValue kvsetItRow = null;
|
||||
private KeyValue snapshotItRow = null;
|
||||
// last iterated Cells for cellSet and snapshot (to restore iterator state after reseek)
|
||||
private Cell cellSetItRow = null;
|
||||
private Cell snapshotItRow = null;
|
||||
|
||||
// iterator based scanning.
|
||||
private Iterator<KeyValue> kvsetIt;
|
||||
private Iterator<KeyValue> snapshotIt;
|
||||
private Iterator<Cell> cellSetIt;
|
||||
private Iterator<Cell> snapshotIt;
|
||||
|
||||
// The kvset and snapshot at the time of creating this scanner
|
||||
private KeyValueSkipListSet kvsetAtCreation;
|
||||
private KeyValueSkipListSet snapshotAtCreation;
|
||||
// The cellSet and snapshot at the time of creating this scanner
|
||||
private CellSkipListSet cellSetAtCreation;
|
||||
private CellSkipListSet snapshotAtCreation;
|
||||
|
||||
// the pre-calculated KeyValue to be returned by peek() or next()
|
||||
private KeyValue theNext;
|
||||
// the pre-calculated Cell to be returned by peek() or next()
|
||||
private Cell theNext;
|
||||
|
||||
// The allocator and snapshot allocator at the time of creating this scanner
|
||||
volatile MemStoreLAB allocatorAtCreation;
|
||||
volatile MemStoreLAB snapshotAllocatorAtCreation;
|
||||
|
||||
// A flag represents whether could stop skipping KeyValues for MVCC
|
||||
// A flag represents whether could stop skipping Cells for MVCC
|
||||
// if have encountered the next row. Only used for reversed scan
|
||||
private boolean stopSkippingKVsIfNextRow = false;
|
||||
private boolean stopSkippingCellsIfNextRow = false;
|
||||
|
||||
private long readPoint;
|
||||
|
||||
|
@ -723,7 +721,7 @@ public class DefaultMemStore implements MemStore {
|
|||
super();
|
||||
|
||||
this.readPoint = readPoint;
|
||||
kvsetAtCreation = kvset;
|
||||
cellSetAtCreation = cellSet;
|
||||
snapshotAtCreation = snapshot;
|
||||
if (allocator != null) {
|
||||
this.allocatorAtCreation = allocator;
|
||||
|
@ -735,17 +733,17 @@ public class DefaultMemStore implements MemStore {
|
|||
}
|
||||
}
|
||||
|
||||
private KeyValue getNext(Iterator<KeyValue> it) {
|
||||
KeyValue startKV = theNext;
|
||||
KeyValue v = null;
|
||||
private Cell getNext(Iterator<Cell> it) {
|
||||
Cell startCell = theNext;
|
||||
Cell v = null;
|
||||
try {
|
||||
while (it.hasNext()) {
|
||||
v = it.next();
|
||||
if (v.getMvccVersion() <= this.readPoint) {
|
||||
if (v.getSequenceId() <= this.readPoint) {
|
||||
return v;
|
||||
}
|
||||
if (stopSkippingKVsIfNextRow && startKV != null
|
||||
&& comparator.compareRows(v, startKV) > 0) {
|
||||
if (stopSkippingCellsIfNextRow && startCell != null
|
||||
&& comparator.compareRows(v, startCell) > 0) {
|
||||
return null;
|
||||
}
|
||||
}
|
||||
|
@ -753,11 +751,11 @@ public class DefaultMemStore implements MemStore {
|
|||
return null;
|
||||
} finally {
|
||||
if (v != null) {
|
||||
// in all cases, remember the last KV iterated to
|
||||
// in all cases, remember the last Cell iterated to
|
||||
if (it == snapshotIt) {
|
||||
snapshotItRow = v;
|
||||
} else {
|
||||
kvsetItRow = v;
|
||||
cellSetItRow = v;
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -776,27 +774,26 @@ public class DefaultMemStore implements MemStore {
|
|||
close();
|
||||
return false;
|
||||
}
|
||||
KeyValue kv = KeyValueUtil.ensureKeyValue(key);
|
||||
// kvset and snapshot will never be null.
|
||||
// if tailSet can't find anything, SortedSet is empty (not null).
|
||||
kvsetIt = kvsetAtCreation.tailSet(kv).iterator();
|
||||
snapshotIt = snapshotAtCreation.tailSet(kv).iterator();
|
||||
kvsetItRow = null;
|
||||
cellSetIt = cellSetAtCreation.tailSet(key).iterator();
|
||||
snapshotIt = snapshotAtCreation.tailSet(key).iterator();
|
||||
cellSetItRow = null;
|
||||
snapshotItRow = null;
|
||||
|
||||
return seekInSubLists(kv);
|
||||
return seekInSubLists(key);
|
||||
}
|
||||
|
||||
|
||||
/**
|
||||
* (Re)initialize the iterators after a seek or a reseek.
|
||||
*/
|
||||
private synchronized boolean seekInSubLists(KeyValue key){
|
||||
kvsetNextRow = getNext(kvsetIt);
|
||||
private synchronized boolean seekInSubLists(Cell key){
|
||||
cellSetNextRow = getNext(cellSetIt);
|
||||
snapshotNextRow = getNext(snapshotIt);
|
||||
|
||||
// Calculate the next value
|
||||
theNext = getLowest(kvsetNextRow, snapshotNextRow);
|
||||
theNext = getLowest(cellSetNextRow, snapshotNextRow);
|
||||
|
||||
// has data
|
||||
return (theNext != null);
|
||||
|
@ -822,37 +819,36 @@ public class DefaultMemStore implements MemStore {
|
|||
get it. So we remember the last keys we iterated to and restore
|
||||
the reseeked set to at least that point.
|
||||
*/
|
||||
KeyValue kv = KeyValueUtil.ensureKeyValue(key);
|
||||
kvsetIt = kvsetAtCreation.tailSet(getHighest(kv, kvsetItRow)).iterator();
|
||||
snapshotIt = snapshotAtCreation.tailSet(getHighest(kv, snapshotItRow)).iterator();
|
||||
cellSetIt = cellSetAtCreation.tailSet(getHighest(key, cellSetItRow)).iterator();
|
||||
snapshotIt = snapshotAtCreation.tailSet(getHighest(key, snapshotItRow)).iterator();
|
||||
|
||||
return seekInSubLists(kv);
|
||||
return seekInSubLists(key);
|
||||
}
|
||||
|
||||
|
||||
@Override
|
||||
public synchronized KeyValue peek() {
|
||||
public synchronized Cell peek() {
|
||||
//DebugPrint.println(" MS@" + hashCode() + " peek = " + getLowest());
|
||||
return theNext;
|
||||
}
|
||||
|
||||
@Override
|
||||
public synchronized KeyValue next() {
|
||||
public synchronized Cell next() {
|
||||
if (theNext == null) {
|
||||
return null;
|
||||
}
|
||||
|
||||
final KeyValue ret = theNext;
|
||||
final Cell ret = theNext;
|
||||
|
||||
// Advance one of the iterators
|
||||
if (theNext == kvsetNextRow) {
|
||||
kvsetNextRow = getNext(kvsetIt);
|
||||
if (theNext == cellSetNextRow) {
|
||||
cellSetNextRow = getNext(cellSetIt);
|
||||
} else {
|
||||
snapshotNextRow = getNext(snapshotIt);
|
||||
}
|
||||
|
||||
// Calculate the next value
|
||||
theNext = getLowest(kvsetNextRow, snapshotNextRow);
|
||||
theNext = getLowest(cellSetNextRow, snapshotNextRow);
|
||||
|
||||
//long readpoint = ReadWriteConsistencyControl.getThreadReadPoint();
|
||||
//DebugPrint.println(" MS@" + hashCode() + " next: " + theNext + " next_next: " +
|
||||
|
@ -865,7 +861,7 @@ public class DefaultMemStore implements MemStore {
|
|||
* This uses comparator.compare() to compare the KeyValue using the memstore
|
||||
* comparator.
|
||||
*/
|
||||
private KeyValue getLowest(KeyValue first, KeyValue second) {
|
||||
private Cell getLowest(Cell first, Cell second) {
|
||||
if (first == null && second == null) {
|
||||
return null;
|
||||
}
|
||||
|
@ -877,11 +873,11 @@ public class DefaultMemStore implements MemStore {
|
|||
}
|
||||
|
||||
/*
|
||||
* Returns the higher of the two key values, or null if they are both null.
|
||||
* This uses comparator.compare() to compare the KeyValue using the memstore
|
||||
* Returns the higher of the two cells, or null if they are both null.
|
||||
* This uses comparator.compare() to compare the Cell using the memstore
|
||||
* comparator.
|
||||
*/
|
||||
private KeyValue getHighest(KeyValue first, KeyValue second) {
|
||||
private Cell getHighest(Cell first, Cell second) {
|
||||
if (first == null && second == null) {
|
||||
return null;
|
||||
}
|
||||
|
@ -893,10 +889,10 @@ public class DefaultMemStore implements MemStore {
|
|||
}
|
||||
|
||||
public synchronized void close() {
|
||||
this.kvsetNextRow = null;
|
||||
this.cellSetNextRow = null;
|
||||
this.snapshotNextRow = null;
|
||||
|
||||
this.kvsetIt = null;
|
||||
this.cellSetIt = null;
|
||||
this.snapshotIt = null;
|
||||
|
||||
if (allocatorAtCreation != null) {
|
||||
|
@ -908,7 +904,7 @@ public class DefaultMemStore implements MemStore {
|
|||
this.snapshotAllocatorAtCreation = null;
|
||||
}
|
||||
|
||||
this.kvsetItRow = null;
|
||||
this.cellSetItRow = null;
|
||||
this.snapshotItRow = null;
|
||||
}
|
||||
|
||||
|
@ -948,47 +944,47 @@ public class DefaultMemStore implements MemStore {
|
|||
*/
|
||||
@Override
|
||||
public synchronized boolean seekToPreviousRow(Cell key) {
|
||||
KeyValue firstKeyOnRow = KeyValueUtil.createFirstOnRow(key.getRowArray(), key.getRowOffset(),
|
||||
Cell firstKeyOnRow = KeyValueUtil.createFirstOnRow(key.getRowArray(), key.getRowOffset(),
|
||||
key.getRowLength());
|
||||
SortedSet<KeyValue> kvHead = kvsetAtCreation.headSet(firstKeyOnRow);
|
||||
KeyValue kvsetBeforeRow = kvHead.isEmpty() ? null : kvHead.last();
|
||||
SortedSet<KeyValue> snapshotHead = snapshotAtCreation
|
||||
SortedSet<Cell> cellHead = cellSetAtCreation.headSet(firstKeyOnRow);
|
||||
Cell cellSetBeforeRow = cellHead.isEmpty() ? null : cellHead.last();
|
||||
SortedSet<Cell> snapshotHead = snapshotAtCreation
|
||||
.headSet(firstKeyOnRow);
|
||||
KeyValue snapshotBeforeRow = snapshotHead.isEmpty() ? null : snapshotHead
|
||||
Cell snapshotBeforeRow = snapshotHead.isEmpty() ? null : snapshotHead
|
||||
.last();
|
||||
KeyValue lastKVBeforeRow = getHighest(kvsetBeforeRow, snapshotBeforeRow);
|
||||
if (lastKVBeforeRow == null) {
|
||||
Cell lastCellBeforeRow = getHighest(cellSetBeforeRow, snapshotBeforeRow);
|
||||
if (lastCellBeforeRow == null) {
|
||||
theNext = null;
|
||||
return false;
|
||||
}
|
||||
KeyValue firstKeyOnPreviousRow = KeyValueUtil.createFirstOnRow(lastKVBeforeRow.getRowArray(),
|
||||
lastKVBeforeRow.getRowOffset(), lastKVBeforeRow.getRowLength());
|
||||
this.stopSkippingKVsIfNextRow = true;
|
||||
Cell firstKeyOnPreviousRow = KeyValueUtil.createFirstOnRow(lastCellBeforeRow.getRowArray(),
|
||||
lastCellBeforeRow.getRowOffset(), lastCellBeforeRow.getRowLength());
|
||||
this.stopSkippingCellsIfNextRow = true;
|
||||
seek(firstKeyOnPreviousRow);
|
||||
this.stopSkippingKVsIfNextRow = false;
|
||||
this.stopSkippingCellsIfNextRow = false;
|
||||
if (peek() == null
|
||||
|| comparator.compareRows(peek(), firstKeyOnPreviousRow) > 0) {
|
||||
return seekToPreviousRow(lastKVBeforeRow);
|
||||
return seekToPreviousRow(lastCellBeforeRow);
|
||||
}
|
||||
return true;
|
||||
}
|
||||
|
||||
@Override
|
||||
public synchronized boolean seekToLastRow() {
|
||||
KeyValue first = kvsetAtCreation.isEmpty() ? null : kvsetAtCreation
|
||||
Cell first = cellSetAtCreation.isEmpty() ? null : cellSetAtCreation
|
||||
.last();
|
||||
KeyValue second = snapshotAtCreation.isEmpty() ? null
|
||||
Cell second = snapshotAtCreation.isEmpty() ? null
|
||||
: snapshotAtCreation.last();
|
||||
KeyValue higherKv = getHighest(first, second);
|
||||
if (higherKv == null) {
|
||||
Cell higherCell = getHighest(first, second);
|
||||
if (higherCell == null) {
|
||||
return false;
|
||||
}
|
||||
KeyValue firstKvOnLastRow = KeyValueUtil.createFirstOnRow(higherKv.getRowArray(),
|
||||
higherKv.getRowOffset(), higherKv.getRowLength());
|
||||
if (seek(firstKvOnLastRow)) {
|
||||
Cell firstCellOnLastRow = KeyValueUtil.createFirstOnRow(higherCell.getRowArray(),
|
||||
higherCell.getRowOffset(), higherCell.getRowLength());
|
||||
if (seek(firstCellOnLastRow)) {
|
||||
return true;
|
||||
} else {
|
||||
return seekToPreviousRow(higherKv);
|
||||
return seekToPreviousRow(higherCell);
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -999,19 +995,18 @@ public class DefaultMemStore implements MemStore {
|
|||
|
||||
public final static long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD +
|
||||
ClassSize.ATOMIC_LONG + (2 * ClassSize.TIMERANGE_TRACKER) +
|
||||
(2 * ClassSize.KEYVALUE_SKIPLIST_SET) + (2 * ClassSize.CONCURRENT_SKIPLISTMAP));
|
||||
(2 * ClassSize.CELL_SKIPLIST_SET) + (2 * ClassSize.CONCURRENT_SKIPLISTMAP));
|
||||
|
||||
/*
|
||||
* Calculate how the MemStore size has changed. Includes overhead of the
|
||||
* backing Map.
|
||||
* @param kv
|
||||
* @param notpresent True if the kv was NOT present in the set.
|
||||
* @param cell
|
||||
* @param notpresent True if the cell was NOT present in the set.
|
||||
* @return Size
|
||||
*/
|
||||
static long heapSizeChange(final KeyValue kv, final boolean notpresent) {
|
||||
return notpresent ?
|
||||
ClassSize.align(ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY + kv.heapSize()):
|
||||
0;
|
||||
static long heapSizeChange(final Cell cell, final boolean notpresent) {
|
||||
return notpresent ? ClassSize.align(ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY
|
||||
+ CellUtil.estimatedHeapSizeOf(cell)) : 0;
|
||||
}
|
||||
|
||||
private long keySize() {
|
||||
|
|
|
@ -35,27 +35,27 @@ import org.apache.hadoop.hbase.regionserver.NonReversedNonLazyKeyValueScanner;
|
|||
*/
|
||||
@InterfaceAudience.Private
|
||||
public class CollectionBackedScanner extends NonReversedNonLazyKeyValueScanner {
|
||||
final private Iterable<KeyValue> data;
|
||||
final private Iterable<Cell> data;
|
||||
final KeyValue.KVComparator comparator;
|
||||
private Iterator<KeyValue> iter;
|
||||
private KeyValue current;
|
||||
private Iterator<Cell> iter;
|
||||
private Cell current;
|
||||
|
||||
public CollectionBackedScanner(SortedSet<KeyValue> set) {
|
||||
public CollectionBackedScanner(SortedSet<Cell> set) {
|
||||
this(set, KeyValue.COMPARATOR);
|
||||
}
|
||||
|
||||
public CollectionBackedScanner(SortedSet<KeyValue> set,
|
||||
public CollectionBackedScanner(SortedSet<Cell> set,
|
||||
KeyValue.KVComparator comparator) {
|
||||
this.comparator = comparator;
|
||||
data = set;
|
||||
init();
|
||||
}
|
||||
|
||||
public CollectionBackedScanner(List<KeyValue> list) {
|
||||
public CollectionBackedScanner(List<Cell> list) {
|
||||
this(list, KeyValue.COMPARATOR);
|
||||
}
|
||||
|
||||
public CollectionBackedScanner(List<KeyValue> list,
|
||||
public CollectionBackedScanner(List<Cell> list,
|
||||
KeyValue.KVComparator comparator) {
|
||||
Collections.sort(list, comparator);
|
||||
this.comparator = comparator;
|
||||
|
@ -64,10 +64,10 @@ public class CollectionBackedScanner extends NonReversedNonLazyKeyValueScanner {
|
|||
}
|
||||
|
||||
public CollectionBackedScanner(KeyValue.KVComparator comparator,
|
||||
KeyValue... array) {
|
||||
Cell... array) {
|
||||
this.comparator = comparator;
|
||||
|
||||
List<KeyValue> tmp = new ArrayList<KeyValue>(array.length);
|
||||
List<Cell> tmp = new ArrayList<Cell>(array.length);
|
||||
Collections.addAll(tmp, array);
|
||||
Collections.sort(tmp, comparator);
|
||||
data = tmp;
|
||||
|
@ -82,13 +82,13 @@ public class CollectionBackedScanner extends NonReversedNonLazyKeyValueScanner {
|
|||
}
|
||||
|
||||
@Override
|
||||
public KeyValue peek() {
|
||||
public Cell peek() {
|
||||
return current;
|
||||
}
|
||||
|
||||
@Override
|
||||
public KeyValue next() {
|
||||
KeyValue oldCurrent = current;
|
||||
public Cell next() {
|
||||
Cell oldCurrent = current;
|
||||
if(iter.hasNext()){
|
||||
current = iter.next();
|
||||
} else {
|
||||
|
@ -98,17 +98,17 @@ public class CollectionBackedScanner extends NonReversedNonLazyKeyValueScanner {
|
|||
}
|
||||
|
||||
@Override
|
||||
public boolean seek(Cell seekKv) {
|
||||
public boolean seek(Cell seekCell) {
|
||||
// restart iterator
|
||||
iter = data.iterator();
|
||||
return reseek(seekKv);
|
||||
return reseek(seekCell);
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean reseek(Cell seekKv) {
|
||||
public boolean reseek(Cell seekCell) {
|
||||
while(iter.hasNext()){
|
||||
KeyValue next = iter.next();
|
||||
int ret = comparator.compare(next, seekKv);
|
||||
Cell next = iter.next();
|
||||
int ret = comparator.compare(next, seekCell);
|
||||
if(ret >= 0){
|
||||
current = next;
|
||||
return true;
|
||||
|
|
|
@ -43,10 +43,10 @@ import org.apache.hadoop.hbase.client.Put;
|
|||
import org.apache.hadoop.hbase.io.hfile.BlockCacheKey;
|
||||
import org.apache.hadoop.hbase.io.hfile.LruCachedBlock;
|
||||
import org.apache.hadoop.hbase.io.hfile.LruBlockCache;
|
||||
import org.apache.hadoop.hbase.regionserver.CellSkipListSet;
|
||||
import org.apache.hadoop.hbase.regionserver.DefaultMemStore;
|
||||
import org.apache.hadoop.hbase.regionserver.HRegion;
|
||||
import org.apache.hadoop.hbase.regionserver.HStore;
|
||||
import org.apache.hadoop.hbase.regionserver.KeyValueSkipListSet;
|
||||
import org.apache.hadoop.hbase.regionserver.TimeRangeTracker;
|
||||
import org.apache.hadoop.hbase.testclassification.IOTests;
|
||||
import org.apache.hadoop.hbase.testclassification.SmallTests;
|
||||
|
@ -236,10 +236,10 @@ public class TestHeapSize {
|
|||
assertEquals(expected, actual);
|
||||
}
|
||||
|
||||
// KeyValueSkipListSet
|
||||
cl = KeyValueSkipListSet.class;
|
||||
// CellSkipListSet
|
||||
cl = CellSkipListSet.class;
|
||||
expected = ClassSize.estimateBase(cl, false);
|
||||
actual = ClassSize.KEYVALUE_SKIPLIST_SET;
|
||||
actual = ClassSize.CELL_SKIPLIST_SET;
|
||||
if (expected != actual) {
|
||||
ClassSize.estimateBase(cl, true);
|
||||
assertEquals(expected, actual);
|
||||
|
@ -305,14 +305,14 @@ public class TestHeapSize {
|
|||
actual = DefaultMemStore.DEEP_OVERHEAD;
|
||||
expected = ClassSize.estimateBase(cl, false);
|
||||
expected += ClassSize.estimateBase(AtomicLong.class, false);
|
||||
expected += (2 * ClassSize.estimateBase(KeyValueSkipListSet.class, false));
|
||||
expected += (2 * ClassSize.estimateBase(CellSkipListSet.class, false));
|
||||
expected += (2 * ClassSize.estimateBase(ConcurrentSkipListMap.class, false));
|
||||
expected += (2 * ClassSize.estimateBase(TimeRangeTracker.class, false));
|
||||
if(expected != actual) {
|
||||
ClassSize.estimateBase(cl, true);
|
||||
ClassSize.estimateBase(AtomicLong.class, true);
|
||||
ClassSize.estimateBase(KeyValueSkipListSet.class, true);
|
||||
ClassSize.estimateBase(KeyValueSkipListSet.class, true);
|
||||
ClassSize.estimateBase(CellSkipListSet.class, true);
|
||||
ClassSize.estimateBase(CellSkipListSet.class, true);
|
||||
ClassSize.estimateBase(ConcurrentSkipListMap.class, true);
|
||||
ClassSize.estimateBase(ConcurrentSkipListMap.class, true);
|
||||
ClassSize.estimateBase(TimeRangeTracker.class, true);
|
||||
|
|
|
@ -69,7 +69,7 @@ public class TestPrefixTreeEncoding {
|
|||
private static final int NUM_COLS_PER_ROW = 20;
|
||||
|
||||
private int numBatchesWritten = 0;
|
||||
private ConcurrentSkipListSet<KeyValue> kvset = new ConcurrentSkipListSet<KeyValue>(
|
||||
private ConcurrentSkipListSet<Cell> kvset = new ConcurrentSkipListSet<Cell>(
|
||||
KeyValue.COMPARATOR);
|
||||
|
||||
private static boolean formatRowNum = false;
|
||||
|
@ -257,18 +257,18 @@ public class TestPrefixTreeEncoding {
|
|||
|
||||
private void dumpInputKVSet() {
|
||||
LOG.info("Dumping input keyvalue set in error case:");
|
||||
for (KeyValue kv : kvset) {
|
||||
for (Cell kv : kvset) {
|
||||
System.out.println(kv);
|
||||
}
|
||||
}
|
||||
|
||||
private static void generateFixedTestData(ConcurrentSkipListSet<KeyValue> kvset, int batchId,
|
||||
private static void generateFixedTestData(ConcurrentSkipListSet<Cell> kvset, int batchId,
|
||||
boolean useTags, PrefixTreeCodec encoder, HFileBlockEncodingContext blkEncodingCtx,
|
||||
DataOutputStream userDataStream) throws Exception {
|
||||
generateFixedTestData(kvset, batchId, true, useTags, encoder, blkEncodingCtx, userDataStream);
|
||||
}
|
||||
|
||||
private static void generateFixedTestData(ConcurrentSkipListSet<KeyValue> kvset,
|
||||
private static void generateFixedTestData(ConcurrentSkipListSet<Cell> kvset,
|
||||
int batchId, boolean partial, boolean useTags, PrefixTreeCodec encoder,
|
||||
HFileBlockEncodingContext blkEncodingCtx, DataOutputStream userDataStream) throws Exception {
|
||||
for (int i = 0; i < NUM_ROWS_PER_BATCH; ++i) {
|
||||
|
@ -287,13 +287,13 @@ public class TestPrefixTreeEncoding {
|
|||
}
|
||||
}
|
||||
encoder.startBlockEncoding(blkEncodingCtx, userDataStream);
|
||||
for (KeyValue kv : kvset) {
|
||||
for (Cell kv : kvset) {
|
||||
encoder.encode(kv, blkEncodingCtx, userDataStream);
|
||||
}
|
||||
encoder.endBlockEncoding(blkEncodingCtx, userDataStream, null);
|
||||
}
|
||||
|
||||
private static void generateRandomTestData(ConcurrentSkipListSet<KeyValue> kvset,
|
||||
private static void generateRandomTestData(ConcurrentSkipListSet<Cell> kvset,
|
||||
int batchId, boolean useTags, PrefixTreeCodec encoder,
|
||||
HFileBlockEncodingContext blkEncodingCtx, DataOutputStream userDataStream) throws Exception {
|
||||
Random random = new Random();
|
||||
|
@ -315,7 +315,7 @@ public class TestPrefixTreeEncoding {
|
|||
}
|
||||
}
|
||||
encoder.startBlockEncoding(blkEncodingCtx, userDataStream);
|
||||
for (KeyValue kv : kvset) {
|
||||
for (Cell kv : kvset) {
|
||||
encoder.encode(kv, blkEncodingCtx, userDataStream);
|
||||
}
|
||||
encoder.endBlockEncoding(blkEncodingCtx, userDataStream, null);
|
||||
|
|
|
@ -21,39 +21,40 @@ package org.apache.hadoop.hbase.regionserver;
|
|||
import java.util.Iterator;
|
||||
import java.util.SortedSet;
|
||||
|
||||
import junit.framework.TestCase;
|
||||
|
||||
import org.apache.hadoop.hbase.Cell;
|
||||
import org.apache.hadoop.hbase.KeyValue;
|
||||
import org.apache.hadoop.hbase.testclassification.RegionServerTests;
|
||||
import org.apache.hadoop.hbase.testclassification.SmallTests;
|
||||
import org.apache.hadoop.hbase.util.Bytes;
|
||||
|
||||
import junit.framework.TestCase;
|
||||
import org.junit.experimental.categories.Category;
|
||||
|
||||
@Category({RegionServerTests.class, SmallTests.class})
|
||||
public class TestKeyValueSkipListSet extends TestCase {
|
||||
private final KeyValueSkipListSet kvsls =
|
||||
new KeyValueSkipListSet(KeyValue.COMPARATOR);
|
||||
public class TestCellSkipListSet extends TestCase {
|
||||
private final CellSkipListSet csls =
|
||||
new CellSkipListSet(KeyValue.COMPARATOR);
|
||||
|
||||
protected void setUp() throws Exception {
|
||||
super.setUp();
|
||||
this.kvsls.clear();
|
||||
this.csls.clear();
|
||||
}
|
||||
|
||||
public void testAdd() throws Exception {
|
||||
byte [] bytes = Bytes.toBytes(getName());
|
||||
KeyValue kv = new KeyValue(bytes, bytes, bytes, bytes);
|
||||
this.kvsls.add(kv);
|
||||
assertTrue(this.kvsls.contains(kv));
|
||||
assertEquals(1, this.kvsls.size());
|
||||
KeyValue first = this.kvsls.first();
|
||||
this.csls.add(kv);
|
||||
assertTrue(this.csls.contains(kv));
|
||||
assertEquals(1, this.csls.size());
|
||||
Cell first = this.csls.first();
|
||||
assertTrue(kv.equals(first));
|
||||
assertTrue(Bytes.equals(kv.getValue(), first.getValue()));
|
||||
// Now try overwritting
|
||||
byte [] overwriteValue = Bytes.toBytes("overwrite");
|
||||
KeyValue overwrite = new KeyValue(bytes, bytes, bytes, overwriteValue);
|
||||
this.kvsls.add(overwrite);
|
||||
assertEquals(1, this.kvsls.size());
|
||||
first = this.kvsls.first();
|
||||
this.csls.add(overwrite);
|
||||
assertEquals(1, this.csls.size());
|
||||
first = this.csls.first();
|
||||
assertTrue(Bytes.equals(overwrite.getValue(), first.getValue()));
|
||||
assertFalse(Bytes.equals(overwrite.getValue(), kv.getValue()));
|
||||
}
|
||||
|
@ -64,11 +65,11 @@ public class TestKeyValueSkipListSet extends TestCase {
|
|||
byte [] value2 = Bytes.toBytes("2");
|
||||
final int total = 3;
|
||||
for (int i = 0; i < total; i++) {
|
||||
this.kvsls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value1));
|
||||
this.csls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value1));
|
||||
}
|
||||
// Assert that we added 'total' values and that they are in order
|
||||
int count = 0;
|
||||
for (KeyValue kv: this.kvsls) {
|
||||
for (Cell kv: this.csls) {
|
||||
assertEquals("" + count, Bytes.toString(kv.getQualifier()));
|
||||
assertTrue(Bytes.equals(kv.getValue(), value1));
|
||||
count++;
|
||||
|
@ -76,12 +77,12 @@ public class TestKeyValueSkipListSet extends TestCase {
|
|||
assertEquals(total, count);
|
||||
// Now overwrite with a new value.
|
||||
for (int i = 0; i < total; i++) {
|
||||
this.kvsls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value2));
|
||||
this.csls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value2));
|
||||
}
|
||||
// Assert that we added 'total' values and that they are in order and that
|
||||
// we are getting back value2
|
||||
count = 0;
|
||||
for (KeyValue kv: this.kvsls) {
|
||||
for (Cell kv: this.csls) {
|
||||
assertEquals("" + count, Bytes.toString(kv.getQualifier()));
|
||||
assertTrue(Bytes.equals(kv.getValue(), value2));
|
||||
count++;
|
||||
|
@ -95,12 +96,12 @@ public class TestKeyValueSkipListSet extends TestCase {
|
|||
byte [] value2 = Bytes.toBytes("2");
|
||||
final int total = 3;
|
||||
for (int i = 0; i < total; i++) {
|
||||
this.kvsls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value1));
|
||||
this.csls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value1));
|
||||
}
|
||||
// Assert that we added 'total' values and that they are in order
|
||||
int count = 0;
|
||||
for (Iterator<KeyValue> i = this.kvsls.descendingIterator(); i.hasNext();) {
|
||||
KeyValue kv = i.next();
|
||||
for (Iterator<Cell> i = this.csls.descendingIterator(); i.hasNext();) {
|
||||
Cell kv = i.next();
|
||||
assertEquals("" + (total - (count + 1)), Bytes.toString(kv.getQualifier()));
|
||||
assertTrue(Bytes.equals(kv.getValue(), value1));
|
||||
count++;
|
||||
|
@ -108,13 +109,13 @@ public class TestKeyValueSkipListSet extends TestCase {
|
|||
assertEquals(total, count);
|
||||
// Now overwrite with a new value.
|
||||
for (int i = 0; i < total; i++) {
|
||||
this.kvsls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value2));
|
||||
this.csls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value2));
|
||||
}
|
||||
// Assert that we added 'total' values and that they are in order and that
|
||||
// we are getting back value2
|
||||
count = 0;
|
||||
for (Iterator<KeyValue> i = this.kvsls.descendingIterator(); i.hasNext();) {
|
||||
KeyValue kv = i.next();
|
||||
for (Iterator<Cell> i = this.csls.descendingIterator(); i.hasNext();) {
|
||||
Cell kv = i.next();
|
||||
assertEquals("" + (total - (count + 1)), Bytes.toString(kv.getQualifier()));
|
||||
assertTrue(Bytes.equals(kv.getValue(), value2));
|
||||
count++;
|
||||
|
@ -131,22 +132,20 @@ public class TestKeyValueSkipListSet extends TestCase {
|
|||
for (int i = 0; i < total; i++) {
|
||||
KeyValue kv = new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value1);
|
||||
if (i == 1) splitter = kv;
|
||||
this.kvsls.add(kv);
|
||||
this.csls.add(kv);
|
||||
}
|
||||
SortedSet<KeyValue> tail = this.kvsls.tailSet(splitter);
|
||||
SortedSet<Cell> tail = this.csls.tailSet(splitter);
|
||||
assertEquals(2, tail.size());
|
||||
SortedSet<KeyValue> head = this.kvsls.headSet(splitter);
|
||||
SortedSet<Cell> head = this.csls.headSet(splitter);
|
||||
assertEquals(1, head.size());
|
||||
// Now ensure that we get back right answer even when we do tail or head.
|
||||
// Now overwrite with a new value.
|
||||
for (int i = 0; i < total; i++) {
|
||||
this.kvsls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value2));
|
||||
this.csls.add(new KeyValue(bytes, bytes, Bytes.toBytes("" + i), value2));
|
||||
}
|
||||
tail = this.kvsls.tailSet(splitter);
|
||||
tail = this.csls.tailSet(splitter);
|
||||
assertTrue(Bytes.equals(tail.first().getValue(), value2));
|
||||
head = this.kvsls.headSet(splitter);
|
||||
head = this.csls.headSet(splitter);
|
||||
assertTrue(Bytes.equals(head.first().getValue(), value2));
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
}
|
|
@ -79,8 +79,8 @@ public class TestDefaultMemStore extends TestCase {
|
|||
byte [] other = Bytes.toBytes("somethingelse");
|
||||
KeyValue samekey = new KeyValue(bytes, bytes, bytes, other);
|
||||
this.memstore.add(samekey);
|
||||
KeyValue found = this.memstore.kvset.first();
|
||||
assertEquals(1, this.memstore.kvset.size());
|
||||
Cell found = this.memstore.cellSet.first();
|
||||
assertEquals(1, this.memstore.cellSet.size());
|
||||
assertTrue(Bytes.toString(found.getValue()), CellUtil.matchingValue(samekey, found));
|
||||
}
|
||||
|
||||
|
@ -483,7 +483,7 @@ public class TestDefaultMemStore extends TestCase {
|
|||
m.add(key2);
|
||||
|
||||
assertTrue("Expected memstore to hold 3 values, actually has " +
|
||||
m.kvset.size(), m.kvset.size() == 3);
|
||||
m.cellSet.size(), m.cellSet.size() == 3);
|
||||
}
|
||||
|
||||
//////////////////////////////////////////////////////////////////////////////
|
||||
|
@ -498,11 +498,11 @@ public class TestDefaultMemStore extends TestCase {
|
|||
// Add more versions to make it a little more interesting.
|
||||
Thread.sleep(1);
|
||||
addRows(this.memstore);
|
||||
KeyValue closestToEmpty = this.memstore.getNextRow(KeyValue.LOWESTKEY);
|
||||
Cell closestToEmpty = this.memstore.getNextRow(KeyValue.LOWESTKEY);
|
||||
assertTrue(KeyValue.COMPARATOR.compareRows(closestToEmpty,
|
||||
new KeyValue(Bytes.toBytes(0), System.currentTimeMillis())) == 0);
|
||||
for (int i = 0; i < ROW_COUNT; i++) {
|
||||
KeyValue nr = this.memstore.getNextRow(new KeyValue(Bytes.toBytes(i),
|
||||
Cell nr = this.memstore.getNextRow(new KeyValue(Bytes.toBytes(i),
|
||||
System.currentTimeMillis()));
|
||||
if (i + 1 == ROW_COUNT) {
|
||||
assertEquals(nr, null);
|
||||
|
@ -558,10 +558,10 @@ public class TestDefaultMemStore extends TestCase {
|
|||
memstore.snapshot();
|
||||
assertEquals(3, memstore.snapshot.size());
|
||||
//Adding value to "new" memstore
|
||||
assertEquals(0, memstore.kvset.size());
|
||||
assertEquals(0, memstore.cellSet.size());
|
||||
memstore.add(new KeyValue(row, fam ,qf4, val));
|
||||
memstore.add(new KeyValue(row, fam ,qf5, val));
|
||||
assertEquals(2, memstore.kvset.size());
|
||||
assertEquals(2, memstore.cellSet.size());
|
||||
}
|
||||
|
||||
//////////////////////////////////////////////////////////////////////////////
|
||||
|
@ -583,7 +583,7 @@ public class TestDefaultMemStore extends TestCase {
|
|||
memstore.add(put2);
|
||||
memstore.add(put3);
|
||||
|
||||
assertEquals(3, memstore.kvset.size());
|
||||
assertEquals(3, memstore.cellSet.size());
|
||||
|
||||
KeyValue del2 = new KeyValue(row, fam, qf1, ts2, KeyValue.Type.Delete, val);
|
||||
memstore.delete(del2);
|
||||
|
@ -594,10 +594,10 @@ public class TestDefaultMemStore extends TestCase {
|
|||
expected.add(put2);
|
||||
expected.add(put1);
|
||||
|
||||
assertEquals(4, memstore.kvset.size());
|
||||
assertEquals(4, memstore.cellSet.size());
|
||||
int i = 0;
|
||||
for(KeyValue kv : memstore.kvset) {
|
||||
assertEquals(expected.get(i++), kv);
|
||||
for(Cell cell : memstore.cellSet) {
|
||||
assertEquals(expected.get(i++), cell);
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -617,7 +617,7 @@ public class TestDefaultMemStore extends TestCase {
|
|||
memstore.add(put2);
|
||||
memstore.add(put3);
|
||||
|
||||
assertEquals(3, memstore.kvset.size());
|
||||
assertEquals(3, memstore.cellSet.size());
|
||||
|
||||
KeyValue del2 =
|
||||
new KeyValue(row, fam, qf1, ts2, KeyValue.Type.DeleteColumn, val);
|
||||
|
@ -630,10 +630,10 @@ public class TestDefaultMemStore extends TestCase {
|
|||
expected.add(put1);
|
||||
|
||||
|
||||
assertEquals(4, memstore.kvset.size());
|
||||
assertEquals(4, memstore.cellSet.size());
|
||||
int i = 0;
|
||||
for (KeyValue kv: memstore.kvset) {
|
||||
assertEquals(expected.get(i++), kv);
|
||||
for (Cell cell: memstore.cellSet) {
|
||||
assertEquals(expected.get(i++), cell);
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -670,10 +670,10 @@ public class TestDefaultMemStore extends TestCase {
|
|||
|
||||
|
||||
|
||||
assertEquals(5, memstore.kvset.size());
|
||||
assertEquals(5, memstore.cellSet.size());
|
||||
int i = 0;
|
||||
for (KeyValue kv: memstore.kvset) {
|
||||
assertEquals(expected.get(i++), kv);
|
||||
for (Cell cell: memstore.cellSet) {
|
||||
assertEquals(expected.get(i++), cell);
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -686,8 +686,8 @@ public class TestDefaultMemStore extends TestCase {
|
|||
memstore.add(new KeyValue(row, fam, qf, ts, val));
|
||||
KeyValue delete = new KeyValue(row, fam, qf, ts, KeyValue.Type.Delete, val);
|
||||
memstore.delete(delete);
|
||||
assertEquals(2, memstore.kvset.size());
|
||||
assertEquals(delete, memstore.kvset.first());
|
||||
assertEquals(2, memstore.cellSet.size());
|
||||
assertEquals(delete, memstore.cellSet.first());
|
||||
}
|
||||
|
||||
public void testRetainsDeleteVersion() throws IOException {
|
||||
|
@ -699,8 +699,8 @@ public class TestDefaultMemStore extends TestCase {
|
|||
"row1", "fam", "a", 100, KeyValue.Type.Delete, "dont-care");
|
||||
memstore.delete(delete);
|
||||
|
||||
assertEquals(2, memstore.kvset.size());
|
||||
assertEquals(delete, memstore.kvset.first());
|
||||
assertEquals(2, memstore.cellSet.size());
|
||||
assertEquals(delete, memstore.cellSet.first());
|
||||
}
|
||||
public void testRetainsDeleteColumn() throws IOException {
|
||||
// add a put to memstore
|
||||
|
@ -711,8 +711,8 @@ public class TestDefaultMemStore extends TestCase {
|
|||
KeyValue.Type.DeleteColumn, "dont-care");
|
||||
memstore.delete(delete);
|
||||
|
||||
assertEquals(2, memstore.kvset.size());
|
||||
assertEquals(delete, memstore.kvset.first());
|
||||
assertEquals(2, memstore.cellSet.size());
|
||||
assertEquals(delete, memstore.cellSet.first());
|
||||
}
|
||||
public void testRetainsDeleteFamily() throws IOException {
|
||||
// add a put to memstore
|
||||
|
@ -723,8 +723,8 @@ public class TestDefaultMemStore extends TestCase {
|
|||
KeyValue.Type.DeleteFamily, "dont-care");
|
||||
memstore.delete(delete);
|
||||
|
||||
assertEquals(2, memstore.kvset.size());
|
||||
assertEquals(delete, memstore.kvset.first());
|
||||
assertEquals(2, memstore.cellSet.size());
|
||||
assertEquals(delete, memstore.cellSet.first());
|
||||
}
|
||||
|
||||
////////////////////////////////////
|
||||
|
|
|
@ -2189,10 +2189,10 @@ public class TestHRegion {
|
|||
// This is kinda hacky, but better than nothing...
|
||||
long now = System.currentTimeMillis();
|
||||
DefaultMemStore memstore = (DefaultMemStore) ((HStore) region.getStore(fam1)).memstore;
|
||||
KeyValue firstKv = memstore.kvset.first();
|
||||
assertTrue(firstKv.getTimestamp() <= now);
|
||||
now = firstKv.getTimestamp();
|
||||
for (Cell cell : memstore.kvset) {
|
||||
Cell firstCell = memstore.cellSet.first();
|
||||
assertTrue(firstCell.getTimestamp() <= now);
|
||||
now = firstCell.getTimestamp();
|
||||
for (Cell cell : memstore.cellSet) {
|
||||
assertTrue(cell.getTimestamp() <= now);
|
||||
now = cell.getTimestamp();
|
||||
}
|
||||
|
|
|
@ -71,18 +71,18 @@ public class TestKeyValueHeap extends HBaseTestCase {
|
|||
//1. The "smallest" KeyValue is in the same scanners as current
|
||||
//2. Current scanner gets empty
|
||||
|
||||
List<KeyValue> l1 = new ArrayList<KeyValue>();
|
||||
List<Cell> l1 = new ArrayList<Cell>();
|
||||
l1.add(new KeyValue(row1, fam1, col5, data));
|
||||
l1.add(new KeyValue(row2, fam1, col1, data));
|
||||
l1.add(new KeyValue(row2, fam1, col2, data));
|
||||
scanners.add(new Scanner(l1));
|
||||
|
||||
List<KeyValue> l2 = new ArrayList<KeyValue>();
|
||||
List<Cell> l2 = new ArrayList<Cell>();
|
||||
l2.add(new KeyValue(row1, fam1, col1, data));
|
||||
l2.add(new KeyValue(row1, fam1, col2, data));
|
||||
scanners.add(new Scanner(l2));
|
||||
|
||||
List<KeyValue> l3 = new ArrayList<KeyValue>();
|
||||
List<Cell> l3 = new ArrayList<Cell>();
|
||||
l3.add(new KeyValue(row1, fam1, col3, data));
|
||||
l3.add(new KeyValue(row1, fam1, col4, data));
|
||||
l3.add(new KeyValue(row1, fam2, col1, data));
|
||||
|
@ -133,18 +133,18 @@ public class TestKeyValueHeap extends HBaseTestCase {
|
|||
//1. Seek KeyValue that is not in scanner
|
||||
//2. Check that smallest that is returned from a seek is correct
|
||||
|
||||
List<KeyValue> l1 = new ArrayList<KeyValue>();
|
||||
List<Cell> l1 = new ArrayList<Cell>();
|
||||
l1.add(new KeyValue(row1, fam1, col5, data));
|
||||
l1.add(new KeyValue(row2, fam1, col1, data));
|
||||
l1.add(new KeyValue(row2, fam1, col2, data));
|
||||
scanners.add(new Scanner(l1));
|
||||
|
||||
List<KeyValue> l2 = new ArrayList<KeyValue>();
|
||||
List<Cell> l2 = new ArrayList<Cell>();
|
||||
l2.add(new KeyValue(row1, fam1, col1, data));
|
||||
l2.add(new KeyValue(row1, fam1, col2, data));
|
||||
scanners.add(new Scanner(l2));
|
||||
|
||||
List<KeyValue> l3 = new ArrayList<KeyValue>();
|
||||
List<Cell> l3 = new ArrayList<Cell>();
|
||||
l3.add(new KeyValue(row1, fam1, col3, data));
|
||||
l3.add(new KeyValue(row1, fam1, col4, data));
|
||||
l3.add(new KeyValue(row1, fam2, col1, data));
|
||||
|
@ -179,18 +179,18 @@ public class TestKeyValueHeap extends HBaseTestCase {
|
|||
public void testScannerLeak() throws IOException {
|
||||
// Test for unclosed scanners (HBASE-1927)
|
||||
|
||||
List<KeyValue> l1 = new ArrayList<KeyValue>();
|
||||
List<Cell> l1 = new ArrayList<Cell>();
|
||||
l1.add(new KeyValue(row1, fam1, col5, data));
|
||||
l1.add(new KeyValue(row2, fam1, col1, data));
|
||||
l1.add(new KeyValue(row2, fam1, col2, data));
|
||||
scanners.add(new Scanner(l1));
|
||||
|
||||
List<KeyValue> l2 = new ArrayList<KeyValue>();
|
||||
List<Cell> l2 = new ArrayList<Cell>();
|
||||
l2.add(new KeyValue(row1, fam1, col1, data));
|
||||
l2.add(new KeyValue(row1, fam1, col2, data));
|
||||
scanners.add(new Scanner(l2));
|
||||
|
||||
List<KeyValue> l3 = new ArrayList<KeyValue>();
|
||||
List<Cell> l3 = new ArrayList<Cell>();
|
||||
l3.add(new KeyValue(row1, fam1, col3, data));
|
||||
l3.add(new KeyValue(row1, fam1, col4, data));
|
||||
l3.add(new KeyValue(row1, fam2, col1, data));
|
||||
|
@ -198,7 +198,7 @@ public class TestKeyValueHeap extends HBaseTestCase {
|
|||
l3.add(new KeyValue(row2, fam1, col3, data));
|
||||
scanners.add(new Scanner(l3));
|
||||
|
||||
List<KeyValue> l4 = new ArrayList<KeyValue>();
|
||||
List<Cell> l4 = new ArrayList<Cell>();
|
||||
scanners.add(new Scanner(l4));
|
||||
|
||||
//Creating KeyValueHeap
|
||||
|
@ -213,10 +213,10 @@ public class TestKeyValueHeap extends HBaseTestCase {
|
|||
|
||||
private static class Scanner extends CollectionBackedScanner {
|
||||
private Iterator<Cell> iter;
|
||||
private KeyValue current;
|
||||
private Cell current;
|
||||
private boolean closed = false;
|
||||
|
||||
public Scanner(List<KeyValue> list) {
|
||||
public Scanner(List<Cell> list) {
|
||||
super(list);
|
||||
}
|
||||
|
||||
|
|
|
@ -119,10 +119,10 @@ public class TestMemStoreChunkPool {
|
|||
assertEquals(3, memstore.snapshot.size());
|
||||
|
||||
// Adding value to "new" memstore
|
||||
assertEquals(0, memstore.kvset.size());
|
||||
assertEquals(0, memstore.cellSet.size());
|
||||
memstore.add(new KeyValue(row, fam, qf4, val));
|
||||
memstore.add(new KeyValue(row, fam, qf5, val));
|
||||
assertEquals(2, memstore.kvset.size());
|
||||
assertEquals(2, memstore.cellSet.size());
|
||||
memstore.clearSnapshot(snapshot.getId());
|
||||
|
||||
int chunkCount = chunkPool.getPoolSize();
|
||||
|
@ -156,10 +156,10 @@ public class TestMemStoreChunkPool {
|
|||
assertEquals(3, memstore.snapshot.size());
|
||||
|
||||
// Adding value to "new" memstore
|
||||
assertEquals(0, memstore.kvset.size());
|
||||
assertEquals(0, memstore.cellSet.size());
|
||||
memstore.add(new KeyValue(row, fam, qf4, val));
|
||||
memstore.add(new KeyValue(row, fam, qf5, val));
|
||||
assertEquals(2, memstore.kvset.size());
|
||||
assertEquals(2, memstore.cellSet.size());
|
||||
|
||||
// opening scanner before clear the snapshot
|
||||
List<KeyValueScanner> scanners = memstore.getScanners(0);
|
||||
|
|
|
@ -526,7 +526,7 @@ public class TestStore {
|
|||
this.store.snapshot();
|
||||
flushStore(store, id++);
|
||||
Assert.assertEquals(storeFilessize, this.store.getStorefiles().size());
|
||||
Assert.assertEquals(0, ((DefaultMemStore)this.store.memstore).kvset.size());
|
||||
Assert.assertEquals(0, ((DefaultMemStore)this.store.memstore).cellSet.size());
|
||||
}
|
||||
|
||||
private void assertCheck() {
|
||||
|
@ -571,7 +571,7 @@ public class TestStore {
|
|||
flushStore(store, id++);
|
||||
Assert.assertEquals(1, this.store.getStorefiles().size());
|
||||
// from the one we inserted up there, and a new one
|
||||
Assert.assertEquals(2, ((DefaultMemStore)this.store.memstore).kvset.size());
|
||||
Assert.assertEquals(2, ((DefaultMemStore)this.store.memstore).cellSet.size());
|
||||
|
||||
// how many key/values for this row are there?
|
||||
Get get = new Get(row);
|
||||
|
@ -645,8 +645,8 @@ public class TestStore {
|
|||
}
|
||||
|
||||
long computedSize=0;
|
||||
for (KeyValue kv : ((DefaultMemStore)this.store.memstore).kvset) {
|
||||
long kvsize = DefaultMemStore.heapSizeChange(kv, true);
|
||||
for (Cell cell : ((DefaultMemStore)this.store.memstore).cellSet) {
|
||||
long kvsize = DefaultMemStore.heapSizeChange(cell, true);
|
||||
//System.out.println(kv + " size= " + kvsize + " kvsize= " + kv.heapSize());
|
||||
computedSize += kvsize;
|
||||
}
|
||||
|
@ -677,7 +677,7 @@ public class TestStore {
|
|||
// then flush.
|
||||
flushStore(store, id++);
|
||||
Assert.assertEquals(1, this.store.getStorefiles().size());
|
||||
Assert.assertEquals(1, ((DefaultMemStore)this.store.memstore).kvset.size());
|
||||
Assert.assertEquals(1, ((DefaultMemStore)this.store.memstore).cellSet.size());
|
||||
|
||||
// now increment again:
|
||||
newValue += 1;
|
||||
|
|
Loading…
Reference in New Issue