Skip to content

HTTPS clone URL

Subversion checkout URL

You can clone with HTTPS or Subversion.

Download ZIP
Browse files

Stats to understand NIO layer performance + BDB exception counts et al

  • Loading branch information...
commit dd29d0e6ba5c3f2734f42c41124f28d9cba70dab 1 parent fbe6718
@vinothchandar vinothchandar authored
Showing with 549 additions and 112 deletions.
  1. +2 −40 config/single_node_cluster/config/stores.xml
  2. +1 −1  src/java/voldemort/client/protocol/vold/VoldemortNativeClientRequestFormat.java
  3. +28 −3 src/java/voldemort/{utils → common/nio}/ByteBufferBackedInputStream.java
  4. +37 −5 src/java/voldemort/{utils → common/nio}/ByteBufferBackedOutputStream.java
  5. +28 −0 src/java/voldemort/common/nio/CommBufferSizeStats.java
  6. +23 −2 src/java/voldemort/{utils → common/nio}/SelectorManager.java
  7. +13 −4 src/java/voldemort/{utils → common/nio}/SelectorManagerWorker.java
  8. +24 −2 src/java/voldemort/server/VoldemortConfig.java
  9. +7 −9 src/java/voldemort/server/niosocket/AsyncRequestHandler.java
  10. +27 −7 src/java/voldemort/server/niosocket/NioSelectorManager.java
  11. +89 −0 src/java/voldemort/server/niosocket/NioSelectorManagerStats.java
  12. +71 −0 src/java/voldemort/server/niosocket/NioSocketService.java
  13. +1 −1  src/java/voldemort/server/protocol/admin/AdminServiceRequestHandler.java
  14. +1 −1  src/java/voldemort/server/protocol/vold/VoldemortNativeRequestHandler.java
  15. +2 −16 src/java/voldemort/store/bdb/BdbStorageConfiguration.java
  16. +23 −7 src/java/voldemort/store/bdb/BdbStorageEngine.java
  17. +4 −0 src/java/voldemort/store/bdb/PartitionPrefixedBdbStorageEngine.java
  18. +87 −1 src/java/voldemort/store/bdb/stats/BdbEnvironmentStats.java
  19. +5 −3 src/java/voldemort/store/socket/clientrequest/ClientRequestExecutor.java
  20. +1 −1  src/java/voldemort/store/socket/clientrequest/ClientRequestExecutorFactory.java
  21. +43 −0 src/java/voldemort/store/stats/Histogram.java
  22. +32 −9 test/unit/voldemort/store/stats/HistogramTest.java
View
42 config/single_node_cluster/config/stores.xml
@@ -14,44 +14,6 @@
</key-serializer>
<value-serializer>
<type>string</type>
- </value-serializer>
- </store>
- <store>
- <name>test-evolution</name>
- <persistence>bdb</persistence>
- <description>Test store</description>
- <owners>harry@hogwarts.edu, hermoine@hogwarts.edu</owners>
- <routing-strategy>consistent-routing</routing-strategy>
- <routing>client</routing>
- <replication-factor>1</replication-factor>
- <required-reads>1</required-reads>
- <required-writes>1</required-writes>
- <key-serializer>
- <type>string</type>
- </key-serializer>
- <value-serializer>
- <type>avro-generic-versioned</type>
- <schema-info version="0">{"type": "record", "name": "myrec","fields": [{ "name": "original", "type": "string" }]}</schema-info>
- <schema-info version="1">{"type": "record", "name": "myrec","fields": [{ "name": "original", "type": "string" }, { "name": "new-field", "type": "string", "default":"" }]}</schema-info>
- </value-serializer>
- </store>
- <store>
- <name>anagpal-test-old</name>
- <persistence>read-only</persistence>
- <description>"test store"</description>
- <owners>anagpal@linkedin.com</owners>
- <routing-strategy>consistent-routing</routing-strategy>
- <routing>client</routing>
- <replication-factor>1</replication-factor>
- <required-reads>1</required-reads>
- <required-writes>1</required-writes>
- <key-serializer>
- <type>json</type>
- <schema-info version="0">"string"</schema-info>
- </key-serializer>
- <value-serializer>
- <type>json</type>
- <schema-info version="0">{"cnt":"int32", "country":"string"}</schema-info>
- </value-serializer>
- </store>
+ </value-serializer>
+ </store>
</stores>
View
2  src/java/voldemort/client/protocol/vold/VoldemortNativeClientRequestFormat.java
@@ -30,11 +30,11 @@
import voldemort.VoldemortException;
import voldemort.client.protocol.RequestFormat;
import voldemort.common.VoldemortOpCode;
+import voldemort.common.nio.ByteBufferBackedInputStream;
import voldemort.server.RequestRoutingType;
import voldemort.store.ErrorCodeMapper;
import voldemort.store.StoreUtils;
import voldemort.utils.ByteArray;
-import voldemort.utils.ByteBufferBackedInputStream;
import voldemort.utils.ByteUtils;
import voldemort.versioning.VectorClock;
import voldemort.versioning.Version;
View
31 .../voldemort/utils/ByteBufferBackedInputStream.java → ...emort/common/nio/ByteBufferBackedInputStream.java
@@ -14,12 +14,14 @@
* the License.
*/
-package voldemort.utils;
+package voldemort.common.nio;
import java.io.IOException;
import java.io.InputStream;
import java.nio.ByteBuffer;
+import org.apache.commons.lang.mutable.MutableLong;
+
import voldemort.annotations.concurrency.NotThreadsafe;
/**
@@ -38,16 +40,34 @@
private ByteBuffer buffer;
+ /**
+ * Reference to a size tracking object, that tracks the size of the buffer
+ * in bytes
+ */
+ private MutableLong sizeTracker;
+
public ByteBufferBackedInputStream(ByteBuffer buffer) {
this.buffer = buffer;
+ this.sizeTracker = null;
+ }
+
+ public ByteBufferBackedInputStream(ByteBuffer buffer, MutableLong sizeTracker) {
+ this.buffer = buffer;
+ this.sizeTracker = sizeTracker;
+ this.sizeTracker.add(buffer.capacity());
}
public ByteBuffer getBuffer() {
return buffer;
}
- public void setBuffer(ByteBuffer buffer) {
- this.buffer = buffer;
+ public void setBuffer(ByteBuffer newBuffer) {
+ // update the size tracker with the new buffer size
+ if((sizeTracker != null && this.buffer != null && newBuffer != null)) {
+ sizeTracker.add(newBuffer.capacity());
+ sizeTracker.subtract(this.buffer.capacity());
+ }
+ this.buffer = newBuffer;
}
@Override
@@ -68,4 +88,9 @@ public int read(byte[] bytes, int off, int len) throws IOException {
return len;
}
+ public void close() {
+ if(sizeTracker != null && this.buffer != null) {
+ sizeTracker.subtract(this.buffer.capacity());
+ }
+ }
}
View
42 ...voldemort/utils/ByteBufferBackedOutputStream.java → ...mort/common/nio/ByteBufferBackedOutputStream.java
@@ -14,13 +14,16 @@
* the License.
*/
-package voldemort.utils;
+package voldemort.common.nio;
import java.io.IOException;
import java.io.OutputStream;
import java.nio.ByteBuffer;
+import org.apache.commons.lang.mutable.MutableLong;
+
import voldemort.annotations.concurrency.NotThreadsafe;
+import voldemort.utils.ByteUtils;
/**
* ByteBufferBackedOutputStream serves two purposes:
@@ -46,17 +49,36 @@
private boolean wasExpanded;
+ /**
+ * Reference to a size tracking object, that tracks the size of the buffer
+ * in bytes
+ */
+ private MutableLong sizeTracker;
+
public ByteBufferBackedOutputStream(ByteBuffer buffer) {
this.buffer = buffer;
wasExpanded = false;
+ this.sizeTracker = null;
+ }
+
+ public ByteBufferBackedOutputStream(ByteBuffer buffer, MutableLong sizeTracker) {
+ this.buffer = buffer;
+ wasExpanded = false;
+ this.sizeTracker = sizeTracker;
+ this.sizeTracker.add(buffer.capacity());
}
public ByteBuffer getBuffer() {
return buffer;
}
- public void setBuffer(ByteBuffer buffer) {
- this.buffer = buffer;
+ public void setBuffer(ByteBuffer newBuffer) {
+ // update the size tracker with the new buffer size
+ if((sizeTracker != null && this.buffer != null && newBuffer != null)) {
+ sizeTracker.add(newBuffer.capacity());
+ sizeTracker.subtract(this.buffer.capacity());
+ }
+ this.buffer = newBuffer;
wasExpanded = false;
}
@@ -78,8 +100,13 @@ private void expandIfNeeded(int len) {
if(need <= 0)
return;
- int newCapacity = buffer.capacity() + need;
- buffer = ByteUtils.expand(buffer, newCapacity * 2);
+ int newCapacity = (buffer.capacity() + need) * 2;
+ // update the size tracker with the new buffer size
+ if(sizeTracker != null) {
+ sizeTracker.add(newCapacity);
+ sizeTracker.subtract(this.buffer.capacity());
+ }
+ buffer = ByteUtils.expand(buffer, newCapacity);
wasExpanded = true;
}
@@ -87,4 +114,9 @@ public boolean wasExpanded() {
return wasExpanded;
}
+ public void close() {
+ if(sizeTracker != null && this.buffer != null) {
+ sizeTracker.subtract(this.buffer.capacity());
+ }
+ }
}
View
28 src/java/voldemort/common/nio/CommBufferSizeStats.java
@@ -0,0 +1,28 @@
+package voldemort.common.nio;
+
+import org.apache.commons.lang.mutable.MutableLong;
+
+/**
+ * Statistics object to track the communication buffer sizes across all the
+ * connections, handled by the selector managers
+ *
+ */
+public class CommBufferSizeStats {
+
+ private MutableLong commReadBufferSizeBytes;
+
+ private MutableLong commWriteBufferSizeBytes;
+
+ public CommBufferSizeStats() {
+ commReadBufferSizeBytes = new MutableLong(0);
+ commWriteBufferSizeBytes = new MutableLong(0);
+ }
+
+ public MutableLong getCommReadBufferSizeTracker() {
+ return commReadBufferSizeBytes;
+ }
+
+ public MutableLong getCommWriteBufferSizeTracker() {
+ return commWriteBufferSizeBytes;
+ }
+}
View
25 src/java/voldemort/utils/SelectorManager.java → src/java/voldemort/common/nio/SelectorManager.java
@@ -14,7 +14,7 @@
* the License.
*/
-package voldemort.utils;
+package voldemort.common.nio;
import java.io.IOException;
import java.nio.channels.ClosedSelectorException;
@@ -98,6 +98,23 @@
protected final Logger logger = Logger.getLogger(getClass());
+ // statistics about the current select loop
+ /**
+ * Number of connections selected (meaning they have some data to be
+ * read/written) in the current processing loop
+ */
+ protected int selectCount = -1;
+ /**
+ * Amount of time taken to process all the connections selected in this
+ * processing loop
+ */
+ protected long processingTimeMs = -1;
+ /**
+ * Amount of time spent in the select() call. This is an indicator of how
+ * busy the thread is
+ */
+ protected long selectTimeMs = -1;
+
public SelectorManager() {
try {
this.selector = Selector.open();
@@ -172,7 +189,10 @@ public void run() {
processEvents();
try {
+ selectTimeMs = System.currentTimeMillis();
int selected = selector.select(SELECTOR_POLL_MS);
+ selectTimeMs = System.currentTimeMillis() - selectTimeMs;
+ selectCount = selected;
if(isClosed.get()) {
if(logger.isInfoEnabled())
@@ -182,6 +202,7 @@ public void run() {
}
if(selected > 0) {
+ processingTimeMs = System.currentTimeMillis();
Iterator<SelectionKey> i = selector.selectedKeys().iterator();
while(i.hasNext()) {
@@ -194,6 +215,7 @@ public void run() {
worker.run();
}
}
+ processingTimeMs = System.currentTimeMillis() - processingTimeMs;
}
} catch(ClosedSelectorException e) {
if(logger.isDebugEnabled())
@@ -217,5 +239,4 @@ public void run() {
}
}
}
-
}
View
17 src/java/voldemort/utils/SelectorManagerWorker.java → ...a/voldemort/common/nio/SelectorManagerWorker.java
@@ -14,7 +14,7 @@
* the License.
*/
-package voldemort.utils;
+package voldemort.common.nio;
import java.io.EOFException;
import java.io.IOException;
@@ -29,6 +29,8 @@
import org.apache.log4j.Level;
import org.apache.log4j.Logger;
+import voldemort.utils.ByteUtils;
+
/**
* SelectorManagerWorker manages a Selector, SocketChannel, and IO streams
* implementation. At the point that the run method is invoked, the Selector
@@ -62,13 +64,16 @@
public SelectorManagerWorker(Selector selector,
SocketChannel socketChannel,
- int socketBufferSize) {
+ int socketBufferSize,
+ CommBufferSizeStats commBufferStats) {
this.selector = selector;
this.socketChannel = socketChannel;
this.socketBufferSize = socketBufferSize;
this.resizeThreshold = socketBufferSize * 2; // This is arbitrary...
- this.inputStream = new ByteBufferBackedInputStream(ByteBuffer.allocate(socketBufferSize));
- this.outputStream = new ByteBufferBackedOutputStream(ByteBuffer.allocate(socketBufferSize));
+ this.inputStream = new ByteBufferBackedInputStream(ByteBuffer.allocate(socketBufferSize),
+ commBufferStats.getCommReadBufferSizeTracker());
+ this.outputStream = new ByteBufferBackedOutputStream(ByteBuffer.allocate(socketBufferSize),
+ commBufferStats.getCommWriteBufferSizeTracker());
this.createTimestamp = System.nanoTime();
this.isClosed = new AtomicBoolean(false);
@@ -162,6 +167,10 @@ protected void closeInternal() {
logger.warn(e.getMessage(), e);
}
}
+
+ // close the streams, so we account for comm buffer frees
+ inputStream.close();
+ outputStream.close();
}
public boolean isClosed() {
View
26 src/java/voldemort/server/VoldemortConfig.java
@@ -79,6 +79,7 @@
private int bdbCleanerMinFileUtilization;
private int bdbCleanerMinUtilization;
private int bdbCleanerLookAheadCacheSize;
+ private long bdbCleanerBytesInterval;
private boolean bdbCheckpointerHighPriority;
private int bdbCleanerMaxBatchFiles;
private boolean bdbReadUncommitted;
@@ -236,12 +237,15 @@ public VoldemortConfig(Props props) {
+ File.separator + "bdb");
this.bdbMaxLogFileSize = props.getBytes("bdb.max.logfile.size", 60 * 1024 * 1024);
this.bdbBtreeFanout = props.getInt("bdb.btree.fanout", 512);
- this.bdbCheckpointBytes = props.getLong("bdb.checkpoint.interval.bytes", 20 * 1024 * 1024);
+ this.bdbCheckpointBytes = props.getLong("bdb.checkpoint.interval.bytes", 200 * 1024 * 1024);
this.bdbCheckpointMs = props.getLong("bdb.checkpoint.interval.ms", 30 * Time.MS_PER_SECOND);
this.bdbOneEnvPerStore = props.getBoolean("bdb.one.env.per.store", false);
this.bdbCleanerMinFileUtilization = props.getInt("bdb.cleaner.min.file.utilization", 5);
this.bdbCleanerMinUtilization = props.getInt("bdb.cleaner.minUtilization", 50);
this.bdbCleanerThreads = props.getInt("bdb.cleaner.threads", 1);
+ // by default, wake up the cleaner everytime we write one whole log file
+ this.bdbCleanerBytesInterval = props.getLong("bdb.cleaner.interval.bytes",
+ this.bdbMaxLogFileSize);
this.bdbCleanerLookAheadCacheSize = props.getInt("bdb.cleaner.lookahead.cache.size", 8192);
this.bdbLockTimeoutMs = props.getLong("bdb.lock.timeout.ms", 500);
this.bdbLockNLockTables = props.getInt("bdb.lock.nLockTables", 1);
@@ -791,6 +795,24 @@ public final void setBdbCleanerThreads(int bdbCleanerThreads) {
}
/**
+ *
+ * Amount of bytes written before the Cleaner wakes up to check for
+ * utilization
+ *
+ * <ul>
+ * <li>property: "bdb.cleaner.interval.bytes"</li>
+ * <li>default: logfile size</li>
+ * </ul>
+ */
+ public long getBdbCleanerBytesInterval() {
+ return bdbCleanerBytesInterval;
+ }
+
+ public final void setCleanerBytesInterval(long bdbCleanerBytesInterval) {
+ this.bdbCleanerBytesInterval = bdbCleanerBytesInterval;
+ }
+
+ /**
* Buffer size used by cleaner to fetch BTree nodes during cleaning.
*
* <ul>
@@ -1086,7 +1108,7 @@ public void setBdbPrefixKeysWithPartitionId(boolean bdbPrefixKeysWithPartitionId
*
* <ul>
* <li>Property : "bdb.checkpoint.interval.bytes"</li>
- * <li>Default : 20MB</li>
+ * <li>Default : 200MB</li>
* </ul>
*
* @return
View
16 src/java/voldemort/server/niosocket/AsyncRequestHandler.java
@@ -25,18 +25,17 @@
import java.nio.channels.Selector;
import java.nio.channels.SocketChannel;
-import org.apache.commons.lang.mutable.MutableInt;
import org.apache.log4j.Level;
import voldemort.VoldemortException;
import voldemort.client.protocol.RequestFormatType;
+import voldemort.common.nio.SelectorManagerWorker;
import voldemort.server.protocol.RequestHandler;
import voldemort.server.protocol.RequestHandlerFactory;
import voldemort.server.protocol.StreamRequestHandler;
import voldemort.server.protocol.StreamRequestHandler.StreamRequestDirection;
import voldemort.server.protocol.StreamRequestHandler.StreamRequestHandlerState;
import voldemort.utils.ByteUtils;
-import voldemort.utils.SelectorManagerWorker;
/**
* AsyncRequestHandler manages a Selector, SocketChannel, and RequestHandler
@@ -60,16 +59,16 @@
private StreamRequestHandler streamRequestHandler;
- private MutableInt serverConnectionCount;
+ private NioSelectorManagerStats nioStats;
public AsyncRequestHandler(Selector selector,
SocketChannel socketChannel,
RequestHandlerFactory requestHandlerFactory,
int socketBufferSize,
- MutableInt serverConnectionCount) {
- super(selector, socketChannel, socketBufferSize);
+ NioSelectorManagerStats nioStats) {
+ super(selector, socketChannel, socketBufferSize, nioStats.getServerCommBufferStats());
this.requestHandlerFactory = requestHandlerFactory;
- this.serverConnectionCount = serverConnectionCount;
+ this.nioStats = nioStats;
}
@Override
@@ -130,8 +129,7 @@ protected void read(SelectionKey selectionKey) throws IOException {
DataInputStream dataInputStream = new DataInputStream(inputStream);
DataOutputStream dataOutputStream = new DataOutputStream(outputStream);
- streamRequestHandler = requestHandler.handleRequest(dataInputStream,
- dataOutputStream);
+ streamRequestHandler = requestHandler.handleRequest(dataInputStream, dataOutputStream);
if(logger.isDebugEnabled()) {
logger.debug("AsyncRequestHandler:read finished request from "
@@ -386,7 +384,7 @@ public void close() {
if(!isClosed.compareAndSet(false, true))
return;
- serverConnectionCount.decrement();
+ nioStats.removeConnection();
closeInternal();
}
}
View
34 src/java/voldemort/server/niosocket/NioSelectorManager.java
@@ -23,11 +23,12 @@
import java.util.Queue;
import java.util.concurrent.ConcurrentLinkedQueue;
-import org.apache.commons.lang.mutable.MutableInt;
import org.apache.log4j.Level;
+import voldemort.common.nio.CommBufferSizeStats;
+import voldemort.common.nio.SelectorManager;
import voldemort.server.protocol.RequestHandlerFactory;
-import voldemort.utils.SelectorManager;
+import voldemort.store.stats.Histogram;
/**
* SelectorManager handles the non-blocking polling of IO events using the
@@ -100,7 +101,7 @@
private final int socketBufferSize;
- private MutableInt numActiveConnections;
+ private final NioSelectorManagerStats stats;
public NioSelectorManager(InetSocketAddress endpoint,
RequestHandlerFactory requestHandlerFactory,
@@ -109,7 +110,7 @@ public NioSelectorManager(InetSocketAddress endpoint,
this.socketChannelQueue = new ConcurrentLinkedQueue<SocketChannel>();
this.requestHandlerFactory = requestHandlerFactory;
this.socketBufferSize = socketBufferSize;
- this.numActiveConnections = new MutableInt(0);
+ this.stats = new NioSelectorManagerStats();
}
public void accept(SocketChannel socketChannel) {
@@ -123,6 +124,9 @@ public void accept(SocketChannel socketChannel) {
@Override
protected void processEvents() {
try {
+ // update stats
+ stats.updateSelectStats(selectCount, selectTimeMs, processingTimeMs);
+
SocketChannel socketChannel = null;
while((socketChannel = socketChannelQueue.poll()) != null) {
@@ -160,11 +164,11 @@ protected void processEvents() {
socketChannel,
requestHandlerFactory,
socketBufferSize,
- numActiveConnections);
+ stats);
if(!isClosed.get()) {
socketChannel.register(selector, SelectionKey.OP_READ, attachment);
- numActiveConnections.increment();
+ stats.addConnection();
}
} catch(ClosedSelectorException e) {
if(logger.isDebugEnabled())
@@ -190,7 +194,7 @@ protected void processEvents() {
* @return
*/
public Integer getNumActiveConnections() {
- return numActiveConnections.toInteger();
+ return stats.getNumActiveConnections();
}
/**
@@ -201,4 +205,20 @@ public Integer getNumActiveConnections() {
public Integer getNumQueuedConnections() {
return socketChannelQueue.size();
}
+
+ public Histogram getSelectTimeMsHistogram() {
+ return stats.getSelectTimeMsHistogram();
+ }
+
+ public Histogram getSelectCountHistogram() {
+ return stats.getSelectCountHistogram();
+ }
+
+ public Histogram getProcessingTimeMsHistogram() {
+ return stats.getProcessingTimeMsHistogram();
+ }
+
+ public CommBufferSizeStats getCommBufferSizeStats() {
+ return stats.getServerCommBufferStats();
+ }
}
View
89 src/java/voldemort/server/niosocket/NioSelectorManagerStats.java
@@ -0,0 +1,89 @@
+package voldemort.server.niosocket;
+
+import org.apache.commons.lang.mutable.MutableInt;
+
+import voldemort.common.nio.CommBufferSizeStats;
+import voldemort.common.nio.SelectorManager;
+import voldemort.store.stats.Histogram;
+
+/**
+ * Encapsulates all the statistics about various metrics in the NIO Network
+ * layer
+ *
+ */
+public class NioSelectorManagerStats {
+
+ private static long SELECTOR_STATS_RESET_INTERVAL = 60000;
+
+ private MutableInt numActiveConnections;
+
+ private Histogram selectTimeMsHistogram;
+
+ private Histogram selectCountHistogram;
+
+ private Histogram processingTimeMsHistogram;
+
+ private CommBufferSizeStats serverCommBufferStats;
+
+ public NioSelectorManagerStats() {
+ this.numActiveConnections = new MutableInt(0);
+ this.serverCommBufferStats = new CommBufferSizeStats();
+
+ // Theoretically, the delay can be only upto SELECTOR_POLL_MS.
+ // But sometimes wallclock time can be higher
+ this.selectTimeMsHistogram = new Histogram(SelectorManager.SELECTOR_POLL_MS * 2,
+ 1,
+ SELECTOR_STATS_RESET_INTERVAL);
+ // Not a scientific limit. Not expecting a server thread to handle more
+ // than 100K connections.
+ this.selectCountHistogram = new Histogram(100000, 1, SELECTOR_STATS_RESET_INTERVAL);
+ // again not scientific. But we really don't care about any processing
+ // time higher than 15 seconds
+ this.processingTimeMsHistogram = new Histogram(15000, 1, SELECTOR_STATS_RESET_INTERVAL);
+ }
+
+ public void addConnection() {
+ numActiveConnections.increment();
+ }
+
+ public void removeConnection() {
+ numActiveConnections.decrement();
+ }
+
+ public void updateSelectStats(int selectCount, long selectTimeMs, long processingTimeMs) {
+ // update selection statistics
+ if(selectCount > -1) {
+ selectCountHistogram.insert(selectCount);
+ selectTimeMsHistogram.insert(selectTimeMs);
+ }
+ // update processing time statistics only if some work was picked up
+ if(processingTimeMs > -1 && selectCount > 0) {
+ processingTimeMsHistogram.insert(processingTimeMs);
+ }
+ }
+
+ /**
+ * Returns the number of active connections for this selector manager
+ *
+ * @return
+ */
+ public Integer getNumActiveConnections() {
+ return numActiveConnections.toInteger();
+ }
+
+ public Histogram getSelectTimeMsHistogram() {
+ return selectTimeMsHistogram;
+ }
+
+ public Histogram getSelectCountHistogram() {
+ return selectCountHistogram;
+ }
+
+ public Histogram getProcessingTimeMsHistogram() {
+ return processingTimeMsHistogram;
+ }
+
+ public CommBufferSizeStats getServerCommBufferStats() {
+ return serverCommBufferStats;
+ }
+}
View
71 src/java/voldemort/server/niosocket/NioSocketService.java
@@ -281,4 +281,75 @@ public final int getNumQueuedConnections() {
return sum;
}
+ @JmxGetter(name = "selectCountAvg", description = "average number of connections selected in each select() call")
+ public final double getSelectCountAvg() {
+ double sum = 0.0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getSelectCountHistogram().getAverage();
+ }
+ return sum / selectorManagers.length;
+ }
+
+ @JmxGetter(name = "selectCount99th", description = "99th percentile of number of connections selected in each select() call")
+ public final double getSelectCount99th() {
+ double sum = 0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getSelectCountHistogram().getQuantile(0.99);
+ }
+ return sum / selectorManagers.length;
+ }
+
+ @JmxGetter(name = "selectTimeMsAvg", description = "average time spent in the select() call")
+ public final double getSelectTimeMsAvg() {
+ double sum = 0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getSelectTimeMsHistogram().getAverage();
+ }
+ return sum / selectorManagers.length;
+ }
+
+ @JmxGetter(name = "selectTimeMs99th", description = "99th percentile of time spent in the select() call")
+ public final double getSelectTimeMs99th() {
+ double sum = 0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getSelectTimeMsHistogram().getQuantile(0.99);
+ }
+ return sum / selectorManagers.length;
+ }
+
+ @JmxGetter(name = "processingTimeMsAvg", description = "average time spent processing all read/write requests, in a select() loop")
+ public final double getProcessingTimeMsAvg() {
+ double sum = 0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getProcessingTimeMsHistogram().getAverage();
+ }
+ return sum / selectorManagers.length;
+ }
+
+ @JmxGetter(name = "processingTimeMs99th", description = "99th percentile of time spent processing all the read/write requests, in a select() loop")
+ public final double getprocessingTimeMs99th() {
+ double sum = 0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getProcessingTimeMsHistogram().getQuantile(0.99);
+ }
+ return sum / selectorManagers.length;
+ }
+
+ @JmxGetter(name = "commReadBufferSize", description = "total amount of memory consumed by all the communication read buffers, in bytes")
+ public final double getCommReadBufferSize() {
+ long sum = 0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getCommBufferSizeStats().getCommReadBufferSizeTracker().longValue();
+ }
+ return sum;
+ }
+
+ @JmxGetter(name = "commWriteBufferSize", description = "total amount of memory consumed by all the communication write buffers, in bytes")
+ public final double getCommWriteBufferSize() {
+ long sum = 0;
+ for(NioSelectorManager manager: selectorManagers) {
+ sum += manager.getCommBufferSizeStats().getCommWriteBufferSizeTracker().longValue();
+ }
+ return sum;
+ }
}
View
2  src/java/voldemort/server/protocol/admin/AdminServiceRequestHandler.java
@@ -40,6 +40,7 @@
import voldemort.client.protocol.pb.VAdminProto.VoldemortAdminRequest;
import voldemort.client.rebalance.RebalancePartitionsInfo;
import voldemort.cluster.Cluster;
+import voldemort.common.nio.ByteBufferBackedInputStream;
import voldemort.server.StoreRepository;
import voldemort.server.VoldemortConfig;
import voldemort.server.protocol.RequestHandler;
@@ -61,7 +62,6 @@
import voldemort.store.slop.SlopStorageEngine;
import voldemort.store.stats.StreamStats;
import voldemort.utils.ByteArray;
-import voldemort.utils.ByteBufferBackedInputStream;
import voldemort.utils.ByteUtils;
import voldemort.utils.ClosableIterator;
import voldemort.utils.EventThrottler;
View
2  src/java/voldemort/server/protocol/vold/VoldemortNativeRequestHandler.java
@@ -13,6 +13,7 @@
import voldemort.VoldemortException;
import voldemort.common.VoldemortOpCode;
+import voldemort.common.nio.ByteBufferBackedInputStream;
import voldemort.server.RequestRoutingType;
import voldemort.server.StoreRepository;
import voldemort.server.protocol.AbstractRequestHandler;
@@ -21,7 +22,6 @@
import voldemort.store.ErrorCodeMapper;
import voldemort.store.Store;
import voldemort.utils.ByteArray;
-import voldemort.utils.ByteBufferBackedInputStream;
import voldemort.utils.ByteUtils;
import voldemort.versioning.VectorClock;
import voldemort.versioning.Version;
View
18 src/java/voldemort/store/bdb/BdbStorageConfiguration.java
@@ -113,6 +113,8 @@ public BdbStorageConfiguration(VoldemortConfig config) {
Boolean.toString(config.getBdbCleanerLazyMigration()));
environmentConfig.setConfigParam(EnvironmentConfig.CLEANER_BACKGROUND_PROACTIVE_MIGRATION,
Boolean.toString(config.getBdbProactiveBackgroundMigration()));
+ environmentConfig.setConfigParam(EnvironmentConfig.CLEANER_BYTES_INTERVAL,
+ Long.toString(config.getBdbCleanerBytesInterval()));
environmentConfig.setLockTimeout(config.getBdbLockTimeoutMs(), TimeUnit.MILLISECONDS);
if(config.getBdbCacheModeEvictLN()) {
@@ -315,22 +317,6 @@ public void cleanLogs() {
}
}
- @JmxOperation(description = "Obtain the number of k-v entries in the store")
- public long getEntryCount(String storeName) throws Exception {
- Environment storeEnv = environments.get(storeName);
- if(storeEnv != null) {
- Database storeDb = null;
- try {
- storeDb = storeEnv.openDatabase(null, storeName, databaseConfig);
- return storeDb.count();
- } finally {
- if(storeDb != null)
- storeDb.close();
- }
- }
- return 0;
- }
-
public void close() {
synchronized(lock) {
try {
View
30 src/java/voldemort/store/bdb/BdbStorageEngine.java
@@ -77,8 +77,9 @@
private final Environment environment;
private final AtomicBoolean isOpen;
private final LockMode readLockMode;
- private final BdbEnvironmentStats bdbEnvironmentStats;
private final AtomicBoolean isTruncating = new AtomicBoolean(false);
+
+ protected final BdbEnvironmentStats bdbEnvironmentStats;
protected final boolean minimizeScanImpact;
public BdbStorageEngine(String name,
@@ -91,6 +92,7 @@ public BdbStorageEngine(String name,
this.isOpen = new AtomicBoolean(true);
this.readLockMode = config.getLockMode();
this.bdbEnvironmentStats = new BdbEnvironmentStats(environment,
+ database,
config.getStatsCacheTtlMs(),
config.getExposeSpaceUtil());
this.minimizeScanImpact = config.getMinimizeScanImpact();
@@ -108,6 +110,7 @@ public String getName() {
cursor.setCacheMode(CacheMode.EVICT_BIN);
return new BdbEntriesIterator(cursor, this);
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
@@ -121,6 +124,7 @@ public String getName() {
cursor.setCacheMode(CacheMode.EVICT_BIN);
return new BdbKeysIterator(cursor, this);
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
@@ -150,6 +154,7 @@ public void truncate() {
environment.truncateDatabase(transaction, this.getName(), false);
succeeded = true;
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new VoldemortException("Failed to truncate Bdb store " + getName(), e);
@@ -191,6 +196,7 @@ private boolean reopenBdbDatabase() {
this.bdbDatabase.getConfig());
return true;
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
throw new StorageInitializationException("Failed to reinitialize BdbStorageEngine for store:"
+ getName() + " after truncation.",
e);
@@ -239,11 +245,12 @@ protected Database getBdbDatabase() {
return Collections.emptyList();
}
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
} finally {
if(logger.isTraceEnabled()) {
- logger.trace("Completed GET from key " + key + " (keyRef: "
+ logger.trace("Completed GET (" + getName() + ") from key " + key + " (keyRef: "
+ System.identityHashCode(key) + ") in "
+ (System.nanoTime() - startTimeNs) + " ns at "
+ System.currentTimeMillis());
@@ -269,7 +276,7 @@ protected Database getBdbDatabase() {
String keyStr = "";
for(ByteArray key: keys)
keyStr += key + " ";
- logger.trace("Completed GETALL from keys " + keyStr + " in "
+ logger.trace("Completed GETALL (" + getName() + ") from keys " + keyStr + " in "
+ (System.nanoTime() - startTimeNs) + " ns at "
+ System.currentTimeMillis());
}
@@ -337,6 +344,7 @@ else if(occurred == Occurred.AFTER)
succeeded = true;
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
} finally {
@@ -345,7 +353,7 @@ else if(occurred == Occurred.AFTER)
else
attemptAbort(transaction);
if(logger.isTraceEnabled()) {
- logger.trace("Completed PUT to key " + key + " (keyRef: "
+ logger.trace("Completed PUT (" + getName() + ") to key " + key + " (keyRef: "
+ System.identityHashCode(key) + " value " + value + " in "
+ (System.nanoTime() - startTimeNs) + " ns at "
+ System.currentTimeMillis());
@@ -415,13 +423,15 @@ public boolean delete(ByteArray key, Version version) throws PersistenceFailureE
return numDeletedVersions > 0;
}
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
} finally {
attemptCommit(transaction);
if(logger.isTraceEnabled()) {
- logger.trace("Completed DELETE of key " + ByteUtils.toHexString(key.get())
- + " (keyRef: " + System.identityHashCode(key) + ") in "
+ logger.trace("Completed DELETE (" + getName() + ") of key "
+ + ByteUtils.toHexString(key.get()) + " (keyRef: "
+ + System.identityHashCode(key) + ") in "
+ (System.nanoTime() - startTimeNs) + " ns at "
+ System.currentTimeMillis());
}
@@ -450,6 +460,7 @@ public void close() throws PersistenceFailureException {
if(this.isOpen.compareAndSet(true, false))
this.getBdbDatabase().close();
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException("Shutdown failed.", e);
}
@@ -459,7 +470,8 @@ private void attemptAbort(Transaction transaction) {
try {
if(transaction != null)
transaction.abort();
- } catch(Exception e) {
+ } catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error("Abort failed!", e);
}
}
@@ -469,6 +481,7 @@ private void attemptCommit(Transaction transaction) {
if(transaction != null)
transaction.commit();
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error("Transaction commit failed!", e);
attemptAbort(transaction);
throw new PersistenceFailureException(e);
@@ -481,6 +494,7 @@ public DatabaseStats getStats(boolean setFast) {
config.setFast(setFast);
return this.getBdbDatabase().getStats(config);
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new VoldemortException(e);
}
@@ -552,6 +566,7 @@ protected boolean makeMore() {
this.cache.add(Pair.create(key, val));
return true;
} catch(DatabaseException e) {
+ bdbEngine.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
@@ -600,6 +615,7 @@ private boolean fetchNextKey() {
current = new ByteArray(keyEntry.getData());
return true;
} catch(DatabaseException e) {
+ bdbEngine.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
View
4 src/java/voldemort/store/bdb/PartitionPrefixedBdbStorageEngine.java
@@ -69,6 +69,7 @@ public PartitionPrefixedBdbStorageEngine(String name,
cursor.setCacheMode(CacheMode.EVICT_BIN);
return new BdbPartitionEntriesIterator(cursor, partition, this);
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
@@ -83,6 +84,7 @@ public PartitionPrefixedBdbStorageEngine(String name,
cursor.setCacheMode(CacheMode.EVICT_BIN);
return new BdbPartitionKeysIterator(cursor, partition, this);
} catch(DatabaseException e) {
+ this.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
@@ -191,6 +193,7 @@ private boolean makeMore() {
this.cache.add(Pair.create(key, val));
return true;
} catch(DatabaseException e) {
+ bdbEngine.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
@@ -262,6 +265,7 @@ private boolean fetchNextKey() {
current = new ByteArray(StoreBinaryFormat.extractKey(keyEntry.getData()));
return true;
} catch(DatabaseException e) {
+ bdbEngine.bdbEnvironmentStats.reportException(e);
logger.error(e);
throw new PersistenceFailureException(e);
}
View
88 src/java/voldemort/store/bdb/stats/BdbEnvironmentStats.java
@@ -1,26 +1,47 @@
package voldemort.store.bdb.stats;
import java.util.concurrent.Callable;
+import java.util.concurrent.atomic.AtomicLong;
import voldemort.VoldemortException;
import voldemort.annotations.Experimental;
import voldemort.annotations.jmx.JmxGetter;
+import voldemort.annotations.jmx.JmxOperation;
import voldemort.utils.CachedCallable;
+import com.sleepycat.je.Database;
+import com.sleepycat.je.DatabaseException;
+import com.sleepycat.je.DatabaseStats;
import com.sleepycat.je.Environment;
import com.sleepycat.je.EnvironmentConfig;
+import com.sleepycat.je.EnvironmentFailureException;
import com.sleepycat.je.EnvironmentStats;
+import com.sleepycat.je.LockTimeoutException;
import com.sleepycat.je.StatsConfig;
public class BdbEnvironmentStats {
+ // Don't fetch entry count/btree stats more than twice a day
+ private final static long INVASIVE_STATS_TTL_MS = 12 * 3600 * 1000;
+
private final Environment environment;
+ private final Database database;
private final CachedCallable<EnvironmentStats> fastStats;
private final CachedCallable<SpaceUtilizationStats> fastSpaceStats;
+ private final CachedCallable<Long> entryCount;
+ private final CachedCallable<DatabaseStats> btreeStats;
private final boolean exposeSpaceStats;
- public BdbEnvironmentStats(Environment environment, long ttlMs, boolean exposeSpaceUtil) {
+ private final AtomicLong numExceptions;
+ private final AtomicLong numLockTimeoutExceptions;
+ private final AtomicLong numEnvironmentFailureExceptions;
+
+ public BdbEnvironmentStats(Environment environment,
+ Database database,
+ long ttlMs,
+ boolean exposeSpaceUtil) {
this.environment = environment;
+ this.database = database;
this.exposeSpaceStats = exposeSpaceUtil;
Callable<EnvironmentStats> fastStatsCallable = new Callable<EnvironmentStats>() {
@@ -37,6 +58,26 @@ public SpaceUtilizationStats call() throws Exception {
}
};
fastSpaceStats = new CachedCallable<SpaceUtilizationStats>(fastDbStatsCallable, ttlMs);
+
+ Callable<Long> entryCountCallable = new Callable<Long>() {
+
+ public Long call() throws Exception {
+ return getEntryCountUncached();
+ }
+ };
+ entryCount = new CachedCallable<Long>(entryCountCallable, INVASIVE_STATS_TTL_MS);
+
+ Callable<DatabaseStats> btreeStatsCallable = new Callable<DatabaseStats>() {
+
+ public DatabaseStats call() throws Exception {
+ return getBtreeStatsUncached();
+ }
+ };
+ btreeStats = new CachedCallable<DatabaseStats>(btreeStatsCallable, INVASIVE_STATS_TTL_MS);
+
+ numExceptions = new AtomicLong(0);
+ numLockTimeoutExceptions = new AtomicLong(0);
+ numEnvironmentFailureExceptions = new AtomicLong(0);
}
private EnvironmentStats getEnvironmentStats(boolean fast) {
@@ -65,6 +106,25 @@ private EnvironmentStats getFastStats() {
}
}
+ private Long getEntryCountUncached() {
+ return database.count();
+ }
+
+ public DatabaseStats getBtreeStatsUncached() throws Exception {
+ // fast stats does not provide detailed Btree structure.
+ // This is invasive and will affect performance.
+ return database.getStats(new StatsConfig().setFast(false));
+ }
+
+ public void reportException(DatabaseException de) {
+ numExceptions.incrementAndGet();
+ if(de instanceof LockTimeoutException) {
+ numLockTimeoutExceptions.incrementAndGet();
+ } else if(de instanceof EnvironmentFailureException) {
+ numEnvironmentFailureExceptions.incrementAndGet();
+ }
+ }
+
@JmxGetter(name = "FastStatsAsString")
public String getFastStatsAsString() {
return getFastStats().toString();
@@ -275,6 +335,32 @@ public long getNumAcquiresNoWaiters() {
return getFastStats().getNAcquiresNoWaiters();
}
+ // 5. Exceptions & general statistics
+ @JmxGetter(name = "numExceptions")
+ public long getNumExceptions() {
+ return numExceptions.longValue();
+ }
+
+ @JmxGetter(name = "numLockTimeoutExceptions")
+ public long getNumLockTimeoutExceptions() {
+ return numLockTimeoutExceptions.longValue();
+ }
+
+ @JmxGetter(name = "numEnvironmentFailureExceptions")
+ public long getNumEnvironmentFailureExceptions() {
+ return numEnvironmentFailureExceptions.longValue();
+ }
+
+ @JmxOperation(description = "Obtain the number of k-v entries in the store")
+ public long getEntryCount() throws Exception {
+ return entryCount.call();
+ }
+
+ @JmxOperation(description = "Obtain statistics about the BTree Index for a store")
+ public String getBtreeStats() throws Exception {
+ return btreeStats.call().toString();
+ }
+
// Compound statistics derived from raw statistics
@JmxGetter(name = "NumWritesTotal")
View
8 src/java/voldemort/store/socket/clientrequest/ClientRequestExecutor.java
@@ -28,7 +28,8 @@
import org.apache.log4j.Level;
-import voldemort.utils.SelectorManagerWorker;
+import voldemort.common.nio.CommBufferSizeStats;
+import voldemort.common.nio.SelectorManagerWorker;
import voldemort.utils.Time;
/**
@@ -55,7 +56,8 @@
public ClientRequestExecutor(Selector selector,
SocketChannel socketChannel,
int socketBufferSize) {
- super(selector, socketChannel, socketBufferSize);
+ // Not tracking or exposing the comm buffer statistics for now
+ super(selector, socketChannel, socketBufferSize, new CommBufferSizeStats());
isExpired = false;
}
@@ -106,7 +108,7 @@ public synchronized void addClientRequest(ClientRequest<?> clientRequest,
if(timeoutMs == -1) {
this.expiration = -1;
} else {
- if (elapsedNs > (Time.NS_PER_MS * timeoutMs)) {
+ if(elapsedNs > (Time.NS_PER_MS * timeoutMs)) {
this.expiration = System.nanoTime();
} else {
this.expiration = System.nanoTime() + (Time.NS_PER_MS * timeoutMs) - elapsedNs;
View
2  src/java/voldemort/store/socket/clientrequest/ClientRequestExecutorFactory.java
@@ -36,10 +36,10 @@
import org.apache.log4j.Level;
import org.apache.log4j.Logger;
+import voldemort.common.nio.SelectorManager;
import voldemort.store.socket.SocketDestination;
import voldemort.store.stats.ClientSocketStats;
import voldemort.utils.DaemonThreadFactory;
-import voldemort.utils.SelectorManager;
import voldemort.utils.Time;
import voldemort.utils.pool.ResourceFactory;
View
43 src/java/voldemort/store/stats/Histogram.java
@@ -21,8 +21,24 @@
private final int[] buckets;
private final int[] bounds;
private int size;
+ private long sum;
private static final Logger logger = Logger.getLogger(Histogram.class);
+ private long resetIntervalMs = -1;
+ private long lastResetTimeMs;
+
+ /**
+ * Initialize an empty histogram
+ *
+ * @param nBuckets The number of buckets to use
+ * @param step The size of each bucket
+ */
+ public Histogram(int nBuckets, int step, long resetIntervalMs) {
+ this(nBuckets, step);
+ this.resetIntervalMs = resetIntervalMs;
+ this.lastResetTimeMs = System.currentTimeMillis();
+ }
+
/**
* Initialize an empty histogram
*
@@ -51,6 +67,8 @@ protected void init() {
public synchronized void reset() {
Arrays.fill(buckets, 0);
size = 0;
+ sum = 0;
+ this.lastResetTimeMs = System.currentTimeMillis();
}
/**
@@ -60,12 +78,14 @@ public synchronized void reset() {
* @param data The value to insert into the histogram
*/
public synchronized void insert(long data) {
+ resetIfNeeded();
int index = findBucket(data);
if(index == -1) {
logger.error(data + " can't be bucketed, is invalid!");
return;
}
buckets[index]++;
+ sum += data;
size++;
}
@@ -77,6 +97,7 @@ public synchronized void insert(long data) {
* @return Lower bound associated with the percentile
*/
public synchronized int getQuantile(double quantile) {
+ resetIfNeeded();
int total = 0;
for(int i = 0; i < nBuckets; i++) {
total += buckets[i];
@@ -88,6 +109,20 @@ public synchronized int getQuantile(double quantile) {
return 0;
}
+ /**
+ * Obtain the average of the data in the histogram
+ *
+ * Note: Caller is responsible for making sure 'sum' does not overflow
+ * within the reset interval
+ *
+ * @return the average over the current samples
+ */
+ public synchronized double getAverage() {
+ if(size == 0)
+ return 0.0;
+ return (sum * 1.0) / size;
+ }
+
private int findBucket(long needle) {
long max = step * nBuckets;
if(needle > max) {
@@ -120,4 +155,12 @@ private int compareToBucket(int bucket, long needle) {
return -1;
}
}
+
+ private void resetIfNeeded() {
+ if(resetIntervalMs > -1) {
+ if((System.currentTimeMillis() - lastResetTimeMs) >= this.resetIntervalMs) {
+ this.reset();
+ }
+ }
+ }
}
View
41 test/unit/voldemort/store/stats/HistogramTest.java
@@ -1,14 +1,14 @@
package voldemort.store.stats;
+import static org.junit.Assert.assertEquals;
+
import org.junit.Before;
import org.junit.Test;
-import static org.junit.Assert.assertEquals;
-
public class HistogramTest {
-
+
private Histogram histogram;
-
+
@Before
public void setUp() {
histogram = new Histogram(10, 5);
@@ -29,20 +29,43 @@ public void setUp() {
histogram.insert(66);
histogram.insert(76);
}
-
+
@Test
- public void testAverage() {
+ public void test50thQuartile() {
assertEquals(histogram.getQuantile(0.50), 30);
}
-
+
@Test
public void test95thQuartile() {
assertEquals(histogram.getQuantile(0.95), 45);
}
-
+
@Test
public void test99thQuartile() {
assertEquals(histogram.getQuantile(0.99), 45);
}
-
+
+ @Test
+ public void testResetHistogram() {
+
+ Histogram resetingHistogram = new Histogram(10, 1, 10);
+ // tests that the functionality is still working
+ for(long data = 0; data < 5; data++) {
+ for(int loop = 0; loop <= data; loop++) {
+ resetingHistogram.insert(data);
+ }
+ }
+ assertEquals(3, resetingHistogram.getQuantile(0.50));
+ assertEquals(4, resetingHistogram.getQuantile(0.99));
+ assertEquals(2.67, resetingHistogram.getAverage(), 0.01);
+
+ // tests that once enough time passes, old data will be discarded
+ try {
+ Thread.sleep(10);
+ } catch(InterruptedException ie) {}
+
+ assertEquals(0, resetingHistogram.getQuantile(0.50));
+ assertEquals(0, resetingHistogram.getQuantile(0.99));
+ assertEquals(0.0, resetingHistogram.getAverage(), 0.0);
+ }
}
Please sign in to comment.
Something went wrong with that request. Please try again.