mirror of
https://github.com/usatiuk/dhfs.git
synced 2025-10-28 20:47:49 +01:00
bunch of iterators with prev
This commit is contained in:
@@ -9,4 +9,24 @@ public interface CloseableKvIterator<K extends Comparable<K>, V> extends Iterato
|
||||
K peekNextKey();
|
||||
|
||||
void skip();
|
||||
|
||||
default K peekPrevKey() {
|
||||
throw new UnsupportedOperationException();
|
||||
}
|
||||
|
||||
default Pair<K, V> prev() {
|
||||
throw new UnsupportedOperationException();
|
||||
}
|
||||
|
||||
default boolean hasPrev() {
|
||||
throw new UnsupportedOperationException();
|
||||
}
|
||||
|
||||
default void skipPrev() {
|
||||
throw new UnsupportedOperationException();
|
||||
}
|
||||
|
||||
default CloseableKvIterator<K, V> reversed() {
|
||||
return new ReversedKvIterator<>(this);
|
||||
}
|
||||
}
|
||||
|
||||
@@ -33,6 +33,27 @@ public class MappingKvIterator<K extends Comparable<K>, V, V_T> implements Close
|
||||
return _backing.hasNext();
|
||||
}
|
||||
|
||||
@Override
|
||||
public K peekPrevKey() {
|
||||
return _backing.peekPrevKey();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V_T> prev() {
|
||||
var got = _backing.prev();
|
||||
return Pair.of(got.getKey(), _transformer.apply(got.getValue()));
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasPrev() {
|
||||
return _backing.hasPrev();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skipPrev() {
|
||||
_backing.skipPrev();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V_T> next() {
|
||||
var got = _backing.next();
|
||||
|
||||
@@ -6,12 +6,13 @@ import org.apache.commons.lang3.tuple.Pair;
|
||||
|
||||
import java.util.*;
|
||||
|
||||
public class MergingKvIterator<K extends Comparable<K>, V> implements CloseableKvIterator<K, V> {
|
||||
public class MergingKvIterator<K extends Comparable<K>, V> extends ReversibleKvIterator<K, V> {
|
||||
private final Map<CloseableKvIterator<K, V>, Integer> _iterators;
|
||||
private final NavigableMap<K, CloseableKvIterator<K, V>> _sortedIterators = new TreeMap<>();
|
||||
private final String _name;
|
||||
|
||||
public MergingKvIterator(String name, IteratorStart startType, K startKey, List<IterProdFn<K, V>> iterators) {
|
||||
_goingForward = true;
|
||||
_name = name;
|
||||
|
||||
IteratorStart initialStartType = startType;
|
||||
@@ -92,9 +93,10 @@ public class MergingKvIterator<K extends Comparable<K>, V> implements CloseableK
|
||||
return;
|
||||
}
|
||||
|
||||
var oursPrio = _iterators.get(iterator);
|
||||
// Expects that reversed iterator returns itself when reversed again
|
||||
var oursPrio = _iterators.get(_goingForward ? iterator : iterator.reversed());
|
||||
var them = _sortedIterators.get(key);
|
||||
var theirsPrio = _iterators.get(them);
|
||||
var theirsPrio = _iterators.get(_goingForward ? them : them.reversed());
|
||||
if (oursPrio < theirsPrio) {
|
||||
_sortedIterators.put(key, iterator);
|
||||
advanceIterator(them);
|
||||
@@ -106,15 +108,36 @@ public class MergingKvIterator<K extends Comparable<K>, V> implements CloseableK
|
||||
}
|
||||
|
||||
@Override
|
||||
public K peekNextKey() {
|
||||
if (_sortedIterators.isEmpty())
|
||||
throw new NoSuchElementException();
|
||||
return _sortedIterators.firstKey();
|
||||
protected void reverse() {
|
||||
var cur = _goingForward ? _sortedIterators.pollFirstEntry() : _sortedIterators.pollLastEntry();
|
||||
_goingForward = !_goingForward;
|
||||
_sortedIterators.clear();
|
||||
for (CloseableKvIterator<K, V> iterator : _iterators.keySet()) {
|
||||
// _goingForward inverted already
|
||||
advanceIterator(!_goingForward ? iterator.reversed() : iterator);
|
||||
}
|
||||
if (_sortedIterators.isEmpty() || cur == null) {
|
||||
return;
|
||||
}
|
||||
// Advance to the expected key, as we might have brought back some iterators
|
||||
// that were at their ends
|
||||
while (!_sortedIterators.isEmpty()
|
||||
&& ((_goingForward && peekImpl().compareTo(cur.getKey()) <= 0)
|
||||
|| (!_goingForward && peekImpl().compareTo(cur.getKey()) >= 0))) {
|
||||
skipImpl();
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skip() {
|
||||
var cur = _sortedIterators.pollFirstEntry();
|
||||
protected K peekImpl() {
|
||||
if (_sortedIterators.isEmpty())
|
||||
throw new NoSuchElementException();
|
||||
return _goingForward ? _sortedIterators.firstKey() : _sortedIterators.lastKey();
|
||||
}
|
||||
|
||||
@Override
|
||||
protected void skipImpl() {
|
||||
var cur = _goingForward ? _sortedIterators.pollFirstEntry() : _sortedIterators.pollLastEntry();
|
||||
if (cur == null) {
|
||||
throw new NoSuchElementException();
|
||||
}
|
||||
@@ -124,20 +147,13 @@ public class MergingKvIterator<K extends Comparable<K>, V> implements CloseableK
|
||||
}
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
for (CloseableKvIterator<K, V> iterator : _iterators.keySet()) {
|
||||
iterator.close();
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasNext() {
|
||||
protected boolean hasImpl() {
|
||||
return !_sortedIterators.isEmpty();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V> next() {
|
||||
var cur = _sortedIterators.pollFirstEntry();
|
||||
protected Pair<K, V> nextImpl() {
|
||||
var cur = _goingForward ? _sortedIterators.pollFirstEntry() : _sortedIterators.pollLastEntry();
|
||||
if (cur == null) {
|
||||
throw new NoSuchElementException();
|
||||
}
|
||||
@@ -147,6 +163,14 @@ public class MergingKvIterator<K extends Comparable<K>, V> implements CloseableK
|
||||
return curVal;
|
||||
}
|
||||
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
for (CloseableKvIterator<K, V> iterator : _iterators.keySet()) {
|
||||
iterator.close();
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public String toString() {
|
||||
return "MergingKvIterator{" +
|
||||
|
||||
@@ -5,23 +5,26 @@ import org.apache.commons.lang3.tuple.Pair;
|
||||
|
||||
import java.util.*;
|
||||
|
||||
public class NavigableMapKvIterator<K extends Comparable<K>, V> implements CloseableKvIterator<K, V> {
|
||||
private final Iterator<Map.Entry<K, V>> _iterator;
|
||||
public class NavigableMapKvIterator<K extends Comparable<K>, V> extends ReversibleKvIterator<K, V> {
|
||||
private final NavigableMap<K, V> _map;
|
||||
private Iterator<Map.Entry<K, V>> _iterator;
|
||||
private Map.Entry<K, V> _next;
|
||||
|
||||
public NavigableMapKvIterator(NavigableMap<K, V> map, IteratorStart start, K key) {
|
||||
_map = map;
|
||||
SortedMap<K, V> _view;
|
||||
_goingForward = true;
|
||||
switch (start) {
|
||||
case GE -> _view = map.tailMap(key, true);
|
||||
case GT -> _view = map.tailMap(key, false);
|
||||
case LE -> {
|
||||
var floorKey = map.floorKey(key);
|
||||
if (floorKey == null) _view = Collections.emptyNavigableMap();
|
||||
if (floorKey == null) _view = _map;
|
||||
else _view = map.tailMap(floorKey, true);
|
||||
}
|
||||
case LT -> {
|
||||
var lowerKey = map.lowerKey(key);
|
||||
if (lowerKey == null) _view = Collections.emptyNavigableMap();
|
||||
if (lowerKey == null) _view = _map;
|
||||
else _view = map.tailMap(lowerKey, true);
|
||||
}
|
||||
default -> throw new IllegalArgumentException("Unknown start type");
|
||||
@@ -30,6 +33,25 @@ public class NavigableMapKvIterator<K extends Comparable<K>, V> implements Close
|
||||
fillNext();
|
||||
}
|
||||
|
||||
@Override
|
||||
protected void reverse() {
|
||||
var oldNext = _next;
|
||||
_next = null;
|
||||
if (_goingForward) {
|
||||
_iterator
|
||||
= oldNext == null
|
||||
? _map.descendingMap().entrySet().iterator()
|
||||
: _map.headMap(oldNext.getKey(), false).descendingMap().entrySet().iterator();
|
||||
} else {
|
||||
_iterator
|
||||
= oldNext == null
|
||||
? _map.entrySet().iterator()
|
||||
: _map.tailMap(oldNext.getKey(), false).entrySet().iterator();
|
||||
}
|
||||
_goingForward = !_goingForward;
|
||||
fillNext();
|
||||
}
|
||||
|
||||
private void fillNext() {
|
||||
while (_iterator.hasNext() && _next == null) {
|
||||
_next = _iterator.next();
|
||||
@@ -37,7 +59,7 @@ public class NavigableMapKvIterator<K extends Comparable<K>, V> implements Close
|
||||
}
|
||||
|
||||
@Override
|
||||
public K peekNextKey() {
|
||||
protected K peekImpl() {
|
||||
if (_next == null) {
|
||||
throw new NoSuchElementException();
|
||||
}
|
||||
@@ -45,7 +67,7 @@ public class NavigableMapKvIterator<K extends Comparable<K>, V> implements Close
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skip() {
|
||||
protected void skipImpl() {
|
||||
if (_next == null) {
|
||||
throw new NoSuchElementException();
|
||||
}
|
||||
@@ -54,16 +76,12 @@ public class NavigableMapKvIterator<K extends Comparable<K>, V> implements Close
|
||||
}
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasNext() {
|
||||
protected boolean hasImpl() {
|
||||
return _next != null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V> next() {
|
||||
protected Pair<K, V> nextImpl() {
|
||||
if (_next == null) {
|
||||
throw new NoSuchElementException("No more elements");
|
||||
}
|
||||
@@ -73,10 +91,13 @@ public class NavigableMapKvIterator<K extends Comparable<K>, V> implements Close
|
||||
return Pair.of(ret);
|
||||
}
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
}
|
||||
|
||||
@Override
|
||||
public String toString() {
|
||||
return "NavigableMapKvIterator{" +
|
||||
"_iterator=" + _iterator +
|
||||
", _next=" + _next +
|
||||
'}';
|
||||
}
|
||||
|
||||
@@ -6,32 +6,58 @@ import org.apache.commons.lang3.tuple.Pair;
|
||||
import java.util.NoSuchElementException;
|
||||
import java.util.function.Function;
|
||||
|
||||
public class PredicateKvIterator<K extends Comparable<K>, V, V_T> implements CloseableKvIterator<K, V_T> {
|
||||
public class PredicateKvIterator<K extends Comparable<K>, V, V_T> extends ReversibleKvIterator<K, V_T> {
|
||||
private final CloseableKvIterator<K, V> _backing;
|
||||
private final Function<V, V_T> _transformer;
|
||||
private Pair<K, V_T> _next;
|
||||
|
||||
public PredicateKvIterator(CloseableKvIterator<K, V> backing, IteratorStart start, K startKey, Function<V, V_T> transformer) {
|
||||
_goingForward = true;
|
||||
_backing = backing;
|
||||
_transformer = transformer;
|
||||
fillNext();
|
||||
if (_next == null) {
|
||||
return;
|
||||
}
|
||||
|
||||
boolean shouldGoBack = false;
|
||||
if (start == IteratorStart.LE) {
|
||||
if (_next.getKey().compareTo(startKey) > 0) {
|
||||
_next = null;
|
||||
if (_next == null || _next.getKey().compareTo(startKey) > 0) {
|
||||
shouldGoBack = true;
|
||||
}
|
||||
} else if (start == IteratorStart.LT) {
|
||||
if (_next.getKey().compareTo(startKey) >= 0) {
|
||||
_next = null;
|
||||
if (_next == null || _next.getKey().compareTo(startKey) >= 0) {
|
||||
shouldGoBack = true;
|
||||
}
|
||||
}
|
||||
|
||||
if (shouldGoBack && _backing.hasPrev()) {
|
||||
_goingForward = false;
|
||||
_next = null;
|
||||
_backing.skipPrev();
|
||||
fillNext();
|
||||
_goingForward = true;
|
||||
_backing.skip();
|
||||
fillNext();
|
||||
}
|
||||
|
||||
|
||||
switch (start) {
|
||||
case LT -> {
|
||||
// assert _next == null || _next.getKey().compareTo(startKey) < 0;
|
||||
}
|
||||
case LE -> {
|
||||
// assert _next == null || _next.getKey().compareTo(startKey) <= 0;
|
||||
}
|
||||
case GT -> {
|
||||
assert _next == null || _next.getKey().compareTo(startKey) > 0;
|
||||
}
|
||||
case GE -> {
|
||||
assert _next == null || _next.getKey().compareTo(startKey) >= 0;
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
private void fillNext() {
|
||||
while (_backing.hasNext() && _next == null) {
|
||||
var next = _backing.next();
|
||||
while ((_goingForward ? _backing.hasNext() : _backing.hasPrev()) && _next == null) {
|
||||
var next = _goingForward ? _backing.next() : _backing.prev();
|
||||
var transformed = _transformer.apply(next.getValue());
|
||||
if (transformed == null)
|
||||
continue;
|
||||
@@ -40,14 +66,27 @@ public class PredicateKvIterator<K extends Comparable<K>, V, V_T> implements Clo
|
||||
}
|
||||
|
||||
@Override
|
||||
public K peekNextKey() {
|
||||
protected void reverse() {
|
||||
_goingForward = !_goingForward;
|
||||
_next = null;
|
||||
|
||||
if (_goingForward && _backing.hasNext())
|
||||
_backing.skip();
|
||||
else if (!_goingForward && _backing.hasPrev())
|
||||
_backing.skipPrev();
|
||||
|
||||
fillNext();
|
||||
}
|
||||
|
||||
@Override
|
||||
protected K peekImpl() {
|
||||
if (_next == null)
|
||||
throw new NoSuchElementException();
|
||||
return _next.getKey();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skip() {
|
||||
protected void skipImpl() {
|
||||
if (_next == null)
|
||||
throw new NoSuchElementException();
|
||||
_next = null;
|
||||
@@ -55,17 +94,12 @@ public class PredicateKvIterator<K extends Comparable<K>, V, V_T> implements Clo
|
||||
}
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
_backing.close();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasNext() {
|
||||
protected boolean hasImpl() {
|
||||
return _next != null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V_T> next() {
|
||||
protected Pair<K, V_T> nextImpl() {
|
||||
if (_next == null)
|
||||
throw new NoSuchElementException("No more elements");
|
||||
var ret = _next;
|
||||
@@ -74,6 +108,11 @@ public class PredicateKvIterator<K extends Comparable<K>, V, V_T> implements Clo
|
||||
return ret;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
_backing.close();
|
||||
}
|
||||
|
||||
@Override
|
||||
public String toString() {
|
||||
return "PredicateKvIterator{" +
|
||||
|
||||
@@ -0,0 +1,61 @@
|
||||
package com.usatiuk.dhfs.objects;
|
||||
|
||||
import org.apache.commons.lang3.tuple.Pair;
|
||||
|
||||
public class ReversedKvIterator<K extends Comparable<K>, V> implements CloseableKvIterator<K, V> {
|
||||
private final CloseableKvIterator<K, V> _backing;
|
||||
|
||||
public ReversedKvIterator(CloseableKvIterator<K, V> backing) {
|
||||
_backing = backing;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
_backing.close();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasNext() {
|
||||
return _backing.hasPrev();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V> next() {
|
||||
return _backing.prev();
|
||||
}
|
||||
|
||||
@Override
|
||||
public K peekNextKey() {
|
||||
return _backing.peekPrevKey();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skip() {
|
||||
_backing.skipPrev();
|
||||
}
|
||||
|
||||
@Override
|
||||
public K peekPrevKey() {
|
||||
return _backing.peekNextKey();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V> prev() {
|
||||
return _backing.next();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasPrev() {
|
||||
return _backing.hasNext();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skipPrev() {
|
||||
_backing.skip();
|
||||
}
|
||||
|
||||
@Override
|
||||
public CloseableKvIterator<K, V> reversed() {
|
||||
return _backing;
|
||||
}
|
||||
}
|
||||
@@ -0,0 +1,79 @@
|
||||
package com.usatiuk.dhfs.objects;
|
||||
|
||||
import org.apache.commons.lang3.tuple.Pair;
|
||||
|
||||
public abstract class ReversibleKvIterator<K extends Comparable<K>, V> implements CloseableKvIterator<K, V> {
|
||||
protected boolean _goingForward;
|
||||
|
||||
protected abstract void reverse();
|
||||
|
||||
private void ensureForward() {
|
||||
if (!_goingForward) {
|
||||
reverse();
|
||||
}
|
||||
}
|
||||
|
||||
private void ensureBackward() {
|
||||
if (_goingForward) {
|
||||
reverse();
|
||||
}
|
||||
}
|
||||
|
||||
abstract protected K peekImpl();
|
||||
|
||||
abstract protected void skipImpl();
|
||||
|
||||
abstract protected boolean hasImpl();
|
||||
|
||||
abstract protected Pair<K, V> nextImpl();
|
||||
|
||||
@Override
|
||||
public K peekNextKey() {
|
||||
ensureForward();
|
||||
return peekImpl();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skip() {
|
||||
ensureForward();
|
||||
skipImpl();
|
||||
}
|
||||
|
||||
|
||||
@Override
|
||||
public boolean hasNext() {
|
||||
ensureForward();
|
||||
return hasImpl();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V> next() {
|
||||
ensureForward();
|
||||
return nextImpl();
|
||||
}
|
||||
|
||||
@Override
|
||||
public K peekPrevKey() {
|
||||
ensureBackward();
|
||||
return peekImpl();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<K, V> prev() {
|
||||
ensureBackward();
|
||||
return nextImpl();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasPrev() {
|
||||
ensureBackward();
|
||||
return hasImpl();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skipPrev() {
|
||||
ensureBackward();
|
||||
skipImpl();
|
||||
}
|
||||
|
||||
}
|
||||
@@ -170,6 +170,29 @@ public class CachingObjectPersistentStore {
|
||||
return _delegate.hasNext();
|
||||
}
|
||||
|
||||
@Override
|
||||
public JObjectKey peekPrevKey() {
|
||||
return _delegate.peekPrevKey();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<JObjectKey, JDataVersionedWrapper> prev() {
|
||||
var prev = _delegate.prev();
|
||||
Log.tracev("Caching: {0}", prev);
|
||||
put(prev.getKey(), Optional.of(prev.getValue()));
|
||||
return prev;
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasPrev() {
|
||||
return _delegate.hasPrev();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skipPrev() {
|
||||
_delegate.skipPrev();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<JObjectKey, JDataVersionedWrapper> next() {
|
||||
var next = _delegate.next();
|
||||
|
||||
@@ -3,6 +3,7 @@ package com.usatiuk.dhfs.objects.persistence;
|
||||
import com.google.protobuf.ByteString;
|
||||
import com.usatiuk.dhfs.objects.CloseableKvIterator;
|
||||
import com.usatiuk.dhfs.objects.JObjectKey;
|
||||
import com.usatiuk.dhfs.objects.ReversibleKvIterator;
|
||||
import com.usatiuk.dhfs.supportlib.UninitializedByteBuffer;
|
||||
import io.quarkus.arc.properties.IfBuildProperty;
|
||||
import io.quarkus.logging.Log;
|
||||
@@ -92,7 +93,7 @@ public class LmdbObjectPersistentStore implements ObjectPersistentStore {
|
||||
}
|
||||
}
|
||||
|
||||
private class LmdbKvIterator implements CloseableKvIterator<JObjectKey, ByteString> {
|
||||
private class LmdbKvIterator extends ReversibleKvIterator<JObjectKey, ByteString> {
|
||||
private final Txn<ByteBuffer> _txn = _env.txnRead();
|
||||
private final Cursor<ByteBuffer> _cursor = _db.openCursor(_txn);
|
||||
private boolean _hasNext = false;
|
||||
@@ -101,6 +102,7 @@ public class LmdbObjectPersistentStore implements ObjectPersistentStore {
|
||||
private final MutableObject<Boolean> _closed = new MutableObject<>(false);
|
||||
|
||||
LmdbKvIterator(IteratorStart start, JObjectKey key) {
|
||||
_goingForward = true;
|
||||
var closedRef = _closed;
|
||||
CLEANER.register(this, () -> {
|
||||
if (!closedRef.getValue()) {
|
||||
@@ -125,6 +127,9 @@ public class LmdbObjectPersistentStore implements ObjectPersistentStore {
|
||||
switch (start) {
|
||||
case LT -> {
|
||||
_hasNext = _cursor.prev();
|
||||
if (!_hasNext) {
|
||||
_hasNext = _cursor.first();
|
||||
}
|
||||
}
|
||||
case GT -> {
|
||||
_hasNext = _cursor.next();
|
||||
@@ -136,6 +141,9 @@ public class LmdbObjectPersistentStore implements ObjectPersistentStore {
|
||||
switch (start) {
|
||||
case LT, LE -> {
|
||||
_hasNext = _cursor.prev();
|
||||
if (!_hasNext) {
|
||||
_hasNext = _cursor.first();
|
||||
}
|
||||
}
|
||||
case GT, GE -> {
|
||||
}
|
||||
@@ -147,10 +155,10 @@ public class LmdbObjectPersistentStore implements ObjectPersistentStore {
|
||||
|
||||
switch (start) {
|
||||
case LT -> {
|
||||
assert !_hasNext || realGot.compareTo(key) < 0;
|
||||
// assert !_hasNext || realGot.compareTo(key) < 0;
|
||||
}
|
||||
case LE -> {
|
||||
assert !_hasNext || realGot.compareTo(key) <= 0;
|
||||
// assert !_hasNext || realGot.compareTo(key) <= 0;
|
||||
}
|
||||
case GT -> {
|
||||
assert !_hasNext || realGot.compareTo(key) > 0;
|
||||
@@ -173,28 +181,25 @@ public class LmdbObjectPersistentStore implements ObjectPersistentStore {
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasNext() {
|
||||
return _hasNext;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<JObjectKey, ByteString> next() {
|
||||
if (!_hasNext) {
|
||||
throw new NoSuchElementException("No more elements");
|
||||
protected void reverse() {
|
||||
if (_hasNext) {
|
||||
if (_goingForward) {
|
||||
_hasNext = _cursor.prev();
|
||||
} else {
|
||||
_hasNext = _cursor.next();
|
||||
}
|
||||
} else {
|
||||
if (_goingForward) {
|
||||
_hasNext = _cursor.last();
|
||||
} else {
|
||||
_hasNext = _cursor.first();
|
||||
}
|
||||
}
|
||||
var ret = Pair.of(JObjectKey.fromByteBuffer(_cursor.key()), ByteString.copyFrom(_cursor.val()));
|
||||
_hasNext = _cursor.next();
|
||||
Log.tracev("Read: {0}, hasNext: {1}", ret, _hasNext);
|
||||
return ret;
|
||||
_goingForward = !_goingForward;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skip() {
|
||||
_hasNext = _cursor.next();
|
||||
}
|
||||
|
||||
@Override
|
||||
public JObjectKey peekNextKey() {
|
||||
protected JObjectKey peekImpl() {
|
||||
if (!_hasNext) {
|
||||
throw new NoSuchElementException("No more elements");
|
||||
}
|
||||
@@ -202,6 +207,33 @@ public class LmdbObjectPersistentStore implements ObjectPersistentStore {
|
||||
_cursor.key().flip();
|
||||
return ret;
|
||||
}
|
||||
|
||||
@Override
|
||||
protected void skipImpl() {
|
||||
if (_goingForward)
|
||||
_hasNext = _cursor.next();
|
||||
else
|
||||
_hasNext = _cursor.prev();
|
||||
}
|
||||
|
||||
@Override
|
||||
protected boolean hasImpl() {
|
||||
return _hasNext;
|
||||
}
|
||||
|
||||
@Override
|
||||
protected Pair<JObjectKey, ByteString> nextImpl() {
|
||||
if (!_hasNext) {
|
||||
throw new NoSuchElementException("No more elements");
|
||||
}
|
||||
var ret = Pair.of(JObjectKey.fromByteBuffer(_cursor.key()), ByteString.copyFrom(_cursor.val()));
|
||||
if (_goingForward)
|
||||
_hasNext = _cursor.next();
|
||||
else
|
||||
_hasNext = _cursor.prev();
|
||||
Log.tracev("Read: {0}, hasNext: {1}", ret, _hasNext);
|
||||
return ret;
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
|
||||
@@ -1,10 +1,6 @@
|
||||
package com.usatiuk.dhfs.objects.persistence;
|
||||
|
||||
import com.google.protobuf.ByteString;
|
||||
import com.usatiuk.dhfs.objects.CloseableKvIterator;
|
||||
import com.usatiuk.dhfs.objects.JDataVersionedWrapper;
|
||||
import com.usatiuk.dhfs.objects.JObjectKey;
|
||||
import com.usatiuk.dhfs.objects.ObjectSerializer;
|
||||
import com.usatiuk.dhfs.objects.*;
|
||||
import jakarta.enterprise.context.ApplicationScoped;
|
||||
import jakarta.inject.Inject;
|
||||
import org.apache.commons.lang3.tuple.Pair;
|
||||
@@ -31,48 +27,10 @@ public class SerializingObjectPersistentStore {
|
||||
return delegateStore.readObject(name).map(serializer::deserialize);
|
||||
}
|
||||
|
||||
private class SerializingKvIterator implements CloseableKvIterator<JObjectKey, JDataVersionedWrapper> {
|
||||
private final CloseableKvIterator<JObjectKey, ByteString> _delegate;
|
||||
|
||||
private SerializingKvIterator(IteratorStart start, JObjectKey key) {
|
||||
_delegate = delegateStore.getIterator(start, key);
|
||||
}
|
||||
|
||||
@Override
|
||||
public JObjectKey peekNextKey() {
|
||||
return _delegate.peekNextKey();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void skip() {
|
||||
_delegate.skip();
|
||||
}
|
||||
|
||||
@Override
|
||||
public void close() {
|
||||
_delegate.close();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean hasNext() {
|
||||
return _delegate.hasNext();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Pair<JObjectKey, JDataVersionedWrapper> next() {
|
||||
var next = _delegate.next();
|
||||
return Pair.of(next.getKey(), serializer.deserialize(next.getValue()));
|
||||
}
|
||||
}
|
||||
|
||||
// Returns an iterator with a view of all commited objects
|
||||
// Does not have to guarantee consistent view, snapshots are handled by upper layers
|
||||
public CloseableKvIterator<JObjectKey, JDataVersionedWrapper> getIterator(IteratorStart start, JObjectKey key) {
|
||||
return new SerializingKvIterator(start, key);
|
||||
}
|
||||
|
||||
public CloseableKvIterator<JObjectKey, JDataVersionedWrapper> getIterator(JObjectKey key) {
|
||||
return getIterator(IteratorStart.GE, key);
|
||||
return new MappingKvIterator<>(delegateStore.getIterator(start, key), d -> serializer.deserialize(d));
|
||||
}
|
||||
|
||||
public TxManifestRaw prepareManifest(TxManifestObj<? extends JDataVersionedWrapper> names) {
|
||||
|
||||
@@ -1,5 +1,5 @@
|
||||
package com.usatiuk.dhfs.objects.snapshot;
|
||||
|
||||
interface SnapshotEntry {
|
||||
public interface SnapshotEntry {
|
||||
long whenToRemove();
|
||||
}
|
||||
|
||||
@@ -1,4 +1,4 @@
|
||||
package com.usatiuk.dhfs.objects.snapshot;
|
||||
|
||||
record SnapshotEntryDeleted(long whenToRemove) implements SnapshotEntry {
|
||||
public record SnapshotEntryDeleted(long whenToRemove) implements SnapshotEntry {
|
||||
}
|
||||
|
||||
@@ -2,5 +2,5 @@ package com.usatiuk.dhfs.objects.snapshot;
|
||||
|
||||
import com.usatiuk.dhfs.objects.JDataVersionedWrapper;
|
||||
|
||||
record SnapshotEntryObject(JDataVersionedWrapper data, long whenToRemove) implements SnapshotEntry {
|
||||
public record SnapshotEntryObject(JDataVersionedWrapper data, long whenToRemove) implements SnapshotEntry {
|
||||
}
|
||||
|
||||
@@ -5,7 +5,7 @@ import com.usatiuk.dhfs.objects.JObjectKey;
|
||||
import javax.annotation.Nonnull;
|
||||
import java.util.Comparator;
|
||||
|
||||
record SnapshotKey(JObjectKey key, long version) implements Comparable<SnapshotKey> {
|
||||
public record SnapshotKey(JObjectKey key, long version) implements Comparable<SnapshotKey> {
|
||||
@Override
|
||||
public int compareTo(@Nonnull SnapshotKey o) {
|
||||
return Comparator.comparing(SnapshotKey::key)
|
||||
|
||||
@@ -1,6 +1,10 @@
|
||||
package com.usatiuk.dhfs.objects;
|
||||
|
||||
import org.junit.jupiter.api.Assertions;
|
||||
|
||||
import java.util.Arrays;
|
||||
import java.util.Iterator;
|
||||
import java.util.List;
|
||||
import java.util.concurrent.Callable;
|
||||
import java.util.concurrent.Executors;
|
||||
|
||||
@@ -60,4 +64,16 @@ public abstract class Just {
|
||||
}
|
||||
}
|
||||
|
||||
public static <K> void checkIterator(Iterator<K> it, List<K> expected) {
|
||||
for (var e : expected) {
|
||||
Assertions.assertTrue(it.hasNext());
|
||||
var next = it.next();
|
||||
Assertions.assertEquals(e, next);
|
||||
}
|
||||
}
|
||||
|
||||
@SafeVarargs
|
||||
public static <K> void checkIterator(Iterator<K> it, K... expected) {
|
||||
checkIterator(it, Arrays.asList(expected));
|
||||
}
|
||||
}
|
||||
|
||||
@@ -146,6 +146,11 @@ public class MergingKvIteratorTest {
|
||||
Assertions.assertEquals(pair, mergingIterator.next());
|
||||
}
|
||||
Assertions.assertFalse(mergingIterator.hasNext());
|
||||
Just.checkIterator(mergingIterator.reversed(), Pair.of(5, 6), Pair.of(2, 4), Pair.of(1, 3));
|
||||
Assertions.assertFalse(mergingIterator.reversed().hasNext());
|
||||
Just.checkIterator(mergingIterator, Pair.of(1,3), Pair.of(2, 4), Pair.of(5, 6));
|
||||
Assertions.assertFalse(mergingIterator.hasNext());
|
||||
|
||||
|
||||
var mergingIterator2 = new MergingKvIterator<>("test", IteratorStart.LE, 5, (mS, mK) -> new NavigableMapKvIterator<>(source2, mS, mK), (mS, mK) -> new NavigableMapKvIterator<>(source1, mS, mK));
|
||||
var expected2 = List.of(Pair.of(5, 6));
|
||||
@@ -154,6 +159,16 @@ public class MergingKvIteratorTest {
|
||||
Assertions.assertEquals(pair, mergingIterator2.next());
|
||||
}
|
||||
Assertions.assertFalse(mergingIterator2.hasNext());
|
||||
Just.checkIterator(mergingIterator2.reversed(), Pair.of(5, 6), Pair.of(2, 5), Pair.of(1, 3));
|
||||
Assertions.assertFalse(mergingIterator2.reversed().hasNext());
|
||||
Just.checkIterator(mergingIterator2, Pair.of(1,3), Pair.of(2, 5), Pair.of(5, 6));
|
||||
Assertions.assertFalse(mergingIterator2.hasNext());
|
||||
|
||||
var mergingIterator3 = new MergingKvIterator<>("test", IteratorStart.LE, 5, (mS, mK) -> new NavigableMapKvIterator<>(source1, mS, mK), (mS, mK) -> new NavigableMapKvIterator<>(source2, mS, mK));
|
||||
Assertions.assertEquals(5, mergingIterator3.peekNextKey());
|
||||
Assertions.assertEquals(2, mergingIterator3.peekPrevKey());
|
||||
Assertions.assertEquals(5, mergingIterator3.peekNextKey());
|
||||
Assertions.assertEquals(2, mergingIterator3.peekPrevKey());
|
||||
}
|
||||
|
||||
@Test
|
||||
@@ -180,6 +195,10 @@ public class MergingKvIteratorTest {
|
||||
Assertions.assertEquals(pair, mergingIterator.next());
|
||||
}
|
||||
Assertions.assertFalse(mergingIterator.hasNext());
|
||||
Just.checkIterator(mergingIterator.reversed(), Pair.of(6, 8), Pair.of(5, 6), Pair.of(2, 4), Pair.of(1, 3));
|
||||
Assertions.assertFalse(mergingIterator.reversed().hasNext());
|
||||
Just.checkIterator(mergingIterator, Pair.of(1, 3), Pair.of(2, 4), Pair.of(5, 6), Pair.of(6, 8));
|
||||
Assertions.assertFalse(mergingIterator.hasNext());
|
||||
|
||||
var mergingIterator2 = new MergingKvIterator<>("test", IteratorStart.LE, 5, (mS, mK) -> new NavigableMapKvIterator<>(source2, mS, mK), (mS, mK) -> new NavigableMapKvIterator<>(source1, mS, mK));
|
||||
var expected2 = List.of(Pair.of(5, 6), Pair.of(6, 8));
|
||||
@@ -188,6 +207,12 @@ public class MergingKvIteratorTest {
|
||||
Assertions.assertEquals(pair, mergingIterator2.next());
|
||||
}
|
||||
Assertions.assertFalse(mergingIterator2.hasNext());
|
||||
|
||||
var mergingIterator3 = new MergingKvIterator<>("test", IteratorStart.LE, 5, (mS, mK) -> new NavigableMapKvIterator<>(source1, mS, mK), (mS, mK) -> new NavigableMapKvIterator<>(source2, mS, mK));
|
||||
Assertions.assertEquals(5, mergingIterator3.peekNextKey());
|
||||
Assertions.assertEquals(2, mergingIterator3.peekPrevKey());
|
||||
Assertions.assertEquals(5, mergingIterator3.peekNextKey());
|
||||
Assertions.assertEquals(2, mergingIterator3.peekPrevKey());
|
||||
}
|
||||
|
||||
@Test
|
||||
@@ -264,6 +289,8 @@ public class MergingKvIteratorTest {
|
||||
Assertions.assertEquals(pair, mergingIterator.next());
|
||||
}
|
||||
Assertions.assertFalse(mergingIterator.hasNext());
|
||||
Just.checkIterator(mergingIterator.reversed(), Pair.of(4, 6), Pair.of(3, 5), Pair.of(1, 3));
|
||||
Just.checkIterator(mergingIterator, Pair.of(1, 3), Pair.of(3, 5), Pair.of(4, 6));
|
||||
|
||||
var mergingIterator2 = new MergingKvIterator<>("test", IteratorStart.LE, 2, (mS, mK) -> new NavigableMapKvIterator<>(source2, mS, mK), (mS, mK) -> new NavigableMapKvIterator<>(source1, mS, mK));
|
||||
var expected2 = List.of(Pair.of(1, 4), Pair.of(3, 5), Pair.of(4, 6));
|
||||
|
||||
@@ -0,0 +1,71 @@
|
||||
package com.usatiuk.dhfs.objects;
|
||||
|
||||
import com.usatiuk.dhfs.objects.persistence.IteratorStart;
|
||||
import org.apache.commons.lang3.tuple.Pair;
|
||||
import org.junit.jupiter.api.Assertions;
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.pcollections.TreePMap;
|
||||
|
||||
import java.util.NavigableMap;
|
||||
|
||||
public class NavigableMapKvIteratorTest {
|
||||
private final NavigableMap<Integer, Integer> _testMap1 = TreePMap.<Integer, Integer>empty().plus(1, 2).plus(2, 3).plus(3, 4);
|
||||
|
||||
@Test
|
||||
void test1() {
|
||||
var iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.LE, 3);
|
||||
Just.checkIterator(iterator, Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.LE, 2);
|
||||
Just.checkIterator(iterator, Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.GE, 2);
|
||||
Just.checkIterator(iterator, Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.GT, 2);
|
||||
Just.checkIterator(iterator, Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.LT, 3);
|
||||
Just.checkIterator(iterator, Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.LT, 2);
|
||||
Just.checkIterator(iterator, Pair.of(1, 2), Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.LT, 1);
|
||||
Just.checkIterator(iterator, Pair.of(1, 2), Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.LE, 1);
|
||||
Just.checkIterator(iterator, Pair.of(1, 2), Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.GT, 3);
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.GT, 4);
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.LE, 0);
|
||||
Just.checkIterator(iterator, Pair.of(1, 2), Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
|
||||
iterator = new NavigableMapKvIterator<>(_testMap1, IteratorStart.GE, 2);
|
||||
Assertions.assertTrue(iterator.hasNext());
|
||||
Assertions.assertEquals(2, iterator.peekNextKey());
|
||||
Assertions.assertEquals(1, iterator.peekPrevKey());
|
||||
Assertions.assertEquals(2, iterator.peekNextKey());
|
||||
Assertions.assertEquals(1, iterator.peekPrevKey());
|
||||
Just.checkIterator(iterator.reversed(), Pair.of(1, 2));
|
||||
Just.checkIterator(iterator, Pair.of(1, 2), Pair.of(2, 3), Pair.of(3, 4));
|
||||
Assertions.assertEquals(Pair.of(3, 4), iterator.prev());
|
||||
Assertions.assertEquals(Pair.of(2, 3), iterator.prev());
|
||||
Assertions.assertEquals(Pair.of(2, 3), iterator.next());
|
||||
}
|
||||
|
||||
}
|
||||
@@ -34,4 +34,123 @@ public class PredicateKvIteratorTest {
|
||||
}
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
}
|
||||
|
||||
@Test
|
||||
public void ltTest2() {
|
||||
var source1 = TreePMap.<Integer, Integer>empty().plus(1, 3).plus(3, 5).plus(4, 6);
|
||||
var pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 1),
|
||||
IteratorStart.LT, 1, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 2),
|
||||
IteratorStart.LT, 2, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 4),
|
||||
IteratorStart.LT, 4, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LE, 4),
|
||||
IteratorStart.LE, 4, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
}
|
||||
|
||||
@Test
|
||||
public void ltTest3() {
|
||||
var source1 = TreePMap.<Integer, Integer>empty().plus(1, 3).plus(3, 5).plus(4, 6).plus(5, 7).plus(6, 8);
|
||||
var pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 4),
|
||||
IteratorStart.LT, 4, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6), Pair.of(6, 8));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 5),
|
||||
IteratorStart.LT, 5, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6), Pair.of(6, 8));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 6),
|
||||
IteratorStart.LT, 6, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6), Pair.of(6, 8));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 7),
|
||||
IteratorStart.LT, 7, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(6, 8));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 8),
|
||||
IteratorStart.LT, 8, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(6, 8));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LE, 6),
|
||||
IteratorStart.LE, 6, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(6, 8));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 6),
|
||||
IteratorStart.LT, 6, v -> (v % 2 == 0) ? v : null);
|
||||
Assertions.assertTrue(pit.hasNext());
|
||||
Assertions.assertEquals(4, pit.peekNextKey());
|
||||
Assertions.assertFalse(pit.hasPrev());
|
||||
Assertions.assertEquals(4, pit.peekNextKey());
|
||||
Assertions.assertFalse(pit.hasPrev());
|
||||
Assertions.assertEquals(Pair.of(4, 6), pit.next());
|
||||
Assertions.assertTrue(pit.hasNext());
|
||||
Assertions.assertEquals(6, pit.peekNextKey());
|
||||
Assertions.assertEquals(4, pit.peekPrevKey());
|
||||
Assertions.assertEquals(6, pit.peekNextKey());
|
||||
Assertions.assertEquals(4, pit.peekPrevKey());
|
||||
}
|
||||
|
||||
@Test
|
||||
public void itTest4() {
|
||||
var source1 = TreePMap.<Integer, Integer>empty().plus(1, 3).plus(3, 5).plus(4, 6).plus(5, 8).plus(6, 10);
|
||||
var pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 4),
|
||||
IteratorStart.LT, 4, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6), Pair.of(5, 8), Pair.of(6, 10));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 5),
|
||||
IteratorStart.LT, 5, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(4, 6), Pair.of(5, 8), Pair.of(6, 10));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 6),
|
||||
IteratorStart.LT, 6, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(5, 8), Pair.of(6, 10));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 7),
|
||||
IteratorStart.LT, 7, v -> (v % 2 == 0) ? v : null);
|
||||
Just.checkIterator(pit, Pair.of(6, 10));
|
||||
Assertions.assertFalse(pit.hasNext());
|
||||
|
||||
pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 6),
|
||||
IteratorStart.LT, 6, v -> (v % 2 == 0) ? v : null);
|
||||
Assertions.assertTrue(pit.hasNext());
|
||||
Assertions.assertEquals(5, pit.peekNextKey());
|
||||
Assertions.assertTrue(pit.hasPrev());
|
||||
Assertions.assertEquals(4, pit.peekPrevKey());
|
||||
Assertions.assertEquals(5, pit.peekNextKey());
|
||||
Assertions.assertEquals(4, pit.peekPrevKey());
|
||||
Assertions.assertEquals(Pair.of(5, 8), pit.next());
|
||||
Assertions.assertTrue(pit.hasNext());
|
||||
Assertions.assertEquals(6, pit.peekNextKey());
|
||||
Assertions.assertEquals(5, pit.peekPrevKey());
|
||||
Assertions.assertEquals(6, pit.peekNextKey());
|
||||
Assertions.assertEquals(5, pit.peekPrevKey());
|
||||
}
|
||||
|
||||
// @Test
|
||||
// public void reverseTest() {
|
||||
// var source1 = TreePMap.<Integer, Integer>empty().plus(1, 3).plus(3, 5).plus(4, 6);
|
||||
// var pit = new PredicateKvIterator<>(new NavigableMapKvIterator<>(source1, IteratorStart.LT, 4),
|
||||
// IteratorStart.LT, 4, v -> (v % 2 == 0) ? v : null);
|
||||
//
|
||||
// }
|
||||
}
|
||||
|
||||
@@ -0,0 +1,30 @@
|
||||
package com.usatiuk.dhfs.objects;
|
||||
|
||||
import io.quarkus.test.junit.QuarkusTestProfile;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.nio.file.Files;
|
||||
import java.nio.file.Path;
|
||||
import java.util.HashMap;
|
||||
import java.util.Map;
|
||||
|
||||
public class TempDataProfile implements QuarkusTestProfile {
|
||||
protected void getConfigOverrides(Map<String, String> toPut) {
|
||||
}
|
||||
|
||||
@Override
|
||||
final public Map<String, String> getConfigOverrides() {
|
||||
Path tempDirWithPrefix;
|
||||
try {
|
||||
tempDirWithPrefix = Files.createTempDirectory("dhfs-test");
|
||||
} catch (IOException e) {
|
||||
throw new RuntimeException(e);
|
||||
}
|
||||
var ret = new HashMap<String, String>();
|
||||
ret.put("dhfs.objects.persistence.files.root", tempDirWithPrefix.resolve("dhfs_root_test").toString());
|
||||
ret.put("dhfs.fuse.root", tempDirWithPrefix.resolve("dhfs_fuse_root_test").toString());
|
||||
ret.put("dhfs.objects.persistence", "lmdb");
|
||||
getConfigOverrides(ret);
|
||||
return ret;
|
||||
}
|
||||
}
|
||||
@@ -0,0 +1,40 @@
|
||||
package com.usatiuk.dhfs.objects;
|
||||
|
||||
import io.quarkus.logging.Log;
|
||||
import io.quarkus.runtime.ShutdownEvent;
|
||||
import io.quarkus.runtime.StartupEvent;
|
||||
import jakarta.annotation.Priority;
|
||||
import jakarta.enterprise.context.ApplicationScoped;
|
||||
import jakarta.enterprise.event.Observes;
|
||||
import org.eclipse.microprofile.config.inject.ConfigProperty;
|
||||
|
||||
import java.io.File;
|
||||
import java.io.IOException;
|
||||
import java.nio.file.Path;
|
||||
import java.util.Objects;
|
||||
|
||||
@ApplicationScoped
|
||||
public class TestDataCleaner {
|
||||
@ConfigProperty(name = "dhfs.objects.persistence.files.root")
|
||||
String tempDirectory;
|
||||
|
||||
void init(@Observes @Priority(1) StartupEvent event) throws IOException {
|
||||
try {
|
||||
purgeDirectory(Path.of(tempDirectory).toFile());
|
||||
} catch (Exception ignored) {
|
||||
Log.warn("Couldn't cleanup test data on init");
|
||||
}
|
||||
}
|
||||
|
||||
void shutdown(@Observes @Priority(1000000000) ShutdownEvent event) throws IOException {
|
||||
purgeDirectory(Path.of(tempDirectory).toFile());
|
||||
}
|
||||
|
||||
void purgeDirectory(File dir) {
|
||||
for (File file : Objects.requireNonNull(dir.listFiles())) {
|
||||
if (file.isDirectory())
|
||||
purgeDirectory(file);
|
||||
file.delete();
|
||||
}
|
||||
}
|
||||
}
|
||||
@@ -0,0 +1,106 @@
|
||||
package com.usatiuk.dhfs.objects.persistence;
|
||||
|
||||
|
||||
import com.google.protobuf.ByteString;
|
||||
import com.usatiuk.dhfs.objects.JObjectKey;
|
||||
import com.usatiuk.dhfs.objects.Just;
|
||||
import com.usatiuk.dhfs.objects.TempDataProfile;
|
||||
import io.quarkus.test.junit.QuarkusTest;
|
||||
import io.quarkus.test.junit.TestProfile;
|
||||
import jakarta.inject.Inject;
|
||||
import org.apache.commons.lang3.tuple.Pair;
|
||||
import org.junit.jupiter.api.Assertions;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.List;
|
||||
|
||||
@QuarkusTest
|
||||
@TestProfile(TempDataProfile.class)
|
||||
public class LmdbKvIteratorTest {
|
||||
|
||||
@Inject
|
||||
LmdbObjectPersistentStore store;
|
||||
|
||||
@Test
|
||||
public void iteratorTest1() {
|
||||
store.commitTx(
|
||||
new TxManifestRaw(
|
||||
List.of(Pair.of(JObjectKey.of(Long.toString(1)), ByteString.copyFrom(new byte[]{2})),
|
||||
Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})),
|
||||
Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4}))),
|
||||
List.of()
|
||||
)
|
||||
);
|
||||
|
||||
var iterator = store.getIterator(IteratorStart.LE, JObjectKey.of(Long.toString(3)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.LE, JObjectKey.of(Long.toString(2)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.GE, JObjectKey.of(Long.toString(2)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.GT, JObjectKey.of(Long.toString(2)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.LT, JObjectKey.of(Long.toString(3)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.LT, JObjectKey.of(Long.toString(2)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(1)), ByteString.copyFrom(new byte[]{2})), Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.LT, JObjectKey.of(Long.toString(1)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(1)), ByteString.copyFrom(new byte[]{2})), Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.LE, JObjectKey.of(Long.toString(1)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(1)), ByteString.copyFrom(new byte[]{2})), Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.GT, JObjectKey.of(Long.toString(3)));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.GT, JObjectKey.of(Long.toString(4)));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.LE, JObjectKey.of(Long.toString(0)));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(1)), ByteString.copyFrom(new byte[]{2})), Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertFalse(iterator.hasNext());
|
||||
iterator.close();
|
||||
|
||||
iterator = store.getIterator(IteratorStart.GE, JObjectKey.of(Long.toString(2)));
|
||||
Assertions.assertTrue(iterator.hasNext());
|
||||
Assertions.assertEquals(JObjectKey.of(Long.toString(2)), iterator.peekNextKey());
|
||||
Assertions.assertEquals(JObjectKey.of(Long.toString(1)), iterator.peekPrevKey());
|
||||
Assertions.assertEquals(JObjectKey.of(Long.toString(2)), iterator.peekNextKey());
|
||||
Assertions.assertEquals(JObjectKey.of(Long.toString(1)), iterator.peekPrevKey());
|
||||
Just.checkIterator(iterator.reversed(), Pair.of(JObjectKey.of(Long.toString(1)), ByteString.copyFrom(new byte[]{2})));
|
||||
Just.checkIterator(iterator, Pair.of(JObjectKey.of(Long.toString(1)), ByteString.copyFrom(new byte[]{2})), Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})));
|
||||
Assertions.assertEquals(Pair.of(JObjectKey.of(Long.toString(3)), ByteString.copyFrom(new byte[]{4})), iterator.prev());
|
||||
Assertions.assertEquals(Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), iterator.prev());
|
||||
Assertions.assertEquals(Pair.of(JObjectKey.of(Long.toString(2)), ByteString.copyFrom(new byte[]{3})), iterator.next());
|
||||
iterator.close();
|
||||
|
||||
store.commitTx(new TxManifestRaw(
|
||||
List.of(),
|
||||
List.of(JObjectKey.of(Long.toString(1)), JObjectKey.of(Long.toString(2)), JObjectKey.of(Long.toString(3)))
|
||||
));
|
||||
}
|
||||
}
|
||||
@@ -0,0 +1,11 @@
|
||||
package com.usatiuk.dhfs.objects.snapshot;
|
||||
|
||||
import com.usatiuk.dhfs.objects.JObjectKey;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Map;
|
||||
|
||||
public class SnapshotKvIteratorTest {
|
||||
|
||||
|
||||
}
|
||||
Reference in New Issue
Block a user