[jira] [HBASE-5469] Add baseline compression efficiency to DataBlockEncodingTool
Summary: DataBlockEncodingTool currently does not provide baseline compression efficiency, e.g. Hadoop compression codec applied to unencoded data. E.g. if we are using LZO to compress blocks, we would like to have the following columns in the report (possibly as percentages of raw data size). Baseline K+V in blockcache | Baseline K + V on disk (LZO compressed) | K + V DataBlockEncoded in block cache | K + V DataBlockEncoded + LZOCompressed (on disk) Background: we never store compressed blocks in cache, but we always store encoded data blocks in cache if data block encoding is enabled for the column family. This patch also has multiple bugfixes and improvements to DataBlockEncodingTool, including presentation format, memory requirements (reduced 3x) and fixing the handling of compression. Test Plan: * Run unit tests. * Run DataBlockEncodingTool on a variety of real-world HFiles. Reviewers: JIRA, dhruba, tedyu, stack, heyongqiang Reviewed By: tedyu Differential Revision: https://reviews.facebook.net/D2409 git-svn-id: https://svn.apache.org/repos/asf/hbase/trunk@1304626 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
parent
1e6c227883
commit
86962028dd
|
@ -40,9 +40,9 @@ abstract class BufferedDataBlockEncoder implements DataBlockEncoder {
|
||||||
private static int INITIAL_KEY_BUFFER_SIZE = 512;
|
private static int INITIAL_KEY_BUFFER_SIZE = 512;
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public ByteBuffer uncompressKeyValues(DataInputStream source,
|
public ByteBuffer decodeKeyValues(DataInputStream source,
|
||||||
boolean includesMemstoreTS) throws IOException {
|
boolean includesMemstoreTS) throws IOException {
|
||||||
return uncompressKeyValues(source, 0, 0, includesMemstoreTS);
|
return decodeKeyValues(source, 0, 0, includesMemstoreTS);
|
||||||
}
|
}
|
||||||
|
|
||||||
protected static class SeekerState {
|
protected static class SeekerState {
|
||||||
|
@ -329,11 +329,10 @@ abstract class BufferedDataBlockEncoder implements DataBlockEncoder {
|
||||||
ByteBuffer in, boolean includesMemstoreTS) throws IOException;
|
ByteBuffer in, boolean includesMemstoreTS) throws IOException;
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public void compressKeyValues(ByteBuffer in,
|
public void encodeKeyValues(ByteBuffer in,
|
||||||
boolean includesMemstoreTS,
|
boolean includesMemstoreTS,
|
||||||
HFileBlockEncodingContext blkEncodingCtx) throws IOException {
|
HFileBlockEncodingContext blkEncodingCtx) throws IOException {
|
||||||
if (!(blkEncodingCtx.getClass().getName().equals(
|
if (blkEncodingCtx.getClass() != HFileBlockDefaultEncodingContext.class) {
|
||||||
HFileBlockDefaultEncodingContext.class.getName()))) {
|
|
||||||
throw new IOException (this.getClass().getName() + " only accepts "
|
throw new IOException (this.getClass().getName() + " only accepts "
|
||||||
+ HFileBlockDefaultEncodingContext.class.getName() + " as the " +
|
+ HFileBlockDefaultEncodingContext.class.getName() + " as the " +
|
||||||
"encoding context.");
|
"encoding context.");
|
||||||
|
|
|
@ -41,7 +41,7 @@ public class CopyKeyDataBlockEncoder extends BufferedDataBlockEncoder {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public ByteBuffer uncompressKeyValues(DataInputStream source,
|
public ByteBuffer decodeKeyValues(DataInputStream source,
|
||||||
int preserveHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
int preserveHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
||||||
throws IOException {
|
throws IOException {
|
||||||
int decompressedSize = source.readInt();
|
int decompressedSize = source.readInt();
|
||||||
|
|
|
@ -37,13 +37,13 @@ import org.apache.hadoop.io.RawComparator;
|
||||||
*
|
*
|
||||||
* After encoding, it also optionally compresses the encoded data if a
|
* After encoding, it also optionally compresses the encoded data if a
|
||||||
* compression algorithm is specified in HFileBlockEncodingContext argument of
|
* compression algorithm is specified in HFileBlockEncodingContext argument of
|
||||||
* {@link #compressKeyValues(ByteBuffer, boolean, HFileBlockEncodingContext)}.
|
* {@link #encodeKeyValues(ByteBuffer, boolean, HFileBlockEncodingContext)}.
|
||||||
*/
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
public interface DataBlockEncoder {
|
public interface DataBlockEncoder {
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Compress KeyValues. It will first encode key value pairs, and then
|
* Encodes KeyValues. It will first encode key value pairs, and then
|
||||||
* optionally do the compression for the encoded data.
|
* optionally do the compression for the encoded data.
|
||||||
*
|
*
|
||||||
* @param in
|
* @param in
|
||||||
|
@ -57,24 +57,24 @@ public interface DataBlockEncoder {
|
||||||
* @throws IOException
|
* @throws IOException
|
||||||
* If there is an error writing to output stream.
|
* If there is an error writing to output stream.
|
||||||
*/
|
*/
|
||||||
public void compressKeyValues(
|
public void encodeKeyValues(
|
||||||
ByteBuffer in, boolean includesMemstoreTS,
|
ByteBuffer in, boolean includesMemstoreTS,
|
||||||
HFileBlockEncodingContext encodingContext) throws IOException;
|
HFileBlockEncodingContext encodingContext) throws IOException;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Uncompress.
|
* Decode.
|
||||||
* @param source Compressed stream of KeyValues.
|
* @param source Compressed stream of KeyValues.
|
||||||
* @param includesMemstoreTS true if including memstore timestamp after every
|
* @param includesMemstoreTS true if including memstore timestamp after every
|
||||||
* key-value pair
|
* key-value pair
|
||||||
* @return Uncompressed block of KeyValues.
|
* @return Uncompressed block of KeyValues.
|
||||||
* @throws IOException If there is an error in source.
|
* @throws IOException If there is an error in source.
|
||||||
*/
|
*/
|
||||||
public ByteBuffer uncompressKeyValues(DataInputStream source,
|
public ByteBuffer decodeKeyValues(DataInputStream source,
|
||||||
boolean includesMemstoreTS) throws IOException;
|
boolean includesMemstoreTS) throws IOException;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Uncompress.
|
* Uncompress.
|
||||||
* @param source Compressed stream of KeyValues.
|
* @param source encoded stream of KeyValues.
|
||||||
* @param allocateHeaderLength allocate this many bytes for the header.
|
* @param allocateHeaderLength allocate this many bytes for the header.
|
||||||
* @param skipLastBytes Do not copy n last bytes.
|
* @param skipLastBytes Do not copy n last bytes.
|
||||||
* @param includesMemstoreTS true if including memstore timestamp after every
|
* @param includesMemstoreTS true if including memstore timestamp after every
|
||||||
|
@ -82,7 +82,7 @@ public interface DataBlockEncoder {
|
||||||
* @return Uncompressed block of KeyValues.
|
* @return Uncompressed block of KeyValues.
|
||||||
* @throws IOException If there is an error in source.
|
* @throws IOException If there is an error in source.
|
||||||
*/
|
*/
|
||||||
public ByteBuffer uncompressKeyValues(DataInputStream source,
|
public ByteBuffer decodeKeyValues(DataInputStream source,
|
||||||
int allocateHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
int allocateHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
||||||
throws IOException;
|
throws IOException;
|
||||||
|
|
||||||
|
|
|
@ -335,7 +335,7 @@ public class DiffKeyDeltaEncoder extends BufferedDataBlockEncoder {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public ByteBuffer uncompressKeyValues(DataInputStream source,
|
public ByteBuffer decodeKeyValues(DataInputStream source,
|
||||||
int allocHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
int allocHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
||||||
throws IOException {
|
throws IOException {
|
||||||
int decompressedSize = source.readInt();
|
int decompressedSize = source.readInt();
|
||||||
|
|
|
@ -19,7 +19,9 @@ package org.apache.hadoop.hbase.io.encoding;
|
||||||
import java.io.ByteArrayInputStream;
|
import java.io.ByteArrayInputStream;
|
||||||
import java.io.ByteArrayOutputStream;
|
import java.io.ByteArrayOutputStream;
|
||||||
import java.io.DataInputStream;
|
import java.io.DataInputStream;
|
||||||
|
import java.io.DataOutputStream;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
|
import java.io.OutputStream;
|
||||||
import java.nio.ByteBuffer;
|
import java.nio.ByteBuffer;
|
||||||
import java.util.Iterator;
|
import java.util.Iterator;
|
||||||
|
|
||||||
|
@ -27,46 +29,44 @@ import org.apache.commons.lang.NotImplementedException;
|
||||||
import org.apache.hadoop.classification.InterfaceAudience;
|
import org.apache.hadoop.classification.InterfaceAudience;
|
||||||
import org.apache.hadoop.hbase.KeyValue;
|
import org.apache.hadoop.hbase.KeyValue;
|
||||||
import org.apache.hadoop.hbase.io.hfile.Compression;
|
import org.apache.hadoop.hbase.io.hfile.Compression;
|
||||||
|
import org.apache.hadoop.hbase.io.hfile.Compression.Algorithm;
|
||||||
import org.apache.hadoop.hbase.io.hfile.HFileBlock;
|
import org.apache.hadoop.hbase.io.hfile.HFileBlock;
|
||||||
|
import org.apache.hadoop.hbase.util.Bytes;
|
||||||
import org.apache.hadoop.io.compress.Compressor;
|
import org.apache.hadoop.io.compress.Compressor;
|
||||||
|
|
||||||
|
import com.google.common.base.Preconditions;
|
||||||
|
import com.google.common.io.NullOutputStream;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Encapsulates a data block compressed using a particular encoding algorithm.
|
* Encapsulates a data block compressed using a particular encoding algorithm.
|
||||||
* Useful for testing and benchmarking.
|
* Useful for testing and benchmarking.
|
||||||
*/
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
public class EncodedDataBlock {
|
public class EncodedDataBlock {
|
||||||
private static final int BUFFER_SIZE = 4 * 1024;
|
private byte[] rawKVs;
|
||||||
protected DataBlockEncoder dataBlockEncoder;
|
private ByteBuffer rawBuffer;
|
||||||
ByteArrayOutputStream uncompressedOutputStream;
|
private DataBlockEncoder dataBlockEncoder;
|
||||||
ByteBuffer uncompressedBuffer;
|
|
||||||
private byte[] cacheCompressData;
|
private byte[] cachedEncodedData;
|
||||||
private boolean includesMemstoreTS;
|
private boolean includesMemstoreTS;
|
||||||
|
|
||||||
private final HFileBlockEncodingContext encodingCxt;
|
private final HFileBlockEncodingContext encodingCtx;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Create a buffer which will be encoded using dataBlockEncoder.
|
* Create a buffer which will be encoded using dataBlockEncoder.
|
||||||
* @param dataBlockEncoder Algorithm used for compression.
|
* @param dataBlockEncoder Algorithm used for compression.
|
||||||
* @param encoding encoding type used
|
* @param encoding encoding type used
|
||||||
|
* @param rawKVs
|
||||||
*/
|
*/
|
||||||
public EncodedDataBlock(DataBlockEncoder dataBlockEncoder,
|
public EncodedDataBlock(DataBlockEncoder dataBlockEncoder,
|
||||||
boolean includesMemstoreTS, DataBlockEncoding encoding) {
|
boolean includesMemstoreTS, DataBlockEncoding encoding, byte[] rawKVs) {
|
||||||
|
Preconditions.checkNotNull(encoding,
|
||||||
|
"Cannot create encoded data block with null encoder");
|
||||||
this.dataBlockEncoder = dataBlockEncoder;
|
this.dataBlockEncoder = dataBlockEncoder;
|
||||||
uncompressedOutputStream = new ByteArrayOutputStream(BUFFER_SIZE);
|
encodingCtx =
|
||||||
encodingCxt =
|
|
||||||
dataBlockEncoder.newDataBlockEncodingContext(Compression.Algorithm.NONE,
|
dataBlockEncoder.newDataBlockEncodingContext(Compression.Algorithm.NONE,
|
||||||
encoding, HFileBlock.DUMMY_HEADER);
|
encoding, HFileBlock.DUMMY_HEADER);
|
||||||
}
|
this.rawKVs = rawKVs;
|
||||||
|
|
||||||
/**
|
|
||||||
* Add KeyValue and compress it.
|
|
||||||
* @param kv Item to be added and compressed.
|
|
||||||
*/
|
|
||||||
public void addKv(KeyValue kv) {
|
|
||||||
cacheCompressData = null;
|
|
||||||
uncompressedOutputStream.write(
|
|
||||||
kv.getBuffer(), kv.getOffset(), kv.getLength());
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
@ -74,19 +74,20 @@ public class EncodedDataBlock {
|
||||||
* @return Forwards sequential iterator.
|
* @return Forwards sequential iterator.
|
||||||
*/
|
*/
|
||||||
public Iterator<KeyValue> getIterator() {
|
public Iterator<KeyValue> getIterator() {
|
||||||
final int uncompressedSize = uncompressedOutputStream.size();
|
final int rawSize = rawKVs.length;
|
||||||
final ByteArrayInputStream bais = new ByteArrayInputStream(
|
byte[] encodedDataWithHeader = getEncodedData();
|
||||||
getCompressedData());
|
int bytesToSkip = encodingCtx.getHeaderSize() + Bytes.SIZEOF_SHORT;
|
||||||
|
ByteArrayInputStream bais = new ByteArrayInputStream(encodedDataWithHeader,
|
||||||
|
bytesToSkip, encodedDataWithHeader.length - bytesToSkip);
|
||||||
final DataInputStream dis = new DataInputStream(bais);
|
final DataInputStream dis = new DataInputStream(bais);
|
||||||
|
|
||||||
|
|
||||||
return new Iterator<KeyValue>() {
|
return new Iterator<KeyValue>() {
|
||||||
private ByteBuffer decompressedData = null;
|
private ByteBuffer decompressedData = null;
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public boolean hasNext() {
|
public boolean hasNext() {
|
||||||
if (decompressedData == null) {
|
if (decompressedData == null) {
|
||||||
return uncompressedSize > 0;
|
return rawSize > 0;
|
||||||
}
|
}
|
||||||
return decompressedData.hasRemaining();
|
return decompressedData.hasRemaining();
|
||||||
}
|
}
|
||||||
|
@ -95,7 +96,7 @@ public class EncodedDataBlock {
|
||||||
public KeyValue next() {
|
public KeyValue next() {
|
||||||
if (decompressedData == null) {
|
if (decompressedData == null) {
|
||||||
try {
|
try {
|
||||||
decompressedData = dataBlockEncoder.uncompressKeyValues(
|
decompressedData = dataBlockEncoder.decodeKeyValues(
|
||||||
dis, includesMemstoreTS);
|
dis, includesMemstoreTS);
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
throw new RuntimeException("Problem with data block encoder, " +
|
throw new RuntimeException("Problem with data block encoder, " +
|
||||||
|
@ -129,99 +130,86 @@ public class EncodedDataBlock {
|
||||||
* @return Size in bytes of compressed data.
|
* @return Size in bytes of compressed data.
|
||||||
*/
|
*/
|
||||||
public int getSize() {
|
public int getSize() {
|
||||||
return getCompressedData().length;
|
return getEncodedData().length;
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Find the size of compressed data assuming that buffer will be compressed
|
* Find the size of compressed data assuming that buffer will be compressed
|
||||||
* using given algorithm.
|
* using given algorithm.
|
||||||
* @param compressor Algorithm used for compression.
|
* @param algo compression algorithm
|
||||||
* @param buffer Array to be compressed.
|
* @param compressor compressor already requested from codec
|
||||||
|
* @param inputBuffer Array to be compressed.
|
||||||
* @param offset Offset to beginning of the data.
|
* @param offset Offset to beginning of the data.
|
||||||
* @param length Length to be compressed.
|
* @param length Length to be compressed.
|
||||||
* @return Size of compressed data in bytes.
|
* @return Size of compressed data in bytes.
|
||||||
|
* @throws IOException
|
||||||
*/
|
*/
|
||||||
public static int checkCompressedSize(Compressor compressor, byte[] buffer,
|
public static int getCompressedSize(Algorithm algo, Compressor compressor,
|
||||||
int offset, int length) {
|
byte[] inputBuffer, int offset, int length) throws IOException {
|
||||||
byte[] compressedBuffer = new byte[buffer.length];
|
DataOutputStream compressedStream = new DataOutputStream(
|
||||||
// in fact the buffer could be of any positive size
|
new NullOutputStream());
|
||||||
compressor.setInput(buffer, offset, length);
|
if (compressor != null) {
|
||||||
compressor.finish();
|
compressor.reset();
|
||||||
int currentPos = 0;
|
|
||||||
while (!compressor.finished()) {
|
|
||||||
try {
|
|
||||||
// we don't care about compressed data,
|
|
||||||
// we just want to callculate number of bytes
|
|
||||||
currentPos += compressor.compress(compressedBuffer, 0,
|
|
||||||
compressedBuffer.length);
|
|
||||||
} catch (IOException e) {
|
|
||||||
throw new RuntimeException(
|
|
||||||
"For some reason compressor couldn't read data. " +
|
|
||||||
"It is likely a problem with " +
|
|
||||||
compressor.getClass().getName(), e);
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
return currentPos;
|
OutputStream compressingStream = algo.createCompressionStream(
|
||||||
|
compressedStream, compressor, 0);
|
||||||
|
|
||||||
|
compressingStream.write(inputBuffer, offset, length);
|
||||||
|
compressingStream.flush();
|
||||||
|
compressingStream.close();
|
||||||
|
|
||||||
|
return compressedStream.size();
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Estimate size after second stage of compression (e.g. LZO).
|
* Estimate size after second stage of compression (e.g. LZO).
|
||||||
* @param compressor Algorithm which will be used for compressions.
|
* @param comprAlgo compression algorithm to be used for compression
|
||||||
|
* @param compressor compressor corresponding to the given compression
|
||||||
|
* algorithm
|
||||||
* @return Size after second stage of compression.
|
* @return Size after second stage of compression.
|
||||||
*/
|
*/
|
||||||
public int checkCompressedSize(Compressor compressor) {
|
public int getEncodedCompressedSize(Algorithm comprAlgo,
|
||||||
// compress
|
Compressor compressor) throws IOException {
|
||||||
byte[] compressedBytes = getCompressedData();
|
byte[] compressedBytes = getEncodedData();
|
||||||
return checkCompressedSize(compressor, compressedBytes, 0,
|
return getCompressedSize(comprAlgo, compressor, compressedBytes, 0,
|
||||||
compressedBytes.length);
|
compressedBytes.length);
|
||||||
}
|
}
|
||||||
|
|
||||||
private byte[] getCompressedData() {
|
/** @return encoded data with header */
|
||||||
// is cached
|
private byte[] getEncodedData() {
|
||||||
if (cacheCompressData != null) {
|
if (cachedEncodedData != null) {
|
||||||
return cacheCompressData;
|
return cachedEncodedData;
|
||||||
}
|
}
|
||||||
cacheCompressData = encodeData();
|
cachedEncodedData = encodeData();
|
||||||
|
return cachedEncodedData;
|
||||||
return cacheCompressData;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
private ByteBuffer getUncompressedBuffer() {
|
private ByteBuffer getUncompressedBuffer() {
|
||||||
if (uncompressedBuffer == null ||
|
if (rawBuffer == null || rawBuffer.limit() < rawKVs.length) {
|
||||||
uncompressedBuffer.limit() < uncompressedOutputStream.size()) {
|
rawBuffer = ByteBuffer.wrap(rawKVs);
|
||||||
uncompressedBuffer = ByteBuffer.wrap(
|
|
||||||
uncompressedOutputStream.toByteArray());
|
|
||||||
}
|
}
|
||||||
return uncompressedBuffer;
|
return rawBuffer;
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Do the encoding .
|
* Do the encoding, but do not cache the encoded data.
|
||||||
* @return encoded byte buffer.
|
* @return encoded data block with header and checksum
|
||||||
*/
|
*/
|
||||||
public byte[] encodeData() {
|
public byte[] encodeData() {
|
||||||
try {
|
try {
|
||||||
this.dataBlockEncoder.compressKeyValues(
|
this.dataBlockEncoder.encodeKeyValues(
|
||||||
getUncompressedBuffer(), includesMemstoreTS, encodingCxt);
|
getUncompressedBuffer(), includesMemstoreTS, encodingCtx);
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
throw new RuntimeException(String.format(
|
throw new RuntimeException(String.format(
|
||||||
"Bug in encoding part of algorithm %s. " +
|
"Bug in encoding part of algorithm %s. " +
|
||||||
"Probably it requested more bytes than are available.",
|
"Probably it requested more bytes than are available.",
|
||||||
toString()), e);
|
toString()), e);
|
||||||
}
|
}
|
||||||
return encodingCxt.getUncompressedBytesWithHeader();
|
return encodingCtx.getUncompressedBytesWithHeader();
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public String toString() {
|
public String toString() {
|
||||||
return dataBlockEncoder.toString();
|
return dataBlockEncoder.toString();
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
|
||||||
* Get uncompressed buffer.
|
|
||||||
* @return The buffer.
|
|
||||||
*/
|
|
||||||
public byte[] getRawKeyValues() {
|
|
||||||
return uncompressedOutputStream.toByteArray();
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
|
@ -362,7 +362,7 @@ public class FastDiffDeltaEncoder extends BufferedDataBlockEncoder {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public ByteBuffer uncompressKeyValues(DataInputStream source,
|
public ByteBuffer decodeKeyValues(DataInputStream source,
|
||||||
int allocHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
int allocHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
||||||
throws IOException {
|
throws IOException {
|
||||||
int decompressedSize = source.readInt();
|
int decompressedSize = source.readInt();
|
||||||
|
|
|
@ -16,10 +16,14 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.hadoop.hbase.io.encoding;
|
package org.apache.hadoop.hbase.io.encoding;
|
||||||
|
|
||||||
|
import java.io.ByteArrayOutputStream;
|
||||||
import java.io.DataInputStream;
|
import java.io.DataInputStream;
|
||||||
import java.io.DataOutputStream;
|
import java.io.DataOutputStream;
|
||||||
|
import java.io.FilterOutputStream;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
|
import java.lang.reflect.Field;
|
||||||
import java.nio.ByteBuffer;
|
import java.nio.ByteBuffer;
|
||||||
|
import java.util.Arrays;
|
||||||
|
|
||||||
import org.apache.hadoop.classification.InterfaceAudience;
|
import org.apache.hadoop.classification.InterfaceAudience;
|
||||||
import org.apache.hadoop.hbase.KeyValue;
|
import org.apache.hadoop.hbase.KeyValue;
|
||||||
|
@ -91,7 +95,7 @@ public class PrefixKeyDeltaEncoder extends BufferedDataBlockEncoder {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public ByteBuffer uncompressKeyValues(DataInputStream source,
|
public ByteBuffer decodeKeyValues(DataInputStream source,
|
||||||
int allocHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
int allocHeaderLength, int skipLastBytes, boolean includesMemstoreTS)
|
||||||
throws IOException {
|
throws IOException {
|
||||||
int decompressedSize = source.readInt();
|
int decompressedSize = source.readInt();
|
||||||
|
@ -101,7 +105,7 @@ public class PrefixKeyDeltaEncoder extends BufferedDataBlockEncoder {
|
||||||
int prevKeyOffset = 0;
|
int prevKeyOffset = 0;
|
||||||
|
|
||||||
while (source.available() > skipLastBytes) {
|
while (source.available() > skipLastBytes) {
|
||||||
prevKeyOffset = uncompressKeyValue(source, buffer, prevKeyOffset);
|
prevKeyOffset = decodeKeyValue(source, buffer, prevKeyOffset);
|
||||||
afterDecodingKeyValue(source, buffer, includesMemstoreTS);
|
afterDecodingKeyValue(source, buffer, includesMemstoreTS);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -113,7 +117,7 @@ public class PrefixKeyDeltaEncoder extends BufferedDataBlockEncoder {
|
||||||
return buffer;
|
return buffer;
|
||||||
}
|
}
|
||||||
|
|
||||||
private int uncompressKeyValue(DataInputStream source, ByteBuffer buffer,
|
private int decodeKeyValue(DataInputStream source, ByteBuffer buffer,
|
||||||
int prevKeyOffset)
|
int prevKeyOffset)
|
||||||
throws IOException, EncoderBufferTooSmallException {
|
throws IOException, EncoderBufferTooSmallException {
|
||||||
int keyLength = ByteBufferUtils.readCompressedInt(source);
|
int keyLength = ByteBufferUtils.readCompressedInt(source);
|
||||||
|
|
|
@ -33,13 +33,12 @@ import org.apache.hadoop.classification.InterfaceAudience;
|
||||||
import org.apache.hadoop.fs.FSDataInputStream;
|
import org.apache.hadoop.fs.FSDataInputStream;
|
||||||
import org.apache.hadoop.fs.FSDataOutputStream;
|
import org.apache.hadoop.fs.FSDataOutputStream;
|
||||||
import org.apache.hadoop.fs.Path;
|
import org.apache.hadoop.fs.Path;
|
||||||
import org.apache.hadoop.hbase.HConstants;
|
|
||||||
import org.apache.hadoop.hbase.fs.HFileSystem;
|
import org.apache.hadoop.hbase.fs.HFileSystem;
|
||||||
import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding;
|
import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding;
|
||||||
|
import org.apache.hadoop.hbase.io.encoding.HFileBlockDecodingContext;
|
||||||
import org.apache.hadoop.hbase.io.encoding.HFileBlockDefaultDecodingContext;
|
import org.apache.hadoop.hbase.io.encoding.HFileBlockDefaultDecodingContext;
|
||||||
import org.apache.hadoop.hbase.io.encoding.HFileBlockDefaultEncodingContext;
|
import org.apache.hadoop.hbase.io.encoding.HFileBlockDefaultEncodingContext;
|
||||||
import org.apache.hadoop.hbase.io.encoding.HFileBlockEncodingContext;
|
import org.apache.hadoop.hbase.io.encoding.HFileBlockEncodingContext;
|
||||||
import org.apache.hadoop.hbase.io.encoding.HFileBlockDecodingContext;
|
|
||||||
import org.apache.hadoop.hbase.io.hfile.Compression.Algorithm;
|
import org.apache.hadoop.hbase.io.hfile.Compression.Algorithm;
|
||||||
import org.apache.hadoop.hbase.regionserver.MemStore;
|
import org.apache.hadoop.hbase.regionserver.MemStore;
|
||||||
import org.apache.hadoop.hbase.regionserver.metrics.SchemaConfigured;
|
import org.apache.hadoop.hbase.regionserver.metrics.SchemaConfigured;
|
||||||
|
@ -123,6 +122,7 @@ public class HFileBlock extends SchemaConfigured implements Cacheable {
|
||||||
|
|
||||||
/** Just an array of bytes of the right size. */
|
/** Just an array of bytes of the right size. */
|
||||||
public static final byte[] DUMMY_HEADER = new byte[HEADER_SIZE];
|
public static final byte[] DUMMY_HEADER = new byte[HEADER_SIZE];
|
||||||
|
|
||||||
static final byte[] DUMMY_HEADER_NO_CHECKSUM =
|
static final byte[] DUMMY_HEADER_NO_CHECKSUM =
|
||||||
new byte[HEADER_SIZE_NO_CHECKSUM];
|
new byte[HEADER_SIZE_NO_CHECKSUM];
|
||||||
|
|
||||||
|
@ -835,10 +835,10 @@ public class HFileBlock extends SchemaConfigured implements Cacheable {
|
||||||
putHeader(onDiskBytesWithHeader, 0,
|
putHeader(onDiskBytesWithHeader, 0,
|
||||||
onDiskBytesWithHeader.length + numBytes,
|
onDiskBytesWithHeader.length + numBytes,
|
||||||
uncompressedBytesWithHeader.length, onDiskBytesWithHeader.length);
|
uncompressedBytesWithHeader.length, onDiskBytesWithHeader.length);
|
||||||
//set the header for the uncompressed bytes (for cache-on-write)
|
// set the header for the uncompressed bytes (for cache-on-write)
|
||||||
putHeader(uncompressedBytesWithHeader, 0,
|
putHeader(uncompressedBytesWithHeader, 0,
|
||||||
onDiskBytesWithHeader.length + numBytes,
|
onDiskBytesWithHeader.length + numBytes,
|
||||||
uncompressedBytesWithHeader.length, onDiskBytesWithHeader.length);
|
uncompressedBytesWithHeader.length, onDiskBytesWithHeader.length);
|
||||||
|
|
||||||
onDiskChecksum = new byte[numBytes];
|
onDiskChecksum = new byte[numBytes];
|
||||||
ChecksumUtil.generateChecksums(
|
ChecksumUtil.generateChecksums(
|
||||||
|
|
|
@ -221,7 +221,7 @@ public class HFileDataBlockEncoderImpl implements HFileDataBlockEncoder {
|
||||||
HFileBlockEncodingContext encodeCtx) {
|
HFileBlockEncodingContext encodeCtx) {
|
||||||
DataBlockEncoder encoder = algo.getEncoder();
|
DataBlockEncoder encoder = algo.getEncoder();
|
||||||
try {
|
try {
|
||||||
encoder.compressKeyValues(in, includesMemstoreTS, encodeCtx);
|
encoder.encodeKeyValues(in, includesMemstoreTS, encodeCtx);
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
throw new RuntimeException(String.format(
|
throw new RuntimeException(String.format(
|
||||||
"Bug in data block encoder "
|
"Bug in data block encoder "
|
||||||
|
|
|
@ -85,7 +85,7 @@ public class TestDataBlockEncoders {
|
||||||
HFileBlockEncodingContext encodingCtx =
|
HFileBlockEncodingContext encodingCtx =
|
||||||
getEncodingContext(Compression.Algorithm.NONE, encoding);
|
getEncodingContext(Compression.Algorithm.NONE, encoding);
|
||||||
|
|
||||||
encoder.compressKeyValues(dataset, includesMemstoreTS,
|
encoder.encodeKeyValues(dataset, includesMemstoreTS,
|
||||||
encodingCtx);
|
encodingCtx);
|
||||||
|
|
||||||
byte[] encodedBytesWithHeader =
|
byte[] encodedBytesWithHeader =
|
||||||
|
@ -106,7 +106,7 @@ public class TestDataBlockEncoders {
|
||||||
DataInputStream dis = new DataInputStream(bais);
|
DataInputStream dis = new DataInputStream(bais);
|
||||||
ByteBuffer actualDataset;
|
ByteBuffer actualDataset;
|
||||||
DataBlockEncoder encoder = encoding.getEncoder();
|
DataBlockEncoder encoder = encoding.getEncoder();
|
||||||
actualDataset = encoder.uncompressKeyValues(dis, includesMemstoreTS);
|
actualDataset = encoder.decodeKeyValues(dis, includesMemstoreTS);
|
||||||
|
|
||||||
dataset.rewind();
|
dataset.rewind();
|
||||||
actualDataset.rewind();
|
actualDataset.rewind();
|
||||||
|
|
|
@ -435,7 +435,7 @@ public class TestHFileBlock {
|
||||||
if (encoder != null) {
|
if (encoder != null) {
|
||||||
HFileBlockEncodingContext encodingCtx =
|
HFileBlockEncodingContext encodingCtx =
|
||||||
encoder.newDataBlockEncodingContext(algo, encoding, dummyHeader);
|
encoder.newDataBlockEncodingContext(algo, encoding, dummyHeader);
|
||||||
encoder.compressKeyValues(rawBuf, includesMemstoreTS,
|
encoder.encodeKeyValues(rawBuf, includesMemstoreTS,
|
||||||
encodingCtx);
|
encodingCtx);
|
||||||
encodedResultWithHeader =
|
encodedResultWithHeader =
|
||||||
encodingCtx.getUncompressedBytesWithHeader();
|
encodingCtx.getUncompressedBytesWithHeader();
|
||||||
|
|
|
@ -20,7 +20,7 @@ import java.io.ByteArrayInputStream;
|
||||||
import java.io.ByteArrayOutputStream;
|
import java.io.ByteArrayOutputStream;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import java.io.InputStream;
|
import java.io.InputStream;
|
||||||
import java.io.OutputStream;
|
import java.text.DecimalFormat;
|
||||||
import java.util.ArrayList;
|
import java.util.ArrayList;
|
||||||
import java.util.Iterator;
|
import java.util.Iterator;
|
||||||
import java.util.List;
|
import java.util.List;
|
||||||
|
@ -38,14 +38,15 @@ import org.apache.hadoop.fs.FileSystem;
|
||||||
import org.apache.hadoop.fs.Path;
|
import org.apache.hadoop.fs.Path;
|
||||||
import org.apache.hadoop.hbase.HBaseConfiguration;
|
import org.apache.hadoop.hbase.HBaseConfiguration;
|
||||||
import org.apache.hadoop.hbase.KeyValue;
|
import org.apache.hadoop.hbase.KeyValue;
|
||||||
import org.apache.hadoop.hbase.io.encoding.EncodedDataBlock;
|
|
||||||
import org.apache.hadoop.hbase.io.encoding.DataBlockEncoder;
|
import org.apache.hadoop.hbase.io.encoding.DataBlockEncoder;
|
||||||
import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding;
|
import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding;
|
||||||
|
import org.apache.hadoop.hbase.io.encoding.EncodedDataBlock;
|
||||||
import org.apache.hadoop.hbase.io.hfile.CacheConfig;
|
import org.apache.hadoop.hbase.io.hfile.CacheConfig;
|
||||||
import org.apache.hadoop.hbase.io.hfile.Compression;
|
import org.apache.hadoop.hbase.io.hfile.Compression;
|
||||||
import org.apache.hadoop.hbase.io.hfile.NoOpDataBlockEncoder;
|
|
||||||
import org.apache.hadoop.hbase.io.hfile.Compression.Algorithm;
|
import org.apache.hadoop.hbase.io.hfile.Compression.Algorithm;
|
||||||
|
import org.apache.hadoop.hbase.io.hfile.NoOpDataBlockEncoder;
|
||||||
import org.apache.hadoop.hbase.util.Bytes;
|
import org.apache.hadoop.hbase.util.Bytes;
|
||||||
|
import org.apache.hadoop.io.compress.CompressionOutputStream;
|
||||||
import org.apache.hadoop.io.compress.Compressor;
|
import org.apache.hadoop.io.compress.Compressor;
|
||||||
import org.apache.hadoop.io.compress.Decompressor;
|
import org.apache.hadoop.io.compress.Decompressor;
|
||||||
|
|
||||||
|
@ -60,33 +61,85 @@ public class DataBlockEncodingTool {
|
||||||
private static final boolean includesMemstoreTS = true;
|
private static final boolean includesMemstoreTS = true;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* How many times should benchmark run.
|
* How many times to run the benchmark. More times means better data in terms
|
||||||
* More times means better data in terms of statistics.
|
* of statistics but slower execution. Has to be strictly larger than
|
||||||
* It has to be larger than BENCHMARK_N_OMIT.
|
* {@link DEFAULT_BENCHMARK_N_OMIT}.
|
||||||
*/
|
*/
|
||||||
public static int BENCHMARK_N_TIMES = 12;
|
private static final int DEFAULT_BENCHMARK_N_TIMES = 12;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* How many first runs should omit benchmark.
|
* How many first runs should not be included in the benchmark. Done in order
|
||||||
* Usually it is one in order to exclude setup cost.
|
* to exclude setup cost.
|
||||||
* Has to be 0 or larger.
|
|
||||||
*/
|
*/
|
||||||
public static int BENCHMARK_N_OMIT = 2;
|
private static final int DEFAULT_BENCHMARK_N_OMIT = 2;
|
||||||
|
|
||||||
|
/** HFile name to be used in benchmark */
|
||||||
|
private static final String OPT_HFILE_NAME = "f";
|
||||||
|
|
||||||
|
/** Maximum number of key/value pairs to process in a single benchmark run */
|
||||||
|
private static final String OPT_KV_LIMIT = "n";
|
||||||
|
|
||||||
|
/** Whether to run a benchmark to measure read throughput */
|
||||||
|
private static final String OPT_MEASURE_THROUGHPUT = "b";
|
||||||
|
|
||||||
|
/** If this is specified, no correctness testing will be done */
|
||||||
|
private static final String OPT_OMIT_CORRECTNESS_TEST = "c";
|
||||||
|
|
||||||
|
/** What encoding algorithm to test */
|
||||||
|
private static final String OPT_ENCODING_ALGORITHM = "a";
|
||||||
|
|
||||||
|
/** Number of times to run each benchmark */
|
||||||
|
private static final String OPT_BENCHMARK_N_TIMES = "t";
|
||||||
|
|
||||||
|
/** Number of first runs of every benchmark to omit from statistics */
|
||||||
|
private static final String OPT_BENCHMARK_N_OMIT = "omit";
|
||||||
|
|
||||||
/** Compression algorithm to use if not specified on the command line */
|
/** Compression algorithm to use if not specified on the command line */
|
||||||
private static final Algorithm DEFAULT_COMPRESSION =
|
private static final Algorithm DEFAULT_COMPRESSION =
|
||||||
Compression.Algorithm.GZ;
|
Compression.Algorithm.GZ;
|
||||||
|
|
||||||
private List<EncodedDataBlock> codecs = new ArrayList<EncodedDataBlock>();
|
private static final DecimalFormat DELIMITED_DECIMAL_FORMAT =
|
||||||
private int totalPrefixLength = 0;
|
new DecimalFormat();
|
||||||
private int totalKeyLength = 0;
|
|
||||||
private int totalValueLength = 0;
|
|
||||||
private int totalKeyRedundancyLength = 0;
|
|
||||||
|
|
||||||
final private String compressionAlgorithmName;
|
static {
|
||||||
final private Algorithm compressionAlgorithm;
|
DELIMITED_DECIMAL_FORMAT.setGroupingSize(3);
|
||||||
final private Compressor compressor;
|
}
|
||||||
final private Decompressor decompressor;
|
|
||||||
|
private static final String PCT_FORMAT = "%.2f %%";
|
||||||
|
private static final String INT_FORMAT = "%d";
|
||||||
|
|
||||||
|
private static int benchmarkNTimes = DEFAULT_BENCHMARK_N_TIMES;
|
||||||
|
private static int benchmarkNOmit = DEFAULT_BENCHMARK_N_OMIT;
|
||||||
|
|
||||||
|
private List<EncodedDataBlock> codecs = new ArrayList<EncodedDataBlock>();
|
||||||
|
private long totalPrefixLength = 0;
|
||||||
|
private long totalKeyLength = 0;
|
||||||
|
private long totalValueLength = 0;
|
||||||
|
private long totalKeyRedundancyLength = 0;
|
||||||
|
private long totalCFLength = 0;
|
||||||
|
|
||||||
|
private byte[] rawKVs;
|
||||||
|
|
||||||
|
private final String compressionAlgorithmName;
|
||||||
|
private final Algorithm compressionAlgorithm;
|
||||||
|
private final Compressor compressor;
|
||||||
|
private final Decompressor decompressor;
|
||||||
|
|
||||||
|
private static enum Manipulation {
|
||||||
|
ENCODING,
|
||||||
|
DECODING,
|
||||||
|
COMPRESSION,
|
||||||
|
DECOMPRESSION;
|
||||||
|
|
||||||
|
@Override
|
||||||
|
public String toString() {
|
||||||
|
String s = super.toString();
|
||||||
|
StringBuilder sb = new StringBuilder();
|
||||||
|
sb.append(s.charAt(0));
|
||||||
|
sb.append(s.substring(1).toLowerCase());
|
||||||
|
return sb.toString();
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* @param compressionAlgorithmName What kind of algorithm should be used
|
* @param compressionAlgorithmName What kind of algorithm should be used
|
||||||
|
@ -110,22 +163,21 @@ public class DataBlockEncodingTool {
|
||||||
throws IOException {
|
throws IOException {
|
||||||
scanner.seek(KeyValue.LOWESTKEY);
|
scanner.seek(KeyValue.LOWESTKEY);
|
||||||
|
|
||||||
KeyValue currentKv;
|
KeyValue currentKV;
|
||||||
|
|
||||||
byte[] previousKey = null;
|
byte[] previousKey = null;
|
||||||
byte[] currentKey;
|
byte[] currentKey;
|
||||||
|
|
||||||
DataBlockEncoding[] encodings = DataBlockEncoding.values();
|
DataBlockEncoding[] encodings = DataBlockEncoding.values();
|
||||||
for(DataBlockEncoding encoding : encodings) {
|
|
||||||
DataBlockEncoder d = encoding.getEncoder();
|
ByteArrayOutputStream uncompressedOutputStream =
|
||||||
codecs.add(new EncodedDataBlock(d, includesMemstoreTS, encoding));
|
new ByteArrayOutputStream();
|
||||||
}
|
|
||||||
|
|
||||||
int j = 0;
|
int j = 0;
|
||||||
while ((currentKv = scanner.next()) != null && j < kvLimit) {
|
while ((currentKV = scanner.next()) != null && j < kvLimit) {
|
||||||
// Iterates through key/value pairs
|
// Iterates through key/value pairs
|
||||||
j++;
|
j++;
|
||||||
currentKey = currentKv.getKey();
|
currentKey = currentKV.getKey();
|
||||||
if (previousKey != null) {
|
if (previousKey != null) {
|
||||||
for (int i = 0; i < previousKey.length && i < currentKey.length &&
|
for (int i = 0; i < previousKey.length && i < currentKey.length &&
|
||||||
previousKey[i] == currentKey[i]; ++i) {
|
previousKey[i] == currentKey[i]; ++i) {
|
||||||
|
@ -133,22 +185,36 @@ public class DataBlockEncodingTool {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
for (EncodedDataBlock codec : codecs) {
|
uncompressedOutputStream.write(currentKV.getBuffer(),
|
||||||
codec.addKv(currentKv);
|
currentKV.getOffset(), currentKV.getLength());
|
||||||
}
|
|
||||||
|
|
||||||
previousKey = currentKey;
|
previousKey = currentKey;
|
||||||
|
|
||||||
totalPrefixLength += currentKv.getLength() - currentKv.getKeyLength() -
|
int kLen = currentKV.getKeyLength();
|
||||||
currentKv.getValueLength();
|
int vLen = currentKV.getValueLength();
|
||||||
totalKeyLength += currentKv.getKeyLength();
|
int cfLen = currentKV.getFamilyLength(currentKV.getFamilyOffset());
|
||||||
totalValueLength += currentKv.getValueLength();
|
int restLen = currentKV.getLength() - kLen - vLen;
|
||||||
|
|
||||||
|
totalKeyLength += kLen;
|
||||||
|
totalValueLength += vLen;
|
||||||
|
totalPrefixLength += restLen;
|
||||||
|
totalCFLength += cfLen;
|
||||||
|
}
|
||||||
|
|
||||||
|
rawKVs = uncompressedOutputStream.toByteArray();
|
||||||
|
|
||||||
|
for (DataBlockEncoding encoding : encodings) {
|
||||||
|
if (encoding == DataBlockEncoding.NONE) {
|
||||||
|
continue;
|
||||||
|
}
|
||||||
|
DataBlockEncoder d = encoding.getEncoder();
|
||||||
|
codecs.add(new EncodedDataBlock(d, includesMemstoreTS, encoding, rawKVs));
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Verify if all data block encoders are working properly.
|
* Verify if all data block encoders are working properly.
|
||||||
*
|
*
|
||||||
* @param scanner Of file which was compressed.
|
* @param scanner Of file which was compressed.
|
||||||
* @param kvLimit Maximal count of KeyValue which will be processed.
|
* @param kvLimit Maximal count of KeyValue which will be processed.
|
||||||
* @return true if all data block encoders compressed/decompressed correctly.
|
* @return true if all data block encoders compressed/decompressed correctly.
|
||||||
|
@ -226,15 +292,14 @@ public class DataBlockEncodingTool {
|
||||||
/**
|
/**
|
||||||
* Benchmark codec's speed.
|
* Benchmark codec's speed.
|
||||||
*/
|
*/
|
||||||
public void benchmarkCodecs() {
|
public void benchmarkCodecs() throws IOException {
|
||||||
|
LOG.info("Starting a throughput benchmark for data block encoding codecs");
|
||||||
int prevTotalSize = -1;
|
int prevTotalSize = -1;
|
||||||
for (EncodedDataBlock codec : codecs) {
|
for (EncodedDataBlock codec : codecs) {
|
||||||
prevTotalSize = benchmarkEncoder(prevTotalSize, codec);
|
prevTotalSize = benchmarkEncoder(prevTotalSize, codec);
|
||||||
}
|
}
|
||||||
|
|
||||||
byte[] buffer = codecs.get(0).getRawKeyValues();
|
benchmarkDefaultCompression(prevTotalSize, rawKVs);
|
||||||
|
|
||||||
benchmarkDefaultCompression(prevTotalSize, buffer);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
@ -250,7 +315,7 @@ public class DataBlockEncodingTool {
|
||||||
|
|
||||||
// decompression time
|
// decompression time
|
||||||
List<Long> durations = new ArrayList<Long>();
|
List<Long> durations = new ArrayList<Long>();
|
||||||
for (int itTime = 0; itTime < BENCHMARK_N_TIMES; ++itTime) {
|
for (int itTime = 0; itTime < benchmarkNTimes; ++itTime) {
|
||||||
totalSize = 0;
|
totalSize = 0;
|
||||||
|
|
||||||
Iterator<KeyValue> it;
|
Iterator<KeyValue> it;
|
||||||
|
@ -264,7 +329,7 @@ public class DataBlockEncodingTool {
|
||||||
totalSize += it.next().getLength();
|
totalSize += it.next().getLength();
|
||||||
}
|
}
|
||||||
final long finishTime = System.nanoTime();
|
final long finishTime = System.nanoTime();
|
||||||
if (itTime >= BENCHMARK_N_OMIT) {
|
if (itTime >= benchmarkNOmit) {
|
||||||
durations.add(finishTime - startTime);
|
durations.add(finishTime - startTime);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -275,26 +340,27 @@ public class DataBlockEncodingTool {
|
||||||
prevTotalSize = totalSize;
|
prevTotalSize = totalSize;
|
||||||
}
|
}
|
||||||
|
|
||||||
// compression time
|
List<Long> encodingDurations = new ArrayList<Long>();
|
||||||
List<Long> compressDurations = new ArrayList<Long>();
|
for (int itTime = 0; itTime < benchmarkNTimes; ++itTime) {
|
||||||
for (int itTime = 0; itTime < BENCHMARK_N_TIMES; ++itTime) {
|
|
||||||
final long startTime = System.nanoTime();
|
final long startTime = System.nanoTime();
|
||||||
codec.encodeData();
|
codec.encodeData();
|
||||||
final long finishTime = System.nanoTime();
|
final long finishTime = System.nanoTime();
|
||||||
if (itTime >= BENCHMARK_N_OMIT) {
|
if (itTime >= benchmarkNOmit) {
|
||||||
compressDurations.add(finishTime - startTime);
|
encodingDurations.add(finishTime - startTime);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
System.out.println(codec.toString() + ":");
|
System.out.println(codec.toString() + ":");
|
||||||
printBenchmarkResult(totalSize, compressDurations, false);
|
printBenchmarkResult(totalSize, encodingDurations, Manipulation.ENCODING);
|
||||||
printBenchmarkResult(totalSize, durations, true);
|
printBenchmarkResult(totalSize, durations, Manipulation.DECODING);
|
||||||
|
System.out.println();
|
||||||
|
|
||||||
return prevTotalSize;
|
return prevTotalSize;
|
||||||
}
|
}
|
||||||
|
|
||||||
private void benchmarkDefaultCompression(int totalSize, byte[] rawBuffer) {
|
private void benchmarkDefaultCompression(int totalSize, byte[] rawBuffer)
|
||||||
benchmarkAlgorithm(compressionAlgorithm, compressor, decompressor,
|
throws IOException {
|
||||||
|
benchmarkAlgorithm(compressionAlgorithm,
|
||||||
compressionAlgorithmName.toUpperCase(), rawBuffer, 0, totalSize);
|
compressionAlgorithmName.toUpperCase(), rawBuffer, 0, totalSize);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -307,24 +373,22 @@ public class DataBlockEncodingTool {
|
||||||
* @param buffer Buffer to be compressed.
|
* @param buffer Buffer to be compressed.
|
||||||
* @param offset Position of the beginning of the data.
|
* @param offset Position of the beginning of the data.
|
||||||
* @param length Length of data in buffer.
|
* @param length Length of data in buffer.
|
||||||
|
* @throws IOException
|
||||||
*/
|
*/
|
||||||
public static void benchmarkAlgorithm(
|
public void benchmarkAlgorithm(Compression.Algorithm algorithm, String name,
|
||||||
Compression.Algorithm algorithm,
|
byte[] buffer, int offset, int length) throws IOException {
|
||||||
Compressor compressorCodec,
|
|
||||||
Decompressor decompressorCodec,
|
|
||||||
String name,
|
|
||||||
byte[] buffer, int offset, int length) {
|
|
||||||
System.out.println(name + ":");
|
System.out.println(name + ":");
|
||||||
|
|
||||||
// compress it
|
// compress it
|
||||||
List<Long> compressDurations = new ArrayList<Long>();
|
List<Long> compressDurations = new ArrayList<Long>();
|
||||||
ByteArrayOutputStream compressedStream = new ByteArrayOutputStream();
|
ByteArrayOutputStream compressedStream = new ByteArrayOutputStream();
|
||||||
OutputStream compressingStream;
|
CompressionOutputStream compressingStream =
|
||||||
|
algorithm.createPlainCompressionStream(compressedStream, compressor);
|
||||||
try {
|
try {
|
||||||
for (int itTime = 0; itTime < BENCHMARK_N_TIMES; ++itTime) {
|
for (int itTime = 0; itTime < benchmarkNTimes; ++itTime) {
|
||||||
final long startTime = System.nanoTime();
|
final long startTime = System.nanoTime();
|
||||||
compressingStream = algorithm.createCompressionStream(
|
compressingStream.resetState();
|
||||||
compressedStream, compressorCodec, 0);
|
compressedStream.reset();
|
||||||
compressingStream.write(buffer, offset, length);
|
compressingStream.write(buffer, offset, length);
|
||||||
compressingStream.flush();
|
compressingStream.flush();
|
||||||
compressedStream.toByteArray();
|
compressedStream.toByteArray();
|
||||||
|
@ -332,36 +396,31 @@ public class DataBlockEncodingTool {
|
||||||
final long finishTime = System.nanoTime();
|
final long finishTime = System.nanoTime();
|
||||||
|
|
||||||
// add time record
|
// add time record
|
||||||
if (itTime >= BENCHMARK_N_OMIT) {
|
if (itTime >= benchmarkNOmit) {
|
||||||
compressDurations.add(finishTime - startTime);
|
compressDurations.add(finishTime - startTime);
|
||||||
}
|
}
|
||||||
|
|
||||||
if (itTime + 1 < BENCHMARK_N_TIMES) { // not the last one
|
|
||||||
compressedStream.reset();
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
throw new RuntimeException(String.format(
|
throw new RuntimeException(String.format(
|
||||||
"Benchmark, or encoding algorithm '%s' cause some stream problems",
|
"Benchmark, or encoding algorithm '%s' cause some stream problems",
|
||||||
name), e);
|
name), e);
|
||||||
}
|
}
|
||||||
printBenchmarkResult(length, compressDurations, false);
|
compressingStream.close();
|
||||||
|
printBenchmarkResult(length, compressDurations, Manipulation.COMPRESSION);
|
||||||
|
|
||||||
byte[] compBuffer = compressedStream.toByteArray();
|
byte[] compBuffer = compressedStream.toByteArray();
|
||||||
|
|
||||||
// uncompress it several times and measure performance
|
// uncompress it several times and measure performance
|
||||||
List<Long> durations = new ArrayList<Long>();
|
List<Long> durations = new ArrayList<Long>();
|
||||||
for (int itTime = 0; itTime < BENCHMARK_N_TIMES; ++itTime) {
|
for (int itTime = 0; itTime < benchmarkNTimes; ++itTime) {
|
||||||
final long startTime = System.nanoTime();
|
final long startTime = System.nanoTime();
|
||||||
byte[] newBuf = new byte[length + 1];
|
byte[] newBuf = new byte[length + 1];
|
||||||
|
|
||||||
try {
|
try {
|
||||||
|
|
||||||
ByteArrayInputStream downStream = new ByteArrayInputStream(compBuffer,
|
ByteArrayInputStream downStream = new ByteArrayInputStream(compBuffer,
|
||||||
0, compBuffer.length);
|
0, compBuffer.length);
|
||||||
InputStream decompressedStream = algorithm.createDecompressionStream(
|
InputStream decompressedStream = algorithm.createDecompressionStream(
|
||||||
downStream, decompressorCodec, 0);
|
downStream, decompressor, 0);
|
||||||
|
|
||||||
int destOffset = 0;
|
int destOffset = 0;
|
||||||
int nextChunk;
|
int nextChunk;
|
||||||
|
@ -370,7 +429,7 @@ public class DataBlockEncodingTool {
|
||||||
}
|
}
|
||||||
decompressedStream.close();
|
decompressedStream.close();
|
||||||
|
|
||||||
// iterate over KeyValue
|
// iterate over KeyValues
|
||||||
KeyValue kv;
|
KeyValue kv;
|
||||||
for (int pos = 0; pos < length; pos += kv.getLength()) {
|
for (int pos = 0; pos < length; pos += kv.getLength()) {
|
||||||
kv = new KeyValue(newBuf, pos);
|
kv = new KeyValue(newBuf, pos);
|
||||||
|
@ -396,91 +455,116 @@ public class DataBlockEncodingTool {
|
||||||
}
|
}
|
||||||
|
|
||||||
// add time record
|
// add time record
|
||||||
if (itTime >= BENCHMARK_N_OMIT) {
|
if (itTime >= benchmarkNOmit) {
|
||||||
durations.add(finishTime - startTime);
|
durations.add(finishTime - startTime);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
printBenchmarkResult(length, durations, true);
|
printBenchmarkResult(length, durations, Manipulation.DECOMPRESSION);
|
||||||
|
System.out.println();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
private static final double BYTES_IN_MB = 1024 * 1024.0;
|
||||||
|
private static final double NS_IN_SEC = 1000.0 * 1000.0 * 1000.0;
|
||||||
|
private static final double MB_SEC_COEF = NS_IN_SEC / BYTES_IN_MB;
|
||||||
|
|
||||||
private static void printBenchmarkResult(int totalSize,
|
private static void printBenchmarkResult(int totalSize,
|
||||||
List<Long> durationsInNanoSed, boolean isDecompression) {
|
List<Long> durationsInNanoSec, Manipulation manipulation) {
|
||||||
|
final int n = durationsInNanoSec.size();
|
||||||
long meanTime = 0;
|
long meanTime = 0;
|
||||||
for (long time : durationsInNanoSed) {
|
for (long time : durationsInNanoSec) {
|
||||||
meanTime += time;
|
meanTime += time;
|
||||||
}
|
}
|
||||||
meanTime /= durationsInNanoSed.size();
|
meanTime /= n;
|
||||||
|
|
||||||
long standardDev = 0;
|
double meanMBPerSec = totalSize * MB_SEC_COEF / meanTime;
|
||||||
for (long time : durationsInNanoSed) {
|
double mbPerSecSTD = 0;
|
||||||
standardDev += (time - meanTime) * (time - meanTime);
|
if (n > 0) {
|
||||||
|
for (long time : durationsInNanoSec) {
|
||||||
|
double mbPerSec = totalSize * MB_SEC_COEF / time;
|
||||||
|
double dev = mbPerSec - meanMBPerSec;
|
||||||
|
mbPerSecSTD += dev * dev;
|
||||||
|
}
|
||||||
|
mbPerSecSTD = Math.sqrt(mbPerSecSTD / n);
|
||||||
}
|
}
|
||||||
standardDev = (long) Math.sqrt(standardDev / durationsInNanoSed.size());
|
|
||||||
|
|
||||||
final double million = 1000.0 * 1000.0 * 1000.0;
|
outputTuple(manipulation + " performance", "%6.2f MB/s (+/- %.2f MB/s)",
|
||||||
double mbPerSec = (totalSize * million) / (1024.0 * 1024.0 * meanTime);
|
meanMBPerSec, mbPerSecSTD);
|
||||||
double mbPerSecDev = (totalSize * million) /
|
}
|
||||||
(1024.0 * 1024.0 * (meanTime - standardDev));
|
|
||||||
|
|
||||||
System.out.println(String.format(
|
private static void outputTuple(String caption, String format,
|
||||||
" %s performance:%s %6.2f MB/s (+/- %.2f MB/s)",
|
Object... values) {
|
||||||
isDecompression ? "Decompression" : "Compression",
|
if (format.startsWith(INT_FORMAT)) {
|
||||||
isDecompression ? "" : " ",
|
format = "%s" + format.substring(INT_FORMAT.length());
|
||||||
mbPerSec, mbPerSecDev - mbPerSec));
|
values[0] = DELIMITED_DECIMAL_FORMAT.format(values[0]);
|
||||||
|
}
|
||||||
|
|
||||||
|
StringBuilder sb = new StringBuilder();
|
||||||
|
sb.append(" ");
|
||||||
|
sb.append(caption);
|
||||||
|
sb.append(":");
|
||||||
|
|
||||||
|
String v = String.format(format, values);
|
||||||
|
int padding = 60 - sb.length() - v.length();
|
||||||
|
for (int i = 0; i < padding; ++i) {
|
||||||
|
sb.append(' ');
|
||||||
|
}
|
||||||
|
sb.append(v);
|
||||||
|
System.out.println(sb);
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Display statistics of different compression algorithms.
|
* Display statistics of different compression algorithms.
|
||||||
|
* @throws IOException
|
||||||
*/
|
*/
|
||||||
public void displayStatistics() {
|
public void displayStatistics() throws IOException {
|
||||||
int totalLength = totalPrefixLength + totalKeyLength + totalValueLength;
|
final String comprAlgo = compressionAlgorithmName.toUpperCase();
|
||||||
if (compressor != null) { // might be null e.g. for pure-Java GZIP
|
long rawBytes = totalKeyLength + totalPrefixLength + totalValueLength;
|
||||||
compressor.reset();
|
|
||||||
}
|
|
||||||
|
|
||||||
for(EncodedDataBlock codec : codecs) {
|
System.out.println("Raw data size:");
|
||||||
|
outputTuple("Raw bytes", INT_FORMAT, rawBytes);
|
||||||
|
outputTuplePct("Key bytes", totalKeyLength);
|
||||||
|
outputTuplePct("Value bytes", totalValueLength);
|
||||||
|
outputTuplePct("KV infrastructure", totalPrefixLength);
|
||||||
|
outputTuplePct("CF overhead", totalCFLength);
|
||||||
|
outputTuplePct("Total key redundancy", totalKeyRedundancyLength);
|
||||||
|
|
||||||
|
int compressedSize = EncodedDataBlock.getCompressedSize(
|
||||||
|
compressionAlgorithm, compressor, rawKVs, 0, rawKVs.length);
|
||||||
|
outputTuple(comprAlgo + " only size", INT_FORMAT,
|
||||||
|
compressedSize);
|
||||||
|
outputSavings(comprAlgo + " only", compressedSize, rawBytes);
|
||||||
|
System.out.println();
|
||||||
|
|
||||||
|
for (EncodedDataBlock codec : codecs) {
|
||||||
System.out.println(codec.toString());
|
System.out.println(codec.toString());
|
||||||
int saved = totalKeyLength + totalPrefixLength + totalValueLength
|
long encodedBytes = codec.getSize();
|
||||||
- codec.getSize();
|
outputTuple("Encoded bytes", INT_FORMAT, encodedBytes);
|
||||||
System.out.println(
|
outputSavings("Key encoding", encodedBytes - totalValueLength,
|
||||||
String.format(" Saved bytes: %8d", saved));
|
rawBytes - totalValueLength);
|
||||||
double keyRatio = (saved * 100.0) / (totalPrefixLength + totalKeyLength);
|
outputSavings("Total encoding", encodedBytes, rawBytes);
|
||||||
double allRatio = (saved * 100.0) / totalLength;
|
|
||||||
System.out.println(
|
|
||||||
String.format(" Key compression ratio: %.2f %%", keyRatio));
|
|
||||||
System.out.println(
|
|
||||||
String.format(" All compression ratio: %.2f %%", allRatio));
|
|
||||||
|
|
||||||
String compressedSizeCaption =
|
int encodedCompressedSize = codec.getEncodedCompressedSize(
|
||||||
String.format(" %s compressed size: ",
|
compressionAlgorithm, compressor);
|
||||||
compressionAlgorithmName.toUpperCase());
|
outputTuple("Encoding + " + comprAlgo + " size", INT_FORMAT,
|
||||||
String compressOnlyRatioCaption =
|
encodedCompressedSize);
|
||||||
String.format(" %s compression ratio: ",
|
outputSavings("Encoding + " + comprAlgo, encodedCompressedSize, rawBytes);
|
||||||
compressionAlgorithmName.toUpperCase());
|
outputSavings("Encoding with " + comprAlgo, encodedCompressedSize,
|
||||||
|
compressedSize);
|
||||||
|
|
||||||
if (compressor != null) {
|
System.out.println();
|
||||||
int compressedSize = codec.checkCompressedSize(compressor);
|
|
||||||
System.out.println(compressedSizeCaption +
|
|
||||||
String.format("%8d", compressedSize));
|
|
||||||
double compressOnlyRatio =
|
|
||||||
100.0 * (1.0 - compressedSize / (0.0 + totalLength));
|
|
||||||
System.out.println(compressOnlyRatioCaption
|
|
||||||
+ String.format("%.2f %%", compressOnlyRatio));
|
|
||||||
} else {
|
|
||||||
System.out.println(compressedSizeCaption + "N/A");
|
|
||||||
System.out.println(compressOnlyRatioCaption + "N/A");
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
}
|
||||||
|
|
||||||
System.out.println(
|
private void outputTuplePct(String caption, long size) {
|
||||||
String.format("Total KV prefix length: %8d", totalPrefixLength));
|
outputTuple(caption, INT_FORMAT + " (" + PCT_FORMAT + ")",
|
||||||
System.out.println(
|
size, size * 100.0 / rawKVs.length);
|
||||||
String.format("Total key length: %8d", totalKeyLength));
|
}
|
||||||
System.out.println(
|
|
||||||
String.format("Total key redundancy: %8d",
|
private void outputSavings(String caption, long part, long whole) {
|
||||||
totalKeyRedundancyLength));
|
double pct = 100.0 * (1 - 1.0 * part / whole);
|
||||||
System.out.println(
|
double times = whole * 1.0 / part;
|
||||||
String.format("Total value length: %8d", totalValueLength));
|
outputTuple(caption + " savings", PCT_FORMAT + " (%.2f x)",
|
||||||
|
pct, times);
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
@ -540,22 +624,35 @@ public class DataBlockEncodingTool {
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* A command line interface to benchmarks.
|
* A command line interface to benchmarks. Parses command-line arguments and
|
||||||
|
* runs the appropriate benchmarks.
|
||||||
* @param args Should have length at least 1 and holds the file path to HFile.
|
* @param args Should have length at least 1 and holds the file path to HFile.
|
||||||
* @throws IOException If you specified the wrong file.
|
* @throws IOException If you specified the wrong file.
|
||||||
*/
|
*/
|
||||||
public static void main(final String[] args) throws IOException {
|
public static void main(final String[] args) throws IOException {
|
||||||
// set up user arguments
|
// set up user arguments
|
||||||
Options options = new Options();
|
Options options = new Options();
|
||||||
options.addOption("f", true, "HFile to analyse (REQUIRED)");
|
options.addOption(OPT_HFILE_NAME, true, "HFile to analyse (REQUIRED)");
|
||||||
options.getOption("f").setArgName("FILENAME");
|
options.getOption(OPT_HFILE_NAME).setArgName("FILENAME");
|
||||||
options.addOption("n", true,
|
options.addOption(OPT_KV_LIMIT, true,
|
||||||
"Limit number of KeyValue which will be analysed");
|
"Maximum number of KeyValues to process. A benchmark stops running " +
|
||||||
options.getOption("n").setArgName("NUMBER");
|
"after iterating over this many KV pairs.");
|
||||||
options.addOption("b", false, "Measure read throughput");
|
options.getOption(OPT_KV_LIMIT).setArgName("NUMBER");
|
||||||
options.addOption("c", false, "Omit corectness tests.");
|
options.addOption(OPT_MEASURE_THROUGHPUT, false,
|
||||||
options.addOption("a", true,
|
"Measure read throughput");
|
||||||
|
options.addOption(OPT_OMIT_CORRECTNESS_TEST, false,
|
||||||
|
"Omit corectness tests.");
|
||||||
|
options.addOption(OPT_ENCODING_ALGORITHM, true,
|
||||||
"What kind of compression algorithm use for comparison.");
|
"What kind of compression algorithm use for comparison.");
|
||||||
|
options.addOption(OPT_BENCHMARK_N_TIMES,
|
||||||
|
true, "Number of times to run each benchmark. Default value: " +
|
||||||
|
DEFAULT_BENCHMARK_N_TIMES);
|
||||||
|
options.addOption(OPT_BENCHMARK_N_OMIT, true,
|
||||||
|
"Number of first runs of every benchmark to exclude from "
|
||||||
|
+ "statistics (" + DEFAULT_BENCHMARK_N_OMIT
|
||||||
|
+ " by default, so that " + "only the last "
|
||||||
|
+ (DEFAULT_BENCHMARK_N_TIMES - DEFAULT_BENCHMARK_N_OMIT)
|
||||||
|
+ " times are included in statistics.)");
|
||||||
|
|
||||||
// parse arguments
|
// parse arguments
|
||||||
CommandLineParser parser = new PosixParser();
|
CommandLineParser parser = new PosixParser();
|
||||||
|
@ -569,24 +666,44 @@ public class DataBlockEncodingTool {
|
||||||
}
|
}
|
||||||
|
|
||||||
int kvLimit = Integer.MAX_VALUE;
|
int kvLimit = Integer.MAX_VALUE;
|
||||||
if (cmd.hasOption("n")) {
|
if (cmd.hasOption(OPT_KV_LIMIT)) {
|
||||||
kvLimit = Integer.parseInt(cmd.getOptionValue("n"));
|
kvLimit = Integer.parseInt(cmd.getOptionValue(OPT_KV_LIMIT));
|
||||||
}
|
}
|
||||||
|
|
||||||
// basic argument sanity checks
|
// basic argument sanity checks
|
||||||
if (!cmd.hasOption("f")) {
|
if (!cmd.hasOption(OPT_HFILE_NAME)) {
|
||||||
System.err.println("ERROR: Filename is required!");
|
LOG.error("Please specify HFile name using the " + OPT_HFILE_NAME
|
||||||
|
+ " option");
|
||||||
printUsage(options);
|
printUsage(options);
|
||||||
System.exit(-1);
|
System.exit(-1);
|
||||||
}
|
}
|
||||||
|
|
||||||
String pathName = cmd.getOptionValue("f");
|
String pathName = cmd.getOptionValue(OPT_HFILE_NAME);
|
||||||
String compressionName = DEFAULT_COMPRESSION.getName();
|
String compressionName = DEFAULT_COMPRESSION.getName();
|
||||||
if (cmd.hasOption("a")) {
|
if (cmd.hasOption(OPT_ENCODING_ALGORITHM)) {
|
||||||
compressionName = cmd.getOptionValue("a").toLowerCase();
|
compressionName =
|
||||||
|
cmd.getOptionValue(OPT_ENCODING_ALGORITHM).toLowerCase();
|
||||||
}
|
}
|
||||||
boolean doBenchmark = cmd.hasOption("b");
|
boolean doBenchmark = cmd.hasOption(OPT_MEASURE_THROUGHPUT);
|
||||||
boolean doVerify = !cmd.hasOption("c");
|
boolean doVerify = !cmd.hasOption(OPT_OMIT_CORRECTNESS_TEST);
|
||||||
|
|
||||||
|
if (cmd.hasOption(OPT_BENCHMARK_N_TIMES)) {
|
||||||
|
benchmarkNTimes = Integer.valueOf(cmd.getOptionValue(
|
||||||
|
OPT_BENCHMARK_N_TIMES));
|
||||||
|
}
|
||||||
|
if (cmd.hasOption(OPT_BENCHMARK_N_OMIT)) {
|
||||||
|
benchmarkNOmit =
|
||||||
|
Integer.valueOf(cmd.getOptionValue(OPT_BENCHMARK_N_OMIT));
|
||||||
|
}
|
||||||
|
if (benchmarkNTimes < benchmarkNOmit) {
|
||||||
|
LOG.error("The number of times to run each benchmark ("
|
||||||
|
+ benchmarkNTimes
|
||||||
|
+ ") must be greater than the number of benchmark runs to exclude "
|
||||||
|
+ "from statistics (" + benchmarkNOmit + ")");
|
||||||
|
System.exit(1);
|
||||||
|
}
|
||||||
|
LOG.info("Running benchmark " + benchmarkNTimes + " times. " +
|
||||||
|
"Excluding the first " + benchmarkNOmit + " times from statistics.");
|
||||||
|
|
||||||
final Configuration conf = HBaseConfiguration.create();
|
final Configuration conf = HBaseConfiguration.create();
|
||||||
try {
|
try {
|
||||||
|
|
|
@ -173,7 +173,6 @@ public class EncodedSeekPerformanceTest {
|
||||||
List<HFileDataBlockEncoder> encoders =
|
List<HFileDataBlockEncoder> encoders =
|
||||||
new ArrayList<HFileDataBlockEncoder>();
|
new ArrayList<HFileDataBlockEncoder>();
|
||||||
|
|
||||||
encoders.add(new HFileDataBlockEncoderImpl(DataBlockEncoding.NONE));
|
|
||||||
for (DataBlockEncoding encodingAlgo : DataBlockEncoding.values()) {
|
for (DataBlockEncoding encodingAlgo : DataBlockEncoding.values()) {
|
||||||
encoders.add(new HFileDataBlockEncoderImpl(DataBlockEncoding.NONE,
|
encoders.add(new HFileDataBlockEncoderImpl(DataBlockEncoding.NONE,
|
||||||
encodingAlgo));
|
encodingAlgo));
|
||||||
|
|
Loading…
Reference in New Issue