提交 9632595d authored 作者: Thomas Mueller's avatar Thomas Mueller

MVStore: use a write buffer instead of a ByteBuffer

上级 ba877c18
......@@ -120,7 +120,7 @@ public class Chunk {
*
* @param buff the target buffer
*/
void writeHeader(ByteBuffer buff) {
void writeHeader(WriteBuffer buff) {
buff.put((byte) 'c');
buff.putInt(length);
buff.putInt(id);
......
......@@ -181,7 +181,7 @@ public class MVStore {
private HashMap<String, String> storeHeader = New.hashMap();
private ByteBuffer writeBuffer;
private WriteBuffer writeBuffer;
private int lastMapId;
......@@ -269,12 +269,14 @@ public class MVStore {
boolean readOnly = config.containsKey("readOnly");
o = config.get("cacheSize");
int mb = o == null ? 16 : (Integer) o;
int maxMemoryBytes = mb * 1024 * 1024;
int averageMemory = Math.max(10, pageSplitSize / 2);
int segmentCount = 16;
int stackMoveDistance = maxMemoryBytes / averageMemory * 2 / 100;
cache = new CacheLongKeyLIRS<Page>(
maxMemoryBytes, averageMemory, segmentCount, stackMoveDistance);
if (mb > 0) {
int maxMemoryBytes = mb * 1024 * 1024;
int averageMemory = Math.max(10, pageSplitSize / 2);
int segmentCount = 16;
int stackMoveDistance = maxMemoryBytes / averageMemory * 2 / 100;
cache = new CacheLongKeyLIRS<Page>(
maxMemoryBytes, averageMemory, segmentCount, stackMoveDistance);
}
o = config.get("writeBufferSize");
mb = o == null ? 4 : (Integer) o;
int writeBufferSize = mb * 1024 * 1024;
......@@ -519,6 +521,11 @@ public class MVStore {
private void readMeta() {
chunks.clear();
Chunk header = readChunkHeader(rootChunkStart);
if (header.start == Long.MAX_VALUE) {
throw DataUtils.newIllegalStateException(
DataUtils.ERROR_FILE_CORRUPT,
"Chunk {0} is invalid", header.id);
}
lastChunkId = header.id;
chunks.put(header.id, header);
meta.setRootPos(header.metaRootPos, -1);
......@@ -543,6 +550,11 @@ public class MVStore {
s = meta.get(s);
Chunk c = Chunk.fromString(s);
if (!chunks.containsKey(c.id)) {
if (c.start == Long.MAX_VALUE) {
throw DataUtils.newIllegalStateException(
DataUtils.ERROR_FILE_CORRUPT,
"Chunk {0} is invalid", c.id);
}
chunks.put(c.id, c);
}
}
......@@ -731,10 +743,14 @@ public class MVStore {
if (s == null) {
throw DataUtils.newIllegalStateException(
DataUtils.ERROR_FILE_CORRUPT,
"Chunk {0} not found",
DataUtils.getPageChunkId(pos));
"Chunk {0} not found", chunkId);
}
c = Chunk.fromString(s);
if (c.start == Long.MAX_VALUE) {
throw DataUtils.newIllegalStateException(
DataUtils.ERROR_FILE_CORRUPT,
"Chunk {0} is invalid", chunkId);
}
chunks.put(c.id, c);
}
return c;
......@@ -892,7 +908,7 @@ public class MVStore {
}
}
Set<Chunk> removedChunks = applyFreedSpace(storeVersion, time);
ByteBuffer buff = getWriteBuffer();
WriteBuffer buff = getWriteBuffer();
// need to patch the header later
c.writeHeader(buff);
c.maxLength = 0;
......@@ -900,28 +916,24 @@ public class MVStore {
for (MVMap<?, ?> m : changed) {
Page p = m.getRoot();
if (p.getTotalCount() > 0) {
buff = p.writeUnsavedRecursive(c, buff);
p.writeUnsavedRecursive(c, buff);
long root = p.getPos();
meta.put("root." + m.getId(), "" + root);
}
}
meta.put("chunk." + c.id, c.asString());
meta.setWriteVersion(version);
// this will (again) modify maxLengthLive, but
// the correct value is written in the chunk header
Page metaRoot = meta.getRoot();
buff = metaRoot.writeUnsavedRecursive(c, buff);
metaRoot.writeUnsavedRecursive(c, buff);
int chunkLength = buff.position();
// round to the next block,
// and one additional block for the store header
int length = MathUtils.roundUpInt(chunkLength, BLOCK_SIZE) + BLOCK_SIZE;
if (length > buff.capacity()) {
buff = DataUtils.ensureCapacity(buff, length - buff.capacity());
}
buff.limit(length);
// free up the space of unused chunks now
......@@ -939,6 +951,14 @@ public class MVStore {
fileStore.markUsed(end, length);
}
boolean storeAtEndOfFile = filePos + length >= end;
// try {
// Thread.sleep(10);
// } catch (InterruptedException e) {
// ; int todo;
// // TODO Auto-generated catch block
// e.printStackTrace();
// }
c.start = filePos;
c.length = chunkLength;
......@@ -955,7 +975,7 @@ public class MVStore {
buff.put(new byte[BLOCK_SIZE - header.length]);
buff.position(0);
fileStore.writeFully(filePos, buff);
fileStore.writeFully(filePos, buff.getBuffer());
releaseWriteBuffer(buff);
......@@ -975,6 +995,7 @@ public class MVStore {
// some pages might have been changed in the meantime (in the newest version)
unsavedPageCount = Math.max(0, unsavedPageCount - currentUnsavedPageCount);
if (!temp) {
metaChanged = false;
lastStoredVersion = storeVersion;
......@@ -988,13 +1009,13 @@ public class MVStore {
*
* @return the buffer
*/
private ByteBuffer getWriteBuffer() {
ByteBuffer buff;
private WriteBuffer getWriteBuffer() {
WriteBuffer buff;
if (writeBuffer != null) {
buff = writeBuffer;
buff.clear();
} else {
buff = ByteBuffer.allocate(1024 * 1024);
buff = new WriteBuffer();
}
return buff;
}
......@@ -1005,7 +1026,7 @@ public class MVStore {
*
* @param buff the buffer than can be re-used
*/
private void releaseWriteBuffer(ByteBuffer buff) {
private void releaseWriteBuffer(WriteBuffer buff) {
if (buff.capacity() <= 4 * 1024 * 1024) {
writeBuffer = buff;
}
......@@ -1123,9 +1144,6 @@ public class MVStore {
private long getEndPosition() {
long size = 2 * BLOCK_SIZE;
for (Chunk c : chunks.values()) {
if (c.start == Long.MAX_VALUE) {
continue;
}
long x = c.start + c.length;
size = Math.max(size, MathUtils.roundUpLong(x, BLOCK_SIZE) + BLOCK_SIZE);
}
......@@ -1200,9 +1218,8 @@ public class MVStore {
}
}
for (Chunk c : move) {
ByteBuffer buff = getWriteBuffer();
WriteBuffer buff = getWriteBuffer();
int length = MathUtils.roundUpInt(c.length, BLOCK_SIZE) + BLOCK_SIZE;
buff = DataUtils.ensureCapacity(buff, length);
buff.limit(length);
ByteBuffer buff2 = fileStore.readFully(c.start, length);
buff.put(buff2);
......@@ -1218,7 +1235,7 @@ public class MVStore {
// fill the header with zeroes
buff.put(new byte[BLOCK_SIZE - header.length]);
buff.position(0);
fileStore.writeFully(end, buff);
fileStore.writeFully(end, buff.getBuffer());
releaseWriteBuffer(buff);
meta.put("chunk." + c.id, c.asString());
}
......@@ -1238,9 +1255,8 @@ public class MVStore {
// previous store operation
continue;
}
ByteBuffer buff = getWriteBuffer();
WriteBuffer buff = getWriteBuffer();
int length = MathUtils.roundUpInt(c.length, BLOCK_SIZE) + BLOCK_SIZE;
buff = DataUtils.ensureCapacity(buff, length);
buff.limit(length);
ByteBuffer buff2 = fileStore.readFully(c.start, length);
buff.put(buff2);
......@@ -1255,7 +1271,7 @@ public class MVStore {
// fill the header with zeroes
buff.put(new byte[BLOCK_SIZE - header.length]);
buff.position(0);
fileStore.writeFully(pos, buff);
fileStore.writeFully(pos, buff.getBuffer());
releaseWriteBuffer(buff);
meta.put("chunk." + c.id, c.asString());
}
......@@ -1432,7 +1448,7 @@ public class MVStore {
throw DataUtils.newIllegalStateException(
DataUtils.ERROR_FILE_CORRUPT, "Position 0");
}
Page p = cache.get(pos);
Page p = cache == null ? null : cache.get(pos);
if (p == null) {
Chunk c = getChunk(pos);
long filePos = c.start;
......@@ -1442,7 +1458,9 @@ public class MVStore {
DataUtils.ERROR_FILE_CORRUPT, "Negative position {0}", filePos);
}
p = Page.read(fileStore, map, pos, filePos, fileStore.size());
cache.put(pos, p, p.getMemory());
if (cache != null) {
cache.put(pos, p, p.getMemory());
}
}
return p;
}
......@@ -1466,7 +1484,9 @@ public class MVStore {
// This could result in a cache miss if the operation is rolled back,
// but we don't optimize for rollback.
// We could also keep the page in the cache, as somebody could read it.
cache.remove(pos);
if (cache != null) {
cache.remove(pos);
}
Chunk c = getChunk(pos);
long version = currentVersion;
......@@ -1800,6 +1820,11 @@ public class MVStore {
}
}
// rollback might have rolled back the stored chunk metadata as well
Chunk c = chunks.get(lastChunkId - 1);
if (c != null) {
meta.put("chunk." + c.id, c.asString());
}
currentVersion = version;
setWriteVersion(version);
lastCommittedVersion = version;
......
......@@ -770,35 +770,33 @@ public class Page {
*
* @param chunk the chunk
* @param buff the target buffer
* @return the target buffer
*/
private ByteBuffer write(Chunk chunk, ByteBuffer buff) {
buff = DataUtils.ensureCapacity(buff, 1024);
private void write(Chunk chunk, WriteBuffer buff) {
int start = buff.position();
buff.putInt(0);
buff.putShort((byte) 0);
DataUtils.writeVarInt(buff, map.getId());
buff.writeVarInt(map.getId());
int len = keyCount;
DataUtils.writeVarInt(buff, len);
buff.writeVarInt(len);
int type = children != null ? DataUtils.PAGE_TYPE_NODE
: DataUtils.PAGE_TYPE_LEAF;
buff.put((byte) type);
int compressStart = buff.position();
DataType keyType = map.getKeyType();
for (int i = 0; i < len; i++) {
buff = keyType.write(buff, keys[i]);
keyType.write(buff, keys[i]);
}
if (type == DataUtils.PAGE_TYPE_NODE) {
for (int i = 0; i <= len; i++) {
buff.putLong(children[i]);
}
for (int i = 0; i <= len; i++) {
DataUtils.writeVarLong(buff, counts[i]);
buff.writeVarLong(counts[i]);
}
} else {
DataType valueType = map.getValueType();
for (int i = 0; i < len; i++) {
buff = valueType.write(buff, values[i]);
valueType.write(buff, values[i]);
}
}
if (map.getStore().getCompress()) {
......@@ -812,7 +810,7 @@ public class Page {
if (compLen + DataUtils.getVarIntLen(compLen - expLen) < expLen) {
buff.position(compressStart - 1);
buff.put((byte) (type + DataUtils.PAGE_COMPRESSED));
DataUtils.writeVarInt(buff, expLen - compLen);
buff.writeVarInt(expLen - compLen);
buff.put(comp, 0, compLen);
}
}
......@@ -833,7 +831,6 @@ public class Page {
chunk.maxLengthLive += max;
chunk.pageCount++;
chunk.pageCountLive++;
return buff;
}
/**
......@@ -842,24 +839,23 @@ public class Page {
*
* @param chunk the chunk
* @param buff the target buffer
* @return the target buffer
*/
ByteBuffer writeUnsavedRecursive(Chunk chunk, ByteBuffer buff) {
void writeUnsavedRecursive(Chunk chunk, WriteBuffer buff) {
if (pos != 0) {
// already stored before
return buff;
return;
}
if (!isLeaf()) {
int len = children.length;
for (int i = 0; i < len; i++) {
Page p = childrenPages[i];
if (p != null) {
buff = p.writeUnsavedRecursive(chunk, buff);
p.writeUnsavedRecursive(chunk, buff);
children[i] = p.getPos();
}
}
}
return write(chunk, buff);
write(chunk, buff);
}
/**
......
/*
* Copyright 2004-2013 H2 Group. Multiple-Licensed under the H2 License,
* Version 1.0, and under the Eclipse Public License, Version 1.0
* (http://h2database.com/html/license.html).
* Initial Developer: H2 Group
*/
package org.h2.mvstore;
import java.nio.ByteBuffer;
/**
* An auto-resize buffer to write data into a ByteBuffer.
*/
public class WriteBuffer {
private static final int MAX_REUSE_LIMIT = 4 * 1024 * 1024;
/**
* The maximum byte to grow a buffer at a time.
*/
private static final int MAX_GROW = 4 * 1024 * 1024;
private ByteBuffer reuse = ByteBuffer.allocate(512 * 1024);
private ByteBuffer buff = reuse;
public void writeVarInt(int x) {
DataUtils.writeVarInt(ensureCapacity(5), x);
}
public void writeVarLong(long x) {
DataUtils.writeVarLong(ensureCapacity(10), x);
}
public void writeStringData(String s, int len) {
ByteBuffer b = ensureCapacity(3 * len);
for (int i = 0; i < len; i++) {
int c = s.charAt(i);
if (c < 0x80) {
b.put((byte) c);
} else if (c >= 0x800) {
b.put((byte) (0xe0 | (c >> 12)));
b.put((byte) (((c >> 6) & 0x3f)));
b.put((byte) (c & 0x3f));
} else {
b.put((byte) (0xc0 | (c >> 6)));
b.put((byte) (c & 0x3f));
}
}
}
public void put(byte x) {
ensureCapacity(1).put(x);
}
public void putChar(char x) {
ensureCapacity(2).putChar(x);
}
public void putShort(short x) {
ensureCapacity(2).putShort(x);
}
public void putInt(int x) {
ensureCapacity(4).putInt(x);
}
public void putLong(long x) {
ensureCapacity(8).putLong(x);
}
public void putFloat(float x) {
ensureCapacity(4).putFloat(x);
}
public void putDouble(double x) {
ensureCapacity(8).putDouble(x);
}
public void put(byte[] bytes) {
ensureCapacity(bytes.length).put(bytes);
}
public void put(byte[] bytes, int offset, int length) {
ensureCapacity(length).put(bytes, offset, length);
}
public void position(int newPosition) {
buff.position(newPosition);
}
public int position() {
return buff.position();
}
public void get(byte[] dst) {
buff.get(dst);
}
public void putInt(int index, int value) {
buff.putInt(index, value);
}
public void putShort(int index, short value) {
buff.putShort(index, value);
}
public void put(ByteBuffer src) {
ensureCapacity(buff.remaining()).put(src);
}
public void limit(int newLimit) {
ensureCapacity(newLimit - buff.position()).limit(newLimit);
}
public int limit() {
return buff.limit();
}
public int capacity() {
return buff.capacity();
}
public ByteBuffer getBuffer() {
return buff;
}
/**
* Clear the buffer after use.
*/
void clear() {
if (buff.limit() > MAX_REUSE_LIMIT) {
buff = reuse;
}
buff.clear();
}
private ByteBuffer ensureCapacity(int len) {
if (buff.remaining() < len) {
grow(len);
}
return buff;
}
private void grow(int len) {
ByteBuffer temp = buff;
len = temp.remaining() + len;
int capacity = temp.capacity();
len = Math.max(len, Math.min(capacity + MAX_GROW, capacity * 2));
buff = ByteBuffer.allocate(len);
temp.flip();
buff.put(temp);
if (len <= MAX_REUSE_LIMIT) {
reuse = buff;
}
}
}
......@@ -17,6 +17,7 @@ import org.h2.mvstore.DataUtils;
import org.h2.mvstore.MVMap;
import org.h2.mvstore.MVMap.Builder;
import org.h2.mvstore.MVStore;
import org.h2.mvstore.WriteBuffer;
import org.h2.mvstore.type.DataType;
import org.h2.mvstore.type.ObjectDataType;
import org.h2.util.New;
......@@ -1324,17 +1325,16 @@ public class TransactionStore {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
VersionedValue v = (VersionedValue) obj;
DataUtils.writeVarLong(buff, v.transactionId);
DataUtils.writeVarLong(buff, v.logId);
buff.writeVarLong(v.transactionId);
buff.writeVarLong(v.logId);
if (v.value == null) {
buff.put((byte) 0);
} else {
buff.put((byte) 1);
buff = valueType.write(buff, v.value);
valueType.write(buff, v.value);
}
return buff;
}
@Override
......@@ -1396,7 +1396,7 @@ public class TransactionStore {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
Object[] array = (Object[]) obj;
for (int i = 0; i < arrayLength; i++) {
DataType t = elementTypes[i];
......@@ -1405,10 +1405,9 @@ public class TransactionStore {
buff.put((byte) 0);
} else {
buff.put((byte) 1);
buff = t.write(buff, o);
t.write(buff, o);
}
}
return buff;
}
@Override
......
......@@ -16,6 +16,7 @@ import java.sql.SQLException;
import org.h2.constant.ErrorCode;
import org.h2.message.DbException;
import org.h2.mvstore.DataUtils;
import org.h2.mvstore.WriteBuffer;
import org.h2.mvstore.type.DataType;
import org.h2.result.SortOrder;
import org.h2.store.DataHandler;
......@@ -146,17 +147,15 @@ public class ValueDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
Value x = (Value) obj;
buff = DataUtils.ensureCapacity(buff, 0);
buff = writeValue(buff, x);
return buff;
writeValue(buff, x);
}
private ByteBuffer writeValue(ByteBuffer buff, Value v) {
private void writeValue(WriteBuffer buff, Value v) {
if (v == ValueNull.INSTANCE) {
buff.put((byte) 0);
return buff;
return;
}
int type = v.getType();
switch (type) {
......@@ -175,12 +174,12 @@ public class ValueDataType implements DataType {
int x = v.getInt();
if (x < 0) {
buff.put((byte) INT_NEG);
writeVarInt(buff, -x);
buff.writeVarInt(-x);
} else if (x < 16) {
buff.put((byte) (INT_0_15 + x));
} else {
buff.put((byte) type);
writeVarInt(buff, x);
buff.writeVarInt(x);
}
break;
}
......@@ -188,12 +187,12 @@ public class ValueDataType implements DataType {
long x = v.getLong();
if (x < 0) {
buff.put((byte) LONG_NEG);
writeVarLong(buff, -x);
buff.writeVarLong(-x);
} else if (x < 8) {
buff.put((byte) (LONG_0_7 + x));
} else {
buff.put((byte) type);
writeVarLong(buff, x);
buff.writeVarLong(x);
}
break;
}
......@@ -210,19 +209,18 @@ public class ValueDataType implements DataType {
if (bits <= 63) {
if (scale == 0) {
buff.put((byte) DECIMAL_SMALL_0);
writeVarLong(buff, b.longValue());
buff.writeVarLong(b.longValue());
} else {
buff.put((byte) DECIMAL_SMALL);
writeVarInt(buff, scale);
writeVarLong(buff, b.longValue());
buff.writeVarInt(scale);
buff.writeVarLong(b.longValue());
}
} else {
buff.put((byte) type);
writeVarInt(buff, scale);
buff.writeVarInt(scale);
byte[] bytes = b.toByteArray();
writeVarInt(buff, bytes.length);
buff = DataUtils.ensureCapacity(buff, bytes.length);
buff.put(bytes, 0, bytes.length);
buff.writeVarInt(bytes.length);
buff.put(bytes);
}
}
break;
......@@ -233,34 +231,33 @@ public class ValueDataType implements DataType {
long nanos = t.getNanos();
long millis = nanos / 1000000;
nanos -= millis * 1000000;
writeVarLong(buff, millis);
writeVarLong(buff, nanos);
buff.writeVarLong(millis);
buff.writeVarLong(nanos);
break;
}
case Value.DATE: {
buff.put((byte) type);
long x = ((ValueDate) v).getDateValue();
writeVarLong(buff, x);
buff.writeVarLong(x);
break;
}
case Value.TIMESTAMP: {
buff.put((byte) type);
ValueTimestamp ts = (ValueTimestamp) v;
long dateValue = ts.getDateValue();
writeVarLong(buff, dateValue);
buff.writeVarLong(dateValue);
long nanos = ts.getNanos();
long millis = nanos / 1000000;
nanos -= millis * 1000000;
writeVarLong(buff, millis);
writeVarLong(buff, nanos);
buff.writeVarLong(millis);
buff.writeVarLong(nanos);
break;
}
case Value.JAVA_OBJECT: {
buff.put((byte) type);
byte[] b = v.getBytesNoCopy();
writeVarInt(buff, b.length);
buff = DataUtils.ensureCapacity(buff, b.length);
buff.put(b, 0, b.length);
buff.writeVarInt(b.length);
buff.put(b);
break;
}
case Value.BYTES: {
......@@ -268,12 +265,11 @@ public class ValueDataType implements DataType {
int len = b.length;
if (len < 32) {
buff.put((byte) (BYTES_0_31 + len));
buff.put(b, 0, b.length);
buff.put(b);
} else {
buff.put((byte) type);
writeVarInt(buff, b.length);
buff = DataUtils.ensureCapacity(buff, b.length);
buff.put(b, 0, b.length);
buff.writeVarInt(b.length);
buff.put(b);
}
break;
}
......@@ -289,17 +285,17 @@ public class ValueDataType implements DataType {
int len = s.length();
if (len < 32) {
buff.put((byte) (STRING_0_31 + len));
buff = writeStringWithoutLength(buff, s, len);
buff.writeStringData(s, len);
} else {
buff.put((byte) type);
buff = writeString(buff, s);
writeString(buff, s);
}
break;
}
case Value.STRING_IGNORECASE:
case Value.STRING_FIXED:
buff.put((byte) type);
buff = writeString(buff, v.getString());
writeString(buff, v.getString());
break;
case Value.DOUBLE: {
double x = v.getDouble();
......@@ -311,7 +307,7 @@ public class ValueDataType implements DataType {
buff.put((byte) DOUBLE_0_1);
} else {
buff.put((byte) type);
writeVarLong(buff, Long.reverse(d));
buff.writeVarLong(Long.reverse(d));
}
}
break;
......@@ -326,7 +322,7 @@ public class ValueDataType implements DataType {
buff.put((byte) FLOAT_0_1);
} else {
buff.put((byte) type);
writeVarInt(buff, Integer.reverse(f));
buff.writeVarInt(Integer.reverse(f));
}
}
break;
......@@ -343,31 +339,29 @@ public class ValueDataType implements DataType {
if (!lob.isLinked()) {
t = -2;
}
writeVarInt(buff, t);
writeVarInt(buff, lob.getTableId());
writeVarInt(buff, lob.getObjectId());
writeVarLong(buff, lob.getPrecision());
buff.writeVarInt(t);
buff.writeVarInt(lob.getTableId());
buff.writeVarInt(lob.getObjectId());
buff.writeVarLong(lob.getPrecision());
buff.put((byte) (lob.isCompressed() ? 1 : 0));
if (t == -2) {
buff = writeString(buff, lob.getFileName());
writeString(buff, lob.getFileName());
}
} else {
writeVarInt(buff, small.length);
buff = DataUtils.ensureCapacity(buff, small.length);
buff.put(small, 0, small.length);
buff.writeVarInt(small.length);
buff.put(small);
}
} else {
ValueLobDb lob = (ValueLobDb) v;
byte[] small = lob.getSmall();
if (small == null) {
writeVarInt(buff, -3);
writeVarInt(buff, lob.getTableId());
writeVarLong(buff, lob.getLobId());
writeVarLong(buff, lob.getPrecision());
buff.writeVarInt(-3);
buff.writeVarInt(lob.getTableId());
buff.writeVarLong(lob.getLobId());
buff.writeVarLong(lob.getPrecision());
} else {
writeVarInt(buff, small.length);
buff = DataUtils.ensureCapacity(buff, small.length);
buff.put(small, 0, small.length);
buff.writeVarInt(small.length);
buff.put(small);
}
}
break;
......@@ -375,10 +369,9 @@ public class ValueDataType implements DataType {
case Value.ARRAY: {
buff.put((byte) type);
Value[] list = ((ValueArray) v).getList();
writeVarInt(buff, list.length);
buff.writeVarInt(list.length);
for (Value x : list) {
buff = DataUtils.ensureCapacity(buff, 0);
buff = writeValue(buff, x);
writeValue(buff, x);
}
break;
}
......@@ -389,20 +382,19 @@ public class ValueDataType implements DataType {
rs.beforeFirst();
ResultSetMetaData meta = rs.getMetaData();
int columnCount = meta.getColumnCount();
writeVarInt(buff, columnCount);
buff.writeVarInt(columnCount);
for (int i = 0; i < columnCount; i++) {
buff = DataUtils.ensureCapacity(buff, 0);
buff = writeString(buff, meta.getColumnName(i + 1));
writeVarInt(buff, meta.getColumnType(i + 1));
writeVarInt(buff, meta.getPrecision(i + 1));
writeVarInt(buff, meta.getScale(i + 1));
writeString(buff, meta.getColumnName(i + 1));
buff.writeVarInt(meta.getColumnType(i + 1));
buff.writeVarInt(meta.getPrecision(i + 1));
buff.writeVarInt(meta.getScale(i + 1));
}
while (rs.next()) {
buff.put((byte) 1);
for (int i = 0; i < columnCount; i++) {
int t = org.h2.value.DataType.convertSQLTypeToValueType(meta.getColumnType(i + 1));
Value val = org.h2.value.DataType.readValue(null, rs, i + 1, t);
buff = writeValue(buff, val);
writeValue(buff, val);
}
}
buff.put((byte) 0);
......@@ -416,55 +408,19 @@ public class ValueDataType implements DataType {
buff.put((byte) type);
byte[] b = v.getBytes();
int len = b.length;
writeVarInt(buff, len);
buff = DataUtils.ensureCapacity(buff, len);
buff.put(b, 0, len);
buff.writeVarInt(len);
buff.put(b);
break;
}
default:
DbException.throwInternalError("type=" + v.getType());
}
return buff;
}
private static void writeVarInt(ByteBuffer buff, int x) {
while ((x & ~0x7f) != 0) {
buff.put((byte) (0x80 | (x & 0x7f)));
x >>>= 7;
}
buff.put((byte) x);
}
private static void writeVarLong(ByteBuffer buff, long x) {
while ((x & ~0x7f) != 0) {
buff.put((byte) ((x & 0x7f) | 0x80));
x >>>= 7;
}
buff.put((byte) x);
}
private static ByteBuffer writeString(ByteBuffer buff, String s) {
private static void writeString(WriteBuffer buff, String s) {
int len = s.length();
writeVarInt(buff, len);
return writeStringWithoutLength(buff, s, len);
}
private static ByteBuffer writeStringWithoutLength(ByteBuffer buff, String s, int len) {
buff = DataUtils.ensureCapacity(buff, 3 * len);
for (int i = 0; i < len; i++) {
int c = s.charAt(i);
if (c < 0x80) {
buff.put((byte) c);
} else if (c >= 0x800) {
buff.put((byte) (0xe0 | (c >> 12)));
buff.put((byte) (((c >> 6) & 0x3f)));
buff.put((byte) (c & 0x3f));
} else {
buff.put((byte) (0xc0 | (c >> 6)));
buff.put((byte) (c & 0x3f));
}
}
return buff;
buff.writeVarInt(len);
buff.writeStringData(s, len);
}
/**
......
......@@ -9,6 +9,7 @@ package org.h2.mvstore.rtree;
import java.nio.ByteBuffer;
import java.util.ArrayList;
import org.h2.mvstore.DataUtils;
import org.h2.mvstore.WriteBuffer;
import org.h2.mvstore.type.DataType;
/**
......@@ -53,7 +54,7 @@ public class SpatialDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
SpatialKey k = (SpatialKey) obj;
int flags = 0;
for (int i = 0; i < dimensions; i++) {
......@@ -61,15 +62,14 @@ public class SpatialDataType implements DataType {
flags |= 1 << i;
}
}
DataUtils.writeVarInt(buff, flags);
buff.writeVarInt(flags);
for (int i = 0; i < dimensions; i++) {
buff.putFloat(k.min(i));
if ((flags & (1 << i)) == 0) {
buff.putFloat(k.max(i));
}
}
DataUtils.writeVarLong(buff, k.getId());
return buff;
buff.writeVarLong(k.getId());
}
@Override
......
......@@ -8,6 +8,8 @@ package org.h2.mvstore.type;
import java.nio.ByteBuffer;
import org.h2.mvstore.WriteBuffer;
/**
* A data type.
*/
......@@ -36,9 +38,8 @@ public interface DataType {
*
* @param buff the target buffer
* @param obj the value
* @return the byte buffer
*/
ByteBuffer write(ByteBuffer buff, Object obj);
void write(WriteBuffer buff, Object obj);
/**
* Read an object.
......
......@@ -19,6 +19,7 @@ import java.util.Date;
import java.util.HashMap;
import java.util.UUID;
import org.h2.mvstore.DataUtils;
import org.h2.mvstore.WriteBuffer;
import org.h2.util.New;
/**
......@@ -109,8 +110,8 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
return last.write(buff, obj);
public void write(WriteBuffer buff, Object obj) {
last.write(buff, obj);
}
private AutoDetectDataType newType(int typeId) {
......@@ -414,8 +415,8 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object o) {
return getType(o).write(buff, o);
public void write(WriteBuffer buff, Object o) {
getType(o).write(buff, o);
}
@Override
......@@ -473,12 +474,12 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (obj != null) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
buff.put((byte) TYPE_NULL);
return buff;
}
@Override
......@@ -513,13 +514,13 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Boolean)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
int tag = ((Boolean) obj) ? TAG_BOOLEAN_TRUE : TYPE_BOOLEAN;
buff.put((byte) tag);
return buff;
}
@Override
......@@ -554,13 +555,13 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Byte)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
buff.put((byte) TYPE_BYTE);
buff.put(((Byte) obj).byteValue());
return buff;
}
@Override
......@@ -595,13 +596,13 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Character)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
buff.put((byte) TYPE_CHAR);
buff.putChar(((Character) obj).charValue());
return buff;
}
@Override
......@@ -636,13 +637,13 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Short)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
buff.put((byte) TYPE_SHORT);
buff.putShort(((Short) obj).shortValue());
return buff;
}
@Override
......@@ -677,9 +678,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Integer)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
int x = (Integer) obj;
if (x < 0) {
......@@ -689,18 +691,17 @@ public class ObjectDataType implements DataType {
buff.putInt(x);
} else {
buff.put((byte) TAG_INTEGER_NEGATIVE);
DataUtils.writeVarInt(buff, -x);
buff.writeVarInt(-x);
}
} else if (x <= 15) {
buff.put((byte) (TAG_INTEGER_0_15 + x));
} else if (x <= DataUtils.COMPRESSED_VAR_INT_MAX) {
buff.put((byte) TYPE_INT);
DataUtils.writeVarInt(buff, x);
buff.writeVarInt(x);
} else {
buff.put((byte) TAG_INTEGER_FIXED);
buff.putInt(x);
}
return buff;
}
@Override
......@@ -743,9 +744,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Long)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
long x = (Long) obj;
if (x < 0) {
......@@ -755,18 +757,17 @@ public class ObjectDataType implements DataType {
buff.putLong(x);
} else {
buff.put((byte) TAG_LONG_NEGATIVE);
DataUtils.writeVarLong(buff, -x);
buff.writeVarLong(-x);
}
} else if (x <= 7) {
buff.put((byte) (TAG_LONG_0_7 + x));
} else if (x <= DataUtils.COMPRESSED_VAR_LONG_MAX) {
buff.put((byte) TYPE_LONG);
DataUtils.writeVarLong(buff, x);
buff.writeVarLong(x);
} else {
buff.put((byte) TAG_LONG_FIXED);
buff.putLong(x);
}
return buff;
}
@Override
......@@ -809,9 +810,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Float)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
float x = (Float) obj;
int f = Float.floatToIntBits(x);
......@@ -823,13 +825,12 @@ public class ObjectDataType implements DataType {
int value = Integer.reverse(f);
if (value >= 0 && value <= DataUtils.COMPRESSED_VAR_INT_MAX) {
buff.put((byte) TYPE_FLOAT);
DataUtils.writeVarInt(buff, value);
buff.writeVarInt(value);
} else {
buff.put((byte) TAG_FLOAT_FIXED);
buff.putFloat(x);
}
}
return buff;
}
@Override
......@@ -872,9 +873,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof Double)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
double x = (Double) obj;
long d = Double.doubleToLongBits(x);
......@@ -886,13 +888,12 @@ public class ObjectDataType implements DataType {
long value = Long.reverse(d);
if (value >= 0 && value <= DataUtils.COMPRESSED_VAR_LONG_MAX) {
buff.put((byte) TYPE_DOUBLE);
DataUtils.writeVarLong(buff, value);
buff.writeVarLong(value);
} else {
buff.put((byte) TAG_DOUBLE_FIXED);
buff.putDouble(x);
}
}
return buff;
}
@Override
......@@ -935,9 +936,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!isBigInteger(obj)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
BigInteger x = (BigInteger) obj;
if (BigInteger.ZERO.equals(x)) {
......@@ -948,16 +950,14 @@ public class ObjectDataType implements DataType {
int bits = x.bitLength();
if (bits <= 63) {
buff.put((byte) TAG_BIG_INTEGER_SMALL);
DataUtils.writeVarLong(buff, x.longValue());
buff.writeVarLong(x.longValue());
} else {
buff.put((byte) TYPE_BIG_INTEGER);
byte[] bytes = x.toByteArray();
DataUtils.writeVarInt(buff, bytes.length);
buff = DataUtils.ensureCapacity(buff, bytes.length);
buff.writeVarInt(bytes.length);
buff.put(bytes);
}
}
return buff;
}
@Override
......@@ -1003,9 +1003,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!isBigDecimal(obj)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
BigDecimal x = (BigDecimal) obj;
if (BigDecimal.ZERO.equals(x)) {
......@@ -1021,19 +1022,17 @@ public class ObjectDataType implements DataType {
buff.put((byte) TAG_BIG_DECIMAL_SMALL);
} else {
buff.put((byte) TAG_BIG_DECIMAL_SMALL_SCALED);
DataUtils.writeVarInt(buff, scale);
buff.writeVarInt(scale);
}
DataUtils.writeVarLong(buff, b.longValue());
buff.writeVarLong(b.longValue());
} else {
buff.put((byte) TYPE_BIG_DECIMAL);
DataUtils.writeVarInt(buff, scale);
buff.writeVarInt(scale);
byte[] bytes = b.toByteArray();
DataUtils.writeVarInt(buff, bytes.length);
buff = DataUtils.ensureCapacity(buff, bytes.length);
buff.writeVarInt(bytes.length);
buff.put(bytes);
}
}
return buff;
}
@Override
......@@ -1085,9 +1084,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof String)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
String s = (String) obj;
int len = s.length();
......@@ -1095,9 +1095,9 @@ public class ObjectDataType implements DataType {
buff.put((byte) (TAG_STRING_0_15 + len));
} else {
buff.put((byte) TYPE_STRING);
DataUtils.writeVarInt(buff, len);
buff.writeVarInt(len);
}
return DataUtils.writeStringData(buff, s, len);
buff.writeStringData(s, len);
}
@Override
......@@ -1138,15 +1138,15 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!(obj instanceof UUID)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
buff.put((byte) TYPE_UUID);
UUID a = (UUID) obj;
buff.putLong(a.getMostSignificantBits());
buff.putLong(a.getLeastSignificantBits());
return buff;
}
@Override
......@@ -1182,14 +1182,14 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!isDate(obj)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
buff.put((byte) TYPE_DATE);
Date a = (Date) obj;
buff.putLong(a.getTime());
return buff;
}
@Override
......@@ -1328,9 +1328,10 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
if (!isArray(obj)) {
return super.write(buff, obj);
super.write(buff, obj);
return;
}
Class<?> type = obj.getClass().getComponentType();
Integer classId = getCommonClassId(type);
......@@ -1344,17 +1345,15 @@ public class ObjectDataType implements DataType {
} else {
buff.put((byte) TYPE_ARRAY);
buff.put((byte) classId.intValue());
DataUtils.writeVarInt(buff, len);
buff.writeVarInt(len);
}
buff = DataUtils.ensureCapacity(buff, data.length);
buff.put(data);
return buff;
return;
}
buff.put((byte) TYPE_ARRAY);
buff.put((byte) classId.intValue());
int len = Array.getLength(obj);
DataUtils.writeVarInt(buff, len);
buff = DataUtils.ensureCapacity(buff, 8 * len);
buff.writeVarInt(len);
for (int i = 0; i < len; i++) {
if (type == boolean.class) {
buff.put((byte) (((boolean[]) obj)[i] ? 1 : 0));
......@@ -1372,7 +1371,7 @@ public class ObjectDataType implements DataType {
buff.putLong(((long[]) obj)[i]);
}
}
return buff;
return;
}
buff.put((byte) TYPE_ARRAY);
buff.put((byte) classId.intValue());
......@@ -1380,15 +1379,14 @@ public class ObjectDataType implements DataType {
buff.put((byte) TYPE_ARRAY);
buff.put((byte) -1);
String c = type.getName();
buff = StringDataType.INSTANCE.write(buff, c);
StringDataType.INSTANCE.write(buff, c);
}
Object[] array = (Object[]) obj;
int len = array.length;
DataUtils.writeVarInt(buff, len);
buff.writeVarInt(len);
for (Object x : array) {
buff = elementType.write(buff, x);
elementType.write(buff, x);
}
return buff;
}
@Override
......@@ -1503,17 +1501,16 @@ public class ObjectDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
DataType t = getType(obj);
if (t != this) {
return t.write(buff, obj);
t.write(buff, obj);
return;
}
buff.put((byte) TYPE_SERIALIZED_OBJECT);
byte[] data = serialize(obj);
DataUtils.writeVarInt(buff, data.length);
buff = DataUtils.ensureCapacity(buff, data.length);
buff.writeVarInt(data.length);
buff.put(data);
return buff;
}
@Override
......
......@@ -8,6 +8,7 @@ package org.h2.mvstore.type;
import java.nio.ByteBuffer;
import org.h2.mvstore.DataUtils;
import org.h2.mvstore.WriteBuffer;
/**
* A string type.
......@@ -33,11 +34,11 @@ public class StringDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
String s = obj.toString();
int len = s.length();
DataUtils.writeVarInt(buff, len);
return DataUtils.writeStringData(buff, s, len);
buff.writeVarInt(len);
buff.writeStringData(s, len);
}
}
......
......@@ -8,6 +8,7 @@ package org.h2.test.store;
import java.nio.ByteBuffer;
import org.h2.mvstore.DataUtils;
import org.h2.mvstore.WriteBuffer;
import org.h2.mvstore.type.DataType;
/**
......@@ -69,15 +70,13 @@ public class RowDataType implements DataType {
}
@Override
public ByteBuffer write(ByteBuffer buff, Object obj) {
public void write(WriteBuffer buff, Object obj) {
Object[] x = (Object[]) obj;
int len = x.length;
DataUtils.writeVarInt(buff, len);
buff.writeVarInt(len);
for (int i = 0; i < len; i++) {
buff = DataUtils.ensureCapacity(buff, 0);
buff = types[i].write(buff, x[i]);
types[i].write(buff, x[i]);
}
return buff;
}
}
......@@ -13,6 +13,8 @@ import java.sql.Timestamp;
import java.util.Arrays;
import java.util.Random;
import java.util.UUID;
import org.h2.mvstore.WriteBuffer;
import org.h2.mvstore.type.ObjectDataType;
import org.h2.test.TestBase;
......@@ -131,17 +133,18 @@ public class TestObjectDataType extends TestBase {
ot.getMemory(last);
assertEquals(0, ot.compare(x, x));
ByteBuffer buff = ByteBuffer.allocate(1024);
WriteBuffer buff = new WriteBuffer();
ot.getMemory(last);
buff = ot.write(buff, x);
ot.write(buff, x);
buff.put((byte) 123);
buff.flip();
ByteBuffer bb = buff.getBuffer();
bb.flip();
ot.getMemory(last);
Object y = ot.read(buff);
assertEquals(123, buff.get());
assertEquals(0, buff.remaining());
Object y = ot.read(bb);
assertEquals(123, bb.get());
assertEquals(0, bb.remaining());
assertEquals(x.getClass().getName(), y.getClass().getName());
ot.getMemory(last);
......
Markdown 格式
0%
您添加了 0 到此讨论。请谨慎行事。
请先完成此评论的编辑!
注册 或者 后发表评论