CavalliumDBEngine/src/main/java/it/cavallium/dbengine/lucene/directory/RocksdbFileStore.java

819 lines
23 KiB
Java
Raw Normal View History

2022-02-26 22:51:22 +01:00
package it.cavallium.dbengine.lucene.directory;
2022-02-28 00:40:17 +01:00
import com.google.common.primitives.Longs;
import com.google.common.util.concurrent.Striped;
2022-10-02 03:09:50 +02:00
import io.netty5.buffer.Buffer;
import io.netty5.buffer.BufferAllocator;
import io.netty5.buffer.BufferComponent;
2022-03-20 14:33:27 +01:00
import it.cavallium.dbengine.database.LLUtils;
2022-02-26 22:51:22 +01:00
import java.io.IOException;
import java.nio.ByteBuffer;
import java.nio.charset.StandardCharsets;
import java.nio.file.Files;
import java.nio.file.Path;
2022-02-28 00:40:17 +01:00
import java.time.Duration;
2022-02-26 22:51:22 +01:00
import java.util.ArrayList;
2022-02-28 00:40:17 +01:00
import java.util.Arrays;
import java.util.Collections;
import java.util.HashMap;
2022-02-26 22:51:22 +01:00
import java.util.List;
2022-02-28 00:40:17 +01:00
import java.util.Map;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.atomic.AtomicLong;
2022-02-26 22:51:22 +01:00
import java.util.concurrent.locks.ReadWriteLock;
2022-02-28 00:40:17 +01:00
import org.apache.lucene.store.AlreadyClosedException;
2022-02-26 22:51:22 +01:00
import org.jetbrains.annotations.Nullable;
2022-03-09 02:29:38 +01:00
import org.rocksdb.ClockCache;
2022-02-26 22:51:22 +01:00
import org.rocksdb.ColumnFamilyDescriptor;
import org.rocksdb.ColumnFamilyHandle;
2022-02-28 00:40:17 +01:00
import org.rocksdb.ColumnFamilyOptions;
import org.rocksdb.CompressionType;
2022-02-26 22:51:22 +01:00
import org.rocksdb.DBOptions;
2022-02-28 03:20:24 +01:00
import org.rocksdb.DirectSlice;
2022-02-28 00:40:17 +01:00
import org.rocksdb.InfoLogLevel;
import org.rocksdb.Options;
2022-02-26 22:51:22 +01:00
import org.rocksdb.ReadOptions;
import org.rocksdb.RocksDB;
import org.rocksdb.RocksDBException;
import org.rocksdb.RocksIterator;
2022-02-28 00:40:17 +01:00
import org.rocksdb.WALRecoveryMode;
2022-02-26 22:51:22 +01:00
import org.rocksdb.WriteOptions;
2022-02-28 00:40:17 +01:00
import org.rocksdb.util.SizeUnit;
2022-02-26 22:51:22 +01:00
public class RocksdbFileStore {
2022-02-28 00:40:17 +01:00
private static final byte[] NEXT_ID_KEY = new byte[]{0x0};
private static final String DEFAULT_COLUMN_FAMILY_STRING = new String(RocksDB.DEFAULT_COLUMN_FAMILY, StandardCharsets.US_ASCII);
2022-02-26 22:51:22 +01:00
static {
RocksDB.loadLibrary();
}
2022-02-28 00:40:17 +01:00
@SuppressWarnings("UnstableApiUsage")
private final Striped<ReadWriteLock> metaLock;
2022-03-08 02:12:13 +01:00
private final ReadWriteLock[] readWriteLocks;
2022-02-28 00:40:17 +01:00
2022-02-28 03:20:24 +01:00
private static final ReadOptions DEFAULT_READ_OPTS = new ReadOptions()
.setVerifyChecksums(false)
.setIgnoreRangeDeletions(true);
private final ReadOptions itReadOpts;
private static final WriteOptions DEFAULT_WRITE_OPTS = new WriteOptions().setDisableWAL(true);
2022-02-26 22:51:22 +01:00
private static final ByteBuffer EMPTY_BYTE_BUF = ByteBuffer.allocateDirect(0);
private final RocksDB db;
2022-03-09 02:29:38 +01:00
public final BufferAllocator bufferAllocator;
2022-02-28 00:40:17 +01:00
private final int blockSize;
private final ColumnFamilyHandle headers;
private final ColumnFamilyHandle filename;
private final ColumnFamilyHandle size;
2022-02-26 22:51:22 +01:00
private final ColumnFamilyHandle data;
2022-02-28 00:40:17 +01:00
private final ConcurrentHashMap<String, Long> filenameToId = new ConcurrentHashMap<>();
private final AtomicLong nextId;
2022-03-08 02:12:13 +01:00
private final boolean closeDbOnClose;
2022-02-28 00:40:17 +01:00
private volatile boolean closed;
private RocksdbFileStore(RocksDB db,
2022-03-09 02:29:38 +01:00
BufferAllocator bufferAllocator,
2022-02-28 00:40:17 +01:00
ColumnFamilyHandle headers,
ColumnFamilyHandle filename,
ColumnFamilyHandle size,
ColumnFamilyHandle data,
int blockSize,
2022-03-08 02:12:13 +01:00
Striped<ReadWriteLock> metaLock,
boolean closeDbOnClose) throws IOException {
2022-02-28 00:40:17 +01:00
try {
this.db = db;
2022-03-09 02:29:38 +01:00
this.bufferAllocator = bufferAllocator;
2022-03-08 02:12:13 +01:00
this.closeDbOnClose = closeDbOnClose;
2022-02-28 00:40:17 +01:00
this.blockSize = blockSize;
this.headers = headers;
this.filename = filename;
this.size = size;
this.data = data;
this.metaLock = metaLock;
2022-03-08 02:12:13 +01:00
ReadWriteLock[] locks = new ReadWriteLock[metaLock.size()];
for (int i = 0; i < metaLock.size(); i++) {
locks[i] = metaLock.getAt(i);
}
this.readWriteLocks = locks;
2022-02-28 00:40:17 +01:00
byte[] nextIdBytes = db.get(headers, NEXT_ID_KEY);
if (nextIdBytes != null) {
this.nextId = new AtomicLong(Longs.fromByteArray(nextIdBytes));
} else {
this.nextId = new AtomicLong();
incFlush();
db.put(headers, NEXT_ID_KEY, Longs.toByteArray(100));
incFlush();
}
2022-03-20 14:33:27 +01:00
this.itReadOpts = new ReadOptions();
if (LLUtils.MANUAL_READAHEAD) {
itReadOpts.setReadaheadSize(blockSize * 4L);
}
itReadOpts.setVerifyChecksums(false)
2022-02-28 03:20:24 +01:00
.setIgnoreRangeDeletions(true);
2022-02-28 00:40:17 +01:00
} catch (RocksDBException e) {
throw new IOException("Failed to open RocksDB meta file store", e);
}
}
2022-02-26 22:51:22 +01:00
2022-03-09 02:29:38 +01:00
private static ByteBuffer readableNioBuffer(Buffer buffer) {
assert buffer.countReadableComponents() == 1 : "Readable components count: " + buffer.countReadableComponents();
2022-10-02 03:09:50 +02:00
return ((BufferComponent) buffer).readableBuffer();
2022-03-09 02:29:38 +01:00
}
private static ByteBuffer writableNioBuffer(Buffer buffer, int newWriterOffset) {
assert buffer.countWritableComponents() == 1 : "Writable components count: " + buffer.countWritableComponents();
buffer.writerOffset(0).ensureWritable(newWriterOffset);
2022-10-02 03:09:50 +02:00
var byteBuf = ((BufferComponent) buffer).writableBuffer();
2022-03-09 02:29:38 +01:00
buffer.writerOffset(newWriterOffset);
assert buffer.capacity() >= newWriterOffset : "Returned capacity " + buffer.capacity() + " < " + newWriterOffset;
return byteBuf;
}
2022-02-28 00:40:17 +01:00
private static DBOptions getDBOptions() {
2022-02-26 22:51:22 +01:00
var options = new DBOptions();
2022-03-12 02:55:18 +01:00
options.setParanoidChecks(false);
2022-02-28 00:40:17 +01:00
options.setWalSizeLimitMB(256);
options.setMaxWriteBatchGroupSizeBytes(2 * SizeUnit.MB);
2022-03-09 02:29:38 +01:00
//options.setAtomicFlush(false);
2022-02-28 00:40:17 +01:00
options.setWalRecoveryMode(WALRecoveryMode.PointInTimeRecovery);
options.setCreateMissingColumnFamilies(true);
2022-02-26 22:51:22 +01:00
options.setCreateIfMissing(true);
2022-03-08 02:12:13 +01:00
//options.setUnorderedWrite(true);
2022-02-28 00:40:17 +01:00
options.setAvoidUnnecessaryBlockingIO(true);
options.setSkipCheckingSstFileSizesOnDbOpen(true);
options.setInfoLogLevel(InfoLogLevel.ERROR_LEVEL);
2022-03-09 02:29:38 +01:00
//options.setAllowMmapReads(true);
//options.setAllowMmapWrites(true);
options.setUseDirectReads(true);
options.setUseDirectIoForFlushAndCompaction(true);
2022-02-28 00:40:17 +01:00
options.setIncreaseParallelism(Runtime.getRuntime().availableProcessors());
options.setDeleteObsoleteFilesPeriodMicros(Duration.ofMinutes(15).toNanos() / 1000L);
2022-03-09 02:29:38 +01:00
options.setRowCache(new ClockCache(512 * 1024 * 1024L));
options.setMaxOpenFiles(500);
2022-02-28 00:40:17 +01:00
return options;
}
public static ColumnFamilyDescriptor getColumnFamilyDescriptor(String name) {
ColumnFamilyOptions opts;
if (name.equals(DEFAULT_COLUMN_FAMILY_STRING) || name.endsWith("_headers")) {
opts = new ColumnFamilyOptions()
.setCompressionType(CompressionType.NO_COMPRESSION)
.setTargetFileSizeBase(SizeUnit.KB);
} else if (name.endsWith("_filename")) {
opts = new ColumnFamilyOptions()
.setCompressionType(CompressionType.NO_COMPRESSION)
.setTargetFileSizeBase(32L * SizeUnit.MB);
} else if (name.endsWith("_size")) {
opts = new ColumnFamilyOptions()
.setCompressionType(CompressionType.NO_COMPRESSION)
.setTargetFileSizeBase(32L * SizeUnit.MB);
} else if (name.endsWith("_data")) {
opts = new ColumnFamilyOptions()
.setCompressionType(CompressionType.LZ4_COMPRESSION)
.setTargetFileSizeBase(128L * SizeUnit.MB);
} else {
opts = new ColumnFamilyOptions();
}
return new ColumnFamilyDescriptor(name.getBytes(StandardCharsets.US_ASCII), opts);
}
2022-03-09 02:29:38 +01:00
private static List<ColumnFamilyDescriptor> getColumnFamilyDescriptors(@Nullable String name) {
String headersName, filenameName, sizeName, dataName;
if (name != null) {
headersName = (name + "_headers");
filenameName = (name + "_filename");
sizeName = (name + "_size");
dataName = (name + "_data");
} else {
headersName = DEFAULT_COLUMN_FAMILY_STRING;
filenameName = "filename";
sizeName = "size";
dataName = "data";
}
return List.of(
getColumnFamilyDescriptor(headersName),
getColumnFamilyDescriptor(filenameName),
getColumnFamilyDescriptor(sizeName),
getColumnFamilyDescriptor(dataName)
);
}
public static RocksdbFileStore create(BufferAllocator bufferAllocator,
RocksDB db,
2022-02-28 00:40:17 +01:00
Map<String, ColumnFamilyHandle> existingHandles,
@Nullable String name,
int blockSize,
Striped<ReadWriteLock> metaLock) throws IOException {
List<ColumnFamilyDescriptor> columnFamilyDescriptors = getColumnFamilyDescriptors(name);
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
List<ColumnFamilyHandle> handles = new ArrayList<>(columnFamilyDescriptors.size());
for (ColumnFamilyDescriptor columnFamilyDescriptor : columnFamilyDescriptors) {
var columnFamilyName = new String(columnFamilyDescriptor.getName(), StandardCharsets.US_ASCII);
ColumnFamilyHandle columnFamilyHandle;
if (existingHandles.containsKey(columnFamilyName)) {
columnFamilyHandle = existingHandles.get(columnFamilyName);
} else {
columnFamilyHandle = db.createColumnFamily(columnFamilyDescriptor);
}
handles.add(columnFamilyHandle);
}
return new RocksdbFileStore(db,
2022-03-09 02:29:38 +01:00
bufferAllocator,
2022-02-28 00:40:17 +01:00
handles.get(0),
handles.get(1),
handles.get(2),
handles.get(3),
blockSize,
2022-03-08 02:12:13 +01:00
metaLock,
false
2022-02-28 00:40:17 +01:00
);
} catch (RocksDBException e) {
throw new IOException(e);
}
}
2022-03-09 02:29:38 +01:00
public static RocksdbFileStore create(BufferAllocator bufferAllocator,
Path path,
int blockSize,
Striped<ReadWriteLock> metaLock) throws IOException {
2022-02-28 00:40:17 +01:00
try {
DBOptions options = getDBOptions();
List<ColumnFamilyDescriptor> descriptors = getColumnFamilyDescriptors(null);
if (Files.notExists(path)) {
Files.createDirectories(path);
}
var handles = new ArrayList<ColumnFamilyHandle>(4);
RocksDB db = RocksDB.open(options, path.toString(), descriptors, handles);
return new RocksdbFileStore(db,
2022-03-09 02:29:38 +01:00
bufferAllocator,
2022-02-28 00:40:17 +01:00
handles.get(0),
handles.get(1),
handles.get(2),
handles.get(3),
blockSize,
2022-03-08 02:12:13 +01:00
metaLock,
true
2022-02-26 22:51:22 +01:00
);
} catch (RocksDBException e) {
throw new IOException("Failed to open RocksDB meta file store", e);
}
}
2022-02-28 00:40:17 +01:00
public static RocksDBInstance createEmpty(Path path) throws IOException {
try {
DBOptions options = getDBOptions();
List<ColumnFamilyDescriptor> descriptors;
if (Files.exists(path)) {
descriptors = RocksDB
.listColumnFamilies(new Options(), path.toString())
.stream()
.map(nameBytes -> {
var name = new String(nameBytes, StandardCharsets.US_ASCII);
return getColumnFamilyDescriptor(name);
})
.toList();
} else {
descriptors = List.of(new ColumnFamilyDescriptor(RocksDB.DEFAULT_COLUMN_FAMILY));
}
if (Files.notExists(path)) {
Files.createDirectories(path);
}
var handles = new ArrayList<ColumnFamilyHandle>(descriptors.size());
RocksDB db = RocksDB.open(options, path.toString(), descriptors, handles);
var handlesMap = new HashMap<String, ColumnFamilyHandle>();
for (int i = 0; i < handles.size(); i++) {
var name = new String(descriptors.get(i).getName(), StandardCharsets.US_ASCII);
handlesMap.put(name, handles.get(i));
}
return new RocksDBInstance(db, Collections.unmodifiableMap(handlesMap));
} catch (RocksDBException e) {
throw new IOException("Failed to open RocksDB meta file store", e);
}
}
private long getFileId(String key) throws RocksDBException, IOException {
Long id = filenameToId.get(key);
if (id != null) {
return id;
} else {
2022-03-09 02:29:38 +01:00
try (var filenameKey = getFilenameKey(key); var filenameValue = getFilenameValue()) {
if (db.get(filename, DEFAULT_READ_OPTS, readableNioBuffer(filenameKey), writableNioBuffer(filenameValue, Long.BYTES))
2022-02-28 00:40:17 +01:00
== RocksDB.NOT_FOUND) {
throw new IOException("File not found: " + key);
}
2022-03-09 02:29:38 +01:00
filenameValue.writerOffset(Long.BYTES);
return filenameValue.readLong();
2022-02-28 00:40:17 +01:00
}
}
}
@Nullable
private Long getFileIdOrNull(String key) throws RocksDBException {
Long id = filenameToId.get(key);
if (id != null) {
return id;
} else {
2022-03-09 02:29:38 +01:00
try (var filenameKey = getFilenameKey(key); var filenameValue = getFilenameValue()) {
if (db.get(filename, DEFAULT_READ_OPTS, readableNioBuffer(filenameKey), writableNioBuffer(filenameValue, Long.BYTES))
2022-02-28 00:40:17 +01:00
== RocksDB.NOT_FOUND) {
return null;
}
2022-03-09 02:29:38 +01:00
filenameValue.writerOffset(Long.BYTES);
return filenameValue.readLong();
2022-02-28 00:40:17 +01:00
}
}
}
private boolean containsFileId(String key) throws RocksDBException {
Long id = filenameToId.get(key);
if (id != null) {
return true;
} else {
2022-03-09 02:29:38 +01:00
try (var filenameKey = getFilenameKey(key)) {
if (db.keyMayExist(filename, DEFAULT_READ_OPTS, readableNioBuffer(filenameKey))) {
return db.get(filename, DEFAULT_READ_OPTS, readableNioBuffer(filenameKey), EMPTY_BYTE_BUF) != RocksDB.NOT_FOUND;
2022-02-28 00:40:17 +01:00
} else {
return false;
}
}
}
}
2022-02-26 22:51:22 +01:00
2022-02-28 00:40:17 +01:00
private void moveFileId(long id, String oldKey, String newKey) throws RocksDBException {
var filenameValue = getFilenameValue();
2022-03-09 02:29:38 +01:00
filenameValue.writeLong(id);
try (var filenameOldKey = getFilenameKey(oldKey); var filenameNewKey = getFilenameKey(newKey); filenameValue) {
db.delete(filename, DEFAULT_WRITE_OPTS, readableNioBuffer(filenameOldKey));
2022-02-28 00:40:17 +01:00
incFlush();
2022-03-09 02:29:38 +01:00
db.put(filename, DEFAULT_WRITE_OPTS, readableNioBuffer(filenameNewKey), readableNioBuffer(filenameValue));
2022-02-28 00:40:17 +01:00
incFlush();
}
}
private void incFlush() throws RocksDBException {
/*
if ((flushCounter.incrementAndGet() % 1) == 0) {
db.flushWal(false);
}
*/
}
private long getFileIdOrAllocate(String key) throws RocksDBException {
Long id = filenameToId.get(key);
if (id != null) {
return id;
} else {
2022-03-09 02:29:38 +01:00
try (var filenameKey = getFilenameKey(key); var filenameValue = getFilenameValue()) {
if (db.get(filename, DEFAULT_READ_OPTS, readableNioBuffer(filenameKey),
writableNioBuffer(filenameValue, Long.BYTES))
2022-02-28 00:40:17 +01:00
== RocksDB.NOT_FOUND) {
2022-03-09 02:29:38 +01:00
filenameValue.writerOffset(0);
filenameValue.readerOffset(0);
2022-02-28 00:40:17 +01:00
var newlyAllocatedId = this.nextId.getAndIncrement();
if (newlyAllocatedId % 100 == 99) {
2022-03-09 02:29:38 +01:00
db.put(headers, new byte[]{0x00}, Longs.toByteArray(newlyAllocatedId + 1 + 100));
2022-02-28 00:40:17 +01:00
incFlush();
}
2022-03-09 02:29:38 +01:00
filenameValue.writeLong(newlyAllocatedId);
2022-02-28 00:40:17 +01:00
db.put(filename,
DEFAULT_WRITE_OPTS,
2022-03-09 02:29:38 +01:00
readableNioBuffer(filenameKey),
readableNioBuffer(filenameValue)
2022-02-28 00:40:17 +01:00
);
incFlush();
filenameToId.put(key, newlyAllocatedId);
return newlyAllocatedId;
}
2022-03-09 02:29:38 +01:00
filenameValue.readerOffset(0);
filenameValue.writerOffset(Long.BYTES);
return filenameValue.readLong();
2022-02-26 22:51:22 +01:00
}
2022-02-28 00:40:17 +01:00
}
}
private void dellocateFilename(String key) throws RocksDBException {
2022-03-09 02:29:38 +01:00
try (var filenameKey = getFilenameKey(key)) {
db.delete(filename, DEFAULT_WRITE_OPTS, readableNioBuffer(filenameKey));
2022-02-28 00:40:17 +01:00
filenameToId.remove(key);
}
}
public boolean contains(String key) throws RocksDBException, IOException {
var l = metaLock.get(key).readLock();
l.lock();
try {
ensureOpen();
return containsFileId(key);
} finally {
l.unlock();
2022-02-26 22:51:22 +01:00
}
}
2022-03-09 02:29:38 +01:00
private Buffer getMetaValueBuf() {
return bufferAllocator.allocate(Long.BYTES);
2022-02-26 22:51:22 +01:00
}
2022-03-09 02:29:38 +01:00
private Buffer getDataValueBuf() {
return bufferAllocator.allocate(blockSize);
2022-02-28 00:40:17 +01:00
}
2022-03-09 02:29:38 +01:00
private Buffer getFilenameValue() {
return bufferAllocator.allocate(Long.BYTES);
2022-02-28 00:40:17 +01:00
}
2022-03-09 02:29:38 +01:00
private Buffer getMetaKey(long id) {
Buffer buf = bufferAllocator.allocate(Long.BYTES);
buf.writeLong(id);
2022-02-28 00:40:17 +01:00
return buf;
2022-02-26 22:51:22 +01:00
}
2022-03-09 02:29:38 +01:00
private Buffer getFilenameKey(String key) {
Buffer buf = bufferAllocator.allocate(key.length());
2022-02-26 22:51:22 +01:00
buf.writeCharSequence(key, StandardCharsets.US_ASCII);
return buf;
}
2022-03-09 02:29:38 +01:00
private Buffer getDataKey(@Nullable Buffer buf, long id, int i) {
2022-02-26 22:51:22 +01:00
if (buf == null) {
2022-03-09 02:29:38 +01:00
buf = bufferAllocator.allocate(Long.BYTES + Integer.BYTES);
2022-02-26 22:51:22 +01:00
}
2022-03-09 02:29:38 +01:00
buf.writeLong(id);
2022-02-28 00:40:17 +01:00
buf.writeInt(i);
2022-02-26 22:51:22 +01:00
return buf;
}
2022-03-09 02:29:38 +01:00
private Buffer getDataKeyPrefix(long id) {
var buf = bufferAllocator.allocate(Long.BYTES);
buf.writeLong(id);
2022-02-28 03:20:24 +01:00
return buf;
}
2022-02-28 00:40:17 +01:00
private byte[] getDataKeyByteArray(long id, int i) {
ByteBuffer bb = ByteBuffer.wrap(new byte[Long.BYTES + Integer.BYTES]);
bb.putLong(id);
bb.putInt(i);
return bb.array();
}
2022-02-26 22:51:22 +01:00
2022-03-09 02:29:38 +01:00
public int load(String name, long position, Buffer buf, int offset, int len) throws IOException {
2022-02-28 00:40:17 +01:00
var l = metaLock.get(name).readLock();
l.lock();
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
ensureOpen();
Long fileId = getFileIdOrNull(name);
if (fileId == null) {
return -1;
}
long size = getSizeInternal(fileId);
2022-02-26 22:51:22 +01:00
if (position >= size) {
return -1;
}
2022-02-28 03:20:24 +01:00
if (buf.capacity() < offset + len) {
2022-02-26 22:51:22 +01:00
throw new IllegalArgumentException("len is too long");
}
long p = position;
int f = offset;
int n = len;
2022-03-09 02:29:38 +01:00
Buffer valBuf = getDataValueBuf();
try (valBuf) {
ByteBuffer valBuffer = writableNioBuffer(valBuf, blockSize);
2022-02-28 03:20:24 +01:00
boolean shouldSeekTo = true;
try (var ro = new ReadOptions(itReadOpts)) {
ro.setIgnoreRangeDeletions(true);
2022-03-09 02:29:38 +01:00
try (Buffer fileIdPrefix = getDataKeyPrefix(fileId)) {
try (var lb = new DirectSlice(readableNioBuffer(fileIdPrefix), Long.BYTES)) {
2022-02-28 03:20:24 +01:00
ro.setIterateLowerBound(lb);
ro.setPrefixSameAsStart(true);
try (RocksIterator it = db.newIterator(data, itReadOpts)) {
int m;
int r;
int i;
do {
m = (int) (p % (long) blockSize);
r = Math.min(blockSize - m, n);
i = (int) (p / (long) blockSize);
//System.out.println("Reading block " + name + "(" + fileId + "):" + i);
if (shouldSeekTo) {
shouldSeekTo = false;
2022-03-09 02:29:38 +01:00
try (Buffer dataKey = getDataKey(null, fileId, i)) {
it.seek(readableNioBuffer(dataKey));
2022-02-28 03:20:24 +01:00
}
if (!it.isValid()) {
throw new IOException("Block " + name + "(" + fileId + ")" + ":" + i + " not found");
}
} else {
it.next();
if (!it.isValid()) {
throw new IOException("Block " + name + "(" + fileId + ")" + ":" + i + " not found");
}
}
assert Arrays.equals(getDataKeyByteArray(fileId, i), it.key());
int dataRead = it.value(valBuffer);
2022-03-09 02:29:38 +01:00
valBuf.writerOffset(dataRead);
2022-02-28 03:20:24 +01:00
2022-03-09 02:29:38 +01:00
valBuf.copyInto(m, buf, f, r);
2022-02-28 03:20:24 +01:00
2022-03-09 02:29:38 +01:00
valBuf.writerOffset(0);
valBuf.readerOffset(0);
2022-02-28 03:20:24 +01:00
p += r;
f += r;
n -= r;
} while (n != 0 && p < size);
return (int) (p - position);
}
2022-02-28 00:40:17 +01:00
}
2022-02-26 22:51:22 +01:00
}
2022-02-28 03:20:24 +01:00
}
2022-02-26 22:51:22 +01:00
}
} catch (RocksDBException ex) {
throw new IOException(ex);
} finally {
2022-02-28 00:40:17 +01:00
l.unlock();
2022-02-26 22:51:22 +01:00
}
}
/**
* @return not exist return -1
*/
public long getSize(String key) throws IOException {
2022-02-28 00:40:17 +01:00
var l = metaLock.get(key).readLock();
l.lock();
try {
ensureOpen();
return getSizeInternal(key);
} finally {
l.unlock();
}
}
/**
* @return not exist return -1
*/
private long getSizeInternal(String key) throws IOException {
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
Long fileId = getFileIdOrNull(key);
if (fileId == null) {
return -1;
}
return getSizeInternal(fileId);
} catch (RocksDBException ex) {
throw new IOException(ex);
}
}
/**
* @return not exist return -1
*/
private long getSizeInternal(long fileId) throws IOException {
try {
2022-03-09 02:29:38 +01:00
try (Buffer metaKey = getMetaKey(fileId); Buffer metaData = getMetaValueBuf()) {
if (db.get(size, DEFAULT_READ_OPTS, readableNioBuffer(metaKey), writableNioBuffer(metaData, Long.BYTES))
2022-02-26 22:51:22 +01:00
!= RocksDB.NOT_FOUND) {
2022-03-09 02:29:38 +01:00
metaData.writerOffset(Long.BYTES);
return metaData.readLong();
2022-02-26 22:51:22 +01:00
} else {
return -1;
}
}
} catch (RocksDBException ex) {
throw new IOException(ex);
}
}
public void remove(String key) throws IOException {
2022-02-28 00:40:17 +01:00
var l = metaLock.get(key).writeLock();
l.lock();
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
ensureOpen();
Long fileId = getFileIdOrNull(key);
if (fileId == null) {
return;
}
long size;
size = getSizeInternal(fileId);
2022-02-26 22:51:22 +01:00
if (size == -1) {
return;
}
2022-03-09 02:29:38 +01:00
Buffer dataKey = null;
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
int n = (int) ((size + blockSize - 1) / blockSize);
if (n == 1) {
dataKey = getDataKey(dataKey, fileId, 0);
2022-03-09 02:29:38 +01:00
db.delete(data, DEFAULT_WRITE_OPTS, readableNioBuffer(dataKey));
2022-02-28 00:40:17 +01:00
} else if (n > 1) {
var dataKey1 = getDataKeyByteArray(fileId, 0);
var dataKey2 = getDataKeyByteArray(fileId, n - 1);
db.deleteRange(data, DEFAULT_WRITE_OPTS, dataKey1, dataKey2);
2022-02-26 22:51:22 +01:00
}
2022-03-09 02:29:38 +01:00
try (Buffer metaKey = getMetaKey(fileId)) {
2022-02-28 00:40:17 +01:00
dellocateFilename(key);
2022-03-09 02:29:38 +01:00
db.delete(this.size, DEFAULT_WRITE_OPTS, readableNioBuffer(metaKey));
2022-02-26 22:51:22 +01:00
}
} finally {
if (dataKey != null) {
2022-03-09 02:29:38 +01:00
dataKey.close();
2022-02-26 22:51:22 +01:00
}
}
} catch (RocksDBException ex) {
throw new IOException(ex);
} finally {
2022-02-28 00:40:17 +01:00
l.unlock();
2022-02-26 22:51:22 +01:00
}
}
public void clear() throws IOException {
2022-03-08 02:12:13 +01:00
for (var lock : readWriteLocks) {
lock.writeLock().lock();
2022-02-28 00:40:17 +01:00
}
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
ensureOpen();
List<String> keySet = listKeyInternal();
2022-02-26 22:51:22 +01:00
for (String key : keySet) {
remove(key);
}
} finally {
2022-03-08 02:12:13 +01:00
for (var lock : readWriteLocks) {
lock.writeLock().unlock();
2022-02-28 00:40:17 +01:00
}
2022-02-26 22:51:22 +01:00
}
}
public List<String> listKey() {
2022-03-08 02:12:13 +01:00
ensureOpen();
for (var lock : readWriteLocks) {
lock.readLock().lock();
2022-02-28 00:40:17 +01:00
}
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
ensureOpen();
return listKeyInternal();
2022-02-26 22:51:22 +01:00
} finally {
2022-03-08 02:12:13 +01:00
for (var lock : readWriteLocks) {
lock.readLock().unlock();
2022-02-28 00:40:17 +01:00
}
}
}
private List<String> listKeyInternal() {
List<String> keys = new ArrayList<>();
2022-05-04 12:36:32 +02:00
try (RocksIterator iterator = db.newIterator(filename)) {
iterator.seekToFirst();
while (iterator.isValid()) {
keys.add(new String(iterator.key(), StandardCharsets.US_ASCII).intern());
iterator.next();
}
return keys;
2022-02-26 22:51:22 +01:00
}
}
2022-03-09 02:29:38 +01:00
public void append(String name, Buffer buf, int offset, int len) throws IOException {
2022-02-28 00:40:17 +01:00
var l = metaLock.get(name).writeLock();
l.lock();
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
ensureOpen();
long size;
long fileId;
int f;
int n;
size = getSizeInternal(name);
2022-02-26 22:51:22 +01:00
if (size == -1) {
size = 0;
}
2022-02-28 00:40:17 +01:00
f = offset;
n = len;
2022-02-26 22:51:22 +01:00
2022-02-28 00:40:17 +01:00
fileId = getFileIdOrAllocate(name);
2022-03-09 02:29:38 +01:00
Buffer dataKey = null;
Buffer bb = getDataValueBuf();
2022-02-26 22:51:22 +01:00
try {
2022-03-09 02:29:38 +01:00
do {
int m = (int) (size % (long) blockSize);
int r = Math.min(blockSize - m, n);
int i = (int) ((size) / (long) blockSize);
dataKey = getDataKey(dataKey, fileId, i);
if (m != 0) {
int dataRead;
if ((dataRead = db.get(data,
DEFAULT_READ_OPTS,
readableNioBuffer(dataKey),
writableNioBuffer(bb, blockSize)
)) == RocksDB.NOT_FOUND) {
throw new IOException("Block " + name + "(" + fileId + "):" + i + " not found");
2022-02-26 22:51:22 +01:00
}
2022-03-09 02:29:38 +01:00
bb.writerOffset(dataRead);
dataKey.readerOffset(0);
} else {
bb.writerOffset(0);
}
2022-02-26 22:51:22 +01:00
2022-03-09 02:29:38 +01:00
bb.ensureWritable(r);
buf.copyInto(f, bb, m, r);
2022-02-26 22:51:22 +01:00
2022-03-09 02:29:38 +01:00
var bbBuf = writableNioBuffer(bb, m + r);
assert bbBuf.capacity() >= m + r : bbBuf.capacity() + " < " + (m + r);
assert bbBuf.position() == 0;
bbBuf.limit(m + r);
assert bbBuf.limit() == m + r;
db.put(data, DEFAULT_WRITE_OPTS, readableNioBuffer(dataKey), bbBuf);
incFlush();
size += r;
f += r;
n -= r;
dataKey.readerOffset(0);
dataKey.writerOffset(0);
bb.readerOffset(0);
bb.writerOffset(0);
} while (n != 0);
2022-02-26 22:51:22 +01:00
} finally {
if (dataKey != null) {
2022-03-09 02:29:38 +01:00
dataKey.close();
2022-02-26 22:51:22 +01:00
}
2022-03-09 02:29:38 +01:00
bb.close();
2022-02-26 22:51:22 +01:00
}
2022-03-09 02:29:38 +01:00
try (Buffer metaKey = getMetaKey(fileId); Buffer metaValue = getMetaValueBuf()) {
metaValue.writeLong(size);
db.put(this.size, DEFAULT_WRITE_OPTS, readableNioBuffer(metaKey), readableNioBuffer(metaValue));
2022-02-28 00:40:17 +01:00
incFlush();
2022-02-26 22:51:22 +01:00
}
} catch (RocksDBException ex) {
throw new IOException(ex);
} finally {
2022-02-28 00:40:17 +01:00
l.unlock();
2022-02-26 22:51:22 +01:00
}
}
public void move(String source, String dest) throws IOException {
2022-02-28 00:40:17 +01:00
var locks = metaLock.bulkGet(List.of(source, dest));
for (ReadWriteLock lock : locks) {
lock.writeLock().lock();
}
2022-02-26 22:51:22 +01:00
try {
2022-02-28 00:40:17 +01:00
ensureOpen();
long sourceFileId = getFileId(source);
moveFileId(sourceFileId, source, dest);
2022-02-26 22:51:22 +01:00
} catch (RocksDBException ex) {
throw new IOException(ex);
} finally {
2022-02-28 00:40:17 +01:00
for (ReadWriteLock lock : locks) {
lock.writeLock().unlock();
}
2022-02-26 22:51:22 +01:00
}
2022-02-28 00:40:17 +01:00
}
2022-02-26 22:51:22 +01:00
2022-02-28 00:40:17 +01:00
private void ensureOpen() {
if (closed) {
throw new AlreadyClosedException("Index already closed");
}
2022-02-26 22:51:22 +01:00
}
public void close() throws IOException {
2022-02-28 00:40:17 +01:00
if (closed) {
return;
}
2022-03-08 02:12:13 +01:00
for (var lock : readWriteLocks) {
lock.writeLock().lock();
2022-02-28 00:40:17 +01:00
}
try {
if (closed) {
return;
}
closed = true;
2022-03-08 02:12:13 +01:00
if (closeDbOnClose) {
try {
db.closeE();
} catch (RocksDBException e) {
throw new IOException(e);
}
}
2022-02-28 00:40:17 +01:00
} finally {
2022-03-08 02:12:13 +01:00
for (var lock : readWriteLocks) {
lock.writeLock().unlock();
2022-02-28 00:40:17 +01:00
}
}
}
public void sync() throws RocksDBException {
/*
db.flushWal(true);
db.flush(new FlushOptions().setAllowWriteStall(true).setWaitForFlush(true));
*/
2022-02-26 22:51:22 +01:00
}
}