Skip to content

HTTPS clone URL

Subversion checkout URL

You can clone with HTTPS or Subversion.

Download ZIP
Browse files

Merge branch 'master' of github.com:zhongjiewu/voldemort

  • Loading branch information...
commit 9bb1704a6f1ec6be1fbf0a05626c6f62de79e409 2 parents b3e7212 + c709be8
Zhongjie Wu zhongjiewu authored
12 src/java/voldemort/client/SocketStoreClientFactory.java
View
@@ -38,7 +38,6 @@
import voldemort.store.socket.SocketStoreFactory;
import voldemort.store.socket.clientrequest.ClientRequestExecutorPool;
import voldemort.utils.ByteArray;
-import voldemort.utils.JmxUtils;
import voldemort.versioning.InconsistencyResolver;
import voldemort.versioning.Versioned;
@@ -69,8 +68,9 @@ public SocketStoreClientFactory(ClientConfig config) {
config.getSocketTimeout(TimeUnit.MILLISECONDS),
config.getSocketBufferSize(),
config.getSocketKeepAlive());
- if(config.isJmxEnabled())
- JmxUtils.registerMbean(storeFactory, JmxUtils.createObjectName(storeFactory.getClass()));
+ if(config.isJmxEnabled()) {
+ ((ClientRequestExecutorPool) storeFactory).registerJmx();
+ }
}
@Override
@@ -87,7 +87,8 @@ public SocketStoreClientFactory(ClientConfig config) {
return getParentStoreClient(storeName, resolver);
}
- private <K, V> StoreClient<K, V> getParentStoreClient(String storeName, InconsistencyResolver<Versioned<V>> resolver) {
+ private <K, V> StoreClient<K, V> getParentStoreClient(String storeName,
+ InconsistencyResolver<Versioned<V>> resolver) {
return super.getStoreClient(storeName, resolver);
}
@@ -96,7 +97,8 @@ public SocketStoreClientFactory(ClientConfig config) {
try {
return super.getRemoteMetadata(key, url);
} catch(VoldemortException e) {
- // Fix SNA-4227: When an error occurs during bootstrap, close the socket
+ // Fix SNA-4227: When an error occurs during bootstrap, close the
+ // socket
SocketDestination destination = new SocketDestination(url.getHost(),
url.getPort(),
getRequestFormatType());
12 src/java/voldemort/store/socket/clientrequest/ClientRequestExecutorFactory.java
View
@@ -37,6 +37,7 @@
import org.apache.log4j.Logger;
import voldemort.store.socket.SocketDestination;
+import voldemort.store.stats.ClientSocketStats;
import voldemort.utils.DaemonThreadFactory;
import voldemort.utils.SelectorManager;
import voldemort.utils.Time;
@@ -61,6 +62,7 @@
private final AtomicInteger counter = new AtomicInteger();
private final Map<SocketDestination, Long> lastClosedTimestamps;
private final Logger logger = Logger.getLogger(getClass());
+ private ClientSocketStats stats;
public ClientRequestExecutorFactory(int selectors,
int connectTimeoutMs,
@@ -94,6 +96,9 @@ public void destroy(SocketDestination dest, ClientRequestExecutor clientRequestE
throws Exception {
clientRequestExecutor.close();
int numDestroyed = destroyed.incrementAndGet();
+ if(stats != null) {
+ stats.connectionDestroy(dest);
+ }
if(logger.isDebugEnabled())
logger.debug("Destroyed socket " + numDestroyed + " connection to " + dest.getHost()
@@ -108,6 +113,9 @@ public void destroy(SocketDestination dest, ClientRequestExecutor clientRequestE
public ClientRequestExecutor create(SocketDestination dest) throws Exception {
int numCreated = created.incrementAndGet();
+ if(stats != null) {
+ stats.connectionCreate(dest);
+ }
if(logger.isDebugEnabled())
logger.debug("Creating socket " + numCreated + " for " + dest.getHost() + ":"
@@ -426,4 +434,8 @@ public void setLastClosedTimestamp(SocketDestination socketDestination) {
lastClosedTimestamps.put(socketDestination, System.nanoTime());
}
+ public void setStats(ClientSocketStats stats) {
+ this.stats = stats;
+ }
+
}
77 src/java/voldemort/store/socket/clientrequest/ClientRequestExecutorPool.java
View
@@ -17,19 +17,17 @@
package voldemort.store.socket.clientrequest;
import java.util.concurrent.TimeUnit;
-import java.util.concurrent.atomic.AtomicInteger;
-import java.util.concurrent.atomic.AtomicLong;
import voldemort.VoldemortException;
-import voldemort.annotations.jmx.JmxGetter;
-import voldemort.annotations.jmx.JmxManaged;
-import voldemort.annotations.jmx.JmxSetter;
import voldemort.client.protocol.RequestFormatType;
import voldemort.server.RequestRoutingType;
import voldemort.store.UnreachableStoreException;
import voldemort.store.socket.SocketDestination;
import voldemort.store.socket.SocketStore;
import voldemort.store.socket.SocketStoreFactory;
+import voldemort.store.stats.ClientSocketStats;
+import voldemort.store.stats.ClientSocketStatsJmx;
+import voldemort.utils.JmxUtils;
import voldemort.utils.Time;
import voldemort.utils.Utils;
import voldemort.utils.pool.KeyedResourcePool;
@@ -46,15 +44,11 @@
* terminated upon calling {@link #close()}.
*/
-@JmxManaged(description = "Voldemort socket pool.")
public class ClientRequestExecutorPool implements SocketStoreFactory {
- private final AtomicInteger monitoringInterval = new AtomicInteger(10000);
- private final AtomicInteger checkouts;
- private final AtomicLong waitNs;
- private final AtomicLong avgWaitNs;
private final KeyedResourcePool<SocketDestination, ClientRequestExecutor> pool;
private final ClientRequestExecutorFactory factory;
+ private ClientSocketStats stats;
public ClientRequestExecutorPool(int selectors,
int maxConnectionsPerNode,
@@ -73,9 +67,8 @@ public ClientRequestExecutorPool(int selectors,
socketBufferSize,
socketKeepAlive);
this.pool = new KeyedResourcePool<SocketDestination, ClientRequestExecutor>(factory, config);
- this.checkouts = new AtomicInteger(0);
- this.waitNs = new AtomicLong(0);
- this.avgWaitNs = new AtomicLong(0);
+ this.stats = new ClientSocketStats(pool);
+ ((ClientRequestExecutorFactory) factory).setStats(stats);
}
public ClientRequestExecutorPool(int maxConnectionsPerNode,
@@ -105,6 +98,13 @@ public SocketStore create(String storeName,
requestRoutingType);
}
+ public void registerJmx() {
+ stats.enableJmx();
+ JmxUtils.registerMbean(new ClientSocketStatsJmx(stats),
+ JmxUtils.createObjectName("voldemort.store.socket.clientrequest",
+ "aggregated"));
+ }
+
/**
* Checkout a socket from the pool
*
@@ -117,7 +117,10 @@ public ClientRequestExecutor checkout(SocketDestination destination) {
// time checkout
long start = System.nanoTime();
ClientRequestExecutor clientRequestExecutor = pool.checkout(destination);
- updateStats(System.nanoTime() - start);
+ long end = System.nanoTime();
+ if(stats != null) {
+ stats.recordCheckoutTimeUs(destination, (end - start) / Time.NS_PER_US);
+ }
return clientRequestExecutor;
} catch(Exception e) {
@@ -126,20 +129,6 @@ public ClientRequestExecutor checkout(SocketDestination destination) {
}
}
- private void updateStats(long checkoutTimeNs) {
- long wait = waitNs.getAndAdd(checkoutTimeNs);
- int count = checkouts.getAndIncrement();
-
- // reset reporting inverval if we have used up the current interval
- int interval = this.monitoringInterval.get();
- if(count % interval == interval - 1) {
- // harmless race condition:
- waitNs.set(0);
- checkouts.set(0);
- avgWaitNs.set(wait / count);
- }
- }
-
/**
* Check the socket back into the pool.
*
@@ -168,36 +157,8 @@ public void close() {
pool.close();
}
- @JmxGetter(name = "socketsCreated", description = "The total number of sockets created by this pool.")
- public int getNumberSocketsCreated() {
- return this.factory.getNumberCreated();
- }
-
- @JmxGetter(name = "socketsDestroyed", description = "The total number of sockets destroyed by this pool.")
- public int getNumberSocketsDestroyed() {
- return this.factory.getNumberDestroyed();
- }
-
- @JmxGetter(name = "numberOfConnections", description = "The number of active connections.")
- public int getNumberOfActiveConnections() {
- return this.pool.getTotalResourceCount();
- }
-
- @JmxGetter(name = "numberOfIdleConnections", description = "The number of idle connections.")
- public int getNumberOfCheckedInConnections() {
- return this.pool.getCheckedInResourceCount();
- }
-
- @JmxGetter(name = "avgWaitTimeMs", description = "The avg. ms of wait time to acquire a connection.")
- public double getAvgWaitTimeMs() {
- return this.avgWaitNs.doubleValue() / Time.NS_PER_MS;
- }
-
- @JmxSetter(name = "monitoringInterval", description = "The number of checkouts over which performance statistics are calculated.")
- public void setMonitoringInterval(int count) {
- if(count <= 0)
- throw new IllegalArgumentException("Monitoring interval must be a positive number.");
- this.monitoringInterval.set(count);
+ public ClientSocketStats getStats() {
+ return stats;
}
}
230 src/java/voldemort/store/stats/ClientSocketStats.java
View
@@ -0,0 +1,230 @@
+/*
+ * Copyright 2008-2011 LinkedIn, Inc
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License"); you may not
+ * use this file except in compliance with the License. You may obtain a copy of
+ * the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package voldemort.store.stats;
+
+import java.util.Iterator;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
+import java.util.concurrent.atomic.AtomicInteger;
+import java.util.concurrent.atomic.AtomicLong;
+
+import voldemort.store.socket.SocketDestination;
+import voldemort.store.socket.clientrequest.ClientRequestExecutor;
+import voldemort.utils.JmxUtils;
+import voldemort.utils.pool.KeyedResourcePool;
+
+/**
+ * Some convenient statistics to track about the client requests
+ *
+ *
+ */
+public class ClientSocketStats {
+
+ private final ClientSocketStats parent;
+ private final ConcurrentMap<SocketDestination, ClientSocketStats> statsMap;
+ private final SocketDestination destination;
+ private final KeyedResourcePool<SocketDestination, ClientRequestExecutor> pool;
+
+ private final AtomicInteger monitoringInterval = new AtomicInteger(10000);
+ private final Histogram checkoutTimeUsHistogram = new Histogram(20000, 100);
+ private final AtomicLong totalCheckoutTimeUs = new AtomicLong(0);
+ private final AtomicLong avgCheckoutTimeUs = new AtomicLong(0);
+ private final AtomicInteger connectionsCreated = new AtomicInteger(0);
+ private final AtomicInteger connectionsDestroyed = new AtomicInteger(0);
+ private final AtomicInteger connectionsCheckedout = new AtomicInteger(0);
+ private boolean jmxEnable = false;
+
+ // private final AtomicInteger connectionsCheckedin = new AtomicInteger(0);
+
+ /**
+ * To construct a per node stats object
+ *
+ * @param parent An optional parent stats object that will maintain
+ * aggregate data across many sockets
+ * @param destination The destination object that defines the node
+ * @param pool The socket pool that will give out connection information
+ */
+ public ClientSocketStats(ClientSocketStats parent,
+ SocketDestination destination,
+ KeyedResourcePool<SocketDestination, ClientRequestExecutor> pool) {
+ this.parent = parent;
+ this.statsMap = null;
+ this.destination = destination;
+ this.pool = pool;
+ }
+
+ /**
+ * Construction of a new aggregate stats object
+ *
+ * @param pool The socket pool that will give out connection information
+ */
+ public ClientSocketStats(KeyedResourcePool<SocketDestination, ClientRequestExecutor> pool) {
+ this.parent = null;
+ this.statsMap = new ConcurrentHashMap<SocketDestination, ClientSocketStats>();
+ this.destination = null;
+ this.pool = pool;
+ }
+
+ /* get per node stats, create one if not exist */
+ private ClientSocketStats getOrCreateNodeStats(SocketDestination destination) {
+ if(destination == null) {
+ return null;
+ }
+ ClientSocketStats stats = statsMap.get(destination);
+ if(stats == null) {
+ stats = new ClientSocketStats(this, destination, pool);
+ statsMap.putIfAbsent(destination, stats);
+ stats = statsMap.get(destination);
+ if(this.jmxEnable) {
+ JmxUtils.registerMbean(new ClientSocketStatsJmx(stats),
+ JmxUtils.createObjectName("voldemort.store.socket.clientrequest",
+ "stats_"
+ + destination.toString()
+ .replace(':',
+ '_')));
+ }
+ }
+ return stats;
+ }
+
+ /**
+ * Record the checkout wait time in us
+ *
+ * @param dest Destination of the socket to checkout. Will actually record
+ * if null. Otherwise will call this on self and corresponding child
+ * with this param null.
+ * @param checkoutTimeUs The number of us to wait before getting a socket
+ */
+ public void recordCheckoutTimeUs(SocketDestination dest, long checkoutTimeUs) {
+ if(dest != null) {
+ getOrCreateNodeStats(dest).recordCheckoutTimeUs(null, checkoutTimeUs);
+ recordCheckoutTimeUs(null, checkoutTimeUs);
+ } else {
+ this.totalCheckoutTimeUs.getAndAdd(checkoutTimeUs);
+ checkoutTimeUsHistogram.insert(checkoutTimeUs);
+ int checkouts = this.connectionsCheckedout.getAndIncrement();
+
+ // reset aggregated stats and all the node stats for new interval
+ if(parent == null && statsMap != null) {
+ int interval = this.monitoringInterval.get();
+ if(checkouts % interval == interval - 1) {
+ // reset all children
+ Iterator<SocketDestination> it = statsMap.keySet().iterator();
+ while(it.hasNext()) {
+ ClientSocketStats stats = statsMap.get(it.next());
+ stats.resetForInterval();
+ }
+ // reset itself
+ resetForInterval();
+ }
+ }
+ }
+ }
+
+ /**
+ * Calculate the average and reset the stats
+ */
+ public void resetForInterval() {
+ // harmless race condition:
+ this.totalCheckoutTimeUs.set(0);
+ this.connectionsCheckedout.set(0);
+ checkoutTimeUsHistogram.reset();
+ }
+
+ public void connectionCreate(SocketDestination dest) {
+ if(dest != null) {
+ getOrCreateNodeStats(dest).connectionCreate(null);
+ connectionCreate(null);
+ } else {
+ this.connectionsCreated.getAndIncrement();
+ }
+ }
+
+ public void connectionDestroy(SocketDestination dest) {
+ if(dest != null) {
+ getOrCreateNodeStats(dest).connectionDestroy(null);
+ connectionDestroy(null);
+ } else {
+ this.connectionsDestroyed.getAndIncrement();
+ }
+ }
+
+ /* getters */
+
+ public int getConnectionsCreated() {
+ return connectionsCreated.intValue();
+ }
+
+ public int getConnectionsDestroyed() {
+ return connectionsDestroyed.intValue();
+ }
+
+ // public int getConnectionsCheckedin() {
+ // return connectionsCheckedin.intValue();
+ // }
+
+ public int getConnectionsCheckedout() {
+ return connectionsCheckedout.intValue();
+ }
+
+ public Histogram getWaitHistogram() {
+ return this.checkoutTimeUsHistogram;
+ }
+
+ public long getAveWaitUs() {
+ long ns = totalCheckoutTimeUs.get();
+ int count = connectionsCheckedout.get();
+ this.avgCheckoutTimeUs.set(count > 0 ? (ns / count) : -1);
+ return this.avgCheckoutTimeUs.longValue();
+ }
+
+ public int getConnectionsActive(SocketDestination destination) {
+ if(destination == null) {
+ return pool.getTotalResourceCount();
+ } else {
+ return pool.getTotalResourceCount(destination);
+ }
+ }
+
+ public int getConnectionsInPool(SocketDestination destination) {
+ if(destination == null) {
+ return pool.getCheckedInResourceCount();
+ } else {
+ return pool.getCheckedInResourcesCount(destination);
+ }
+ }
+
+ public void setMonitoringInterval(int count) {
+ this.monitoringInterval.set(count);
+ }
+
+ public int getMonitoringInterval() {
+ return this.monitoringInterval.get();
+ }
+
+ public void enableJmx() {
+ jmxEnable = true;
+ }
+
+ ConcurrentMap<SocketDestination, ClientSocketStats> getStatsMap() {
+ return statsMap;
+ }
+
+ SocketDestination getDestination() {
+ return destination;
+ }
+}
101 src/java/voldemort/store/stats/ClientSocketStatsJmx.java
View
@@ -0,0 +1,101 @@
+/*
+ * Copyright 2008-2011 LinkedIn, Inc
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License"); you may not
+ * use this file except in compliance with the License. You may obtain a copy of
+ * the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package voldemort.store.stats;
+
+import voldemort.annotations.jmx.JmxGetter;
+import voldemort.annotations.jmx.JmxManaged;
+import voldemort.annotations.jmx.JmxSetter;
+import voldemort.utils.Time;
+
+/**
+ *
+ * A wrapper class to expose client socket stats via JMX
+ *
+ */
+
+@JmxManaged(description = "Voldemort socket pool.")
+public class ClientSocketStatsJmx {
+
+ private final ClientSocketStats stats;
+
+ /**
+ * Class for JMX
+ */
+ public ClientSocketStatsJmx(ClientSocketStats stats) {
+ this.stats = stats;
+ }
+
+ @JmxGetter(name = "socketsCreated", description = "Number of sockets created.")
+ public int getConnectionsCreated() {
+ return stats.getConnectionsCreated();
+ }
+
+ @JmxGetter(name = "socketsDestroyed", description = "Number of sockets destroyed.")
+ public int getConnectionsDestroyed() {
+ return stats.getConnectionsDestroyed();
+ }
+
+ @JmxGetter(name = "socketsCheckedout", description = "Number of sockets checked out.")
+ public int getConnectionsCheckinout() {
+ return stats.getConnectionsCheckedout();
+ }
+
+ @JmxGetter(name = "waitMsAverage", description = "Average ms to wait to get a socket.")
+ public double getWaitMsAverage() {
+ return (double) stats.getAveWaitUs() / Time.US_PER_MS;
+ }
+
+ @JmxGetter(name = "waitMsQ50th", description = "50th percentile wait time to get a connection.")
+ public double get() {
+ return (double) stats.getWaitHistogram().getQuantile(0.5) / Time.US_PER_MS;
+ }
+
+ @JmxGetter(name = "waitMsQ99th", description = "99th percentile wait time to get a connection.")
+ public double getWaitMsQ99th() {
+ return (double) stats.getWaitHistogram().getQuantile(0.99) / Time.US_PER_MS;
+ }
+
+ @JmxGetter(name = "socketsActive", description = "Total number of sockets, checkedin and checkout.")
+ public int getConnActive() {
+ int result = -1;
+ try {
+ result = stats.getConnectionsActive(stats.getDestination());
+ } catch(Exception e) {}
+ return result;
+ }
+
+ @JmxGetter(name = "socketsInPool", description = "Total number of sockets in the pool.")
+ public int getConnAvailable() {
+ int result = -1;
+ try {
+ result = stats.getConnectionsInPool(stats.getDestination());
+ } catch(Exception e) {}
+ return result;
+ }
+
+ @JmxGetter(name = "monitoringInterval", description = "The number of checkouts over which performance statics are calculated.")
+ public int getMonitoringInterval() {
+ return stats.getMonitoringInterval();
+ }
+
+ @JmxSetter(name = "monitoringInterval", description = "The number of checkouts over which performance statics are calculated.")
+ public void setMonitoringInterval(int count) {
+ if(count <= 0)
+ throw new IllegalArgumentException("Monitoring interval must be a positive number.");
+ stats.setMonitoringInterval(count);
+ }
+}
13 test/unit/voldemort/server/socket/ClientRequestExecutorPoolTest.java
View
@@ -112,20 +112,19 @@ public void testCloseWithInFlightSockets() throws Exception {
for(int i = 0; i < maxConnectionsPerNode; i++)
list.add(pool.checkout(dest1));
- assertEquals(list.size(), pool.getNumberSocketsCreated());
- assertEquals(list.size(), pool.getNumberOfActiveConnections());
+ assertEquals(list.size(), pool.getStats().getConnectionsCreated());
+ assertEquals(list.size(), pool.getStats().getConnectionsActive(null));
pool.close(dest1);
- assertEquals(list.size(), pool.getNumberOfActiveConnections());
- assertEquals(0, pool.getNumberSocketsDestroyed());
+ assertEquals(list.size(), pool.getStats().getConnectionsActive(null));
+ assertEquals(0, pool.getStats().getConnectionsDestroyed());
for(ClientRequestExecutor sas: list)
pool.checkin(dest1, sas);
- assertEquals(0, pool.getNumberOfActiveConnections());
- assertEquals(list.size(), pool.getNumberSocketsDestroyed());
- assertEquals(0, pool.getNumberOfCheckedInConnections());
+ assertEquals(0, pool.getStats().getConnectionsActive(null));
+ assertEquals(list.size(), pool.getStats().getConnectionsCreated());
}
@Test
174 test/unit/voldemort/store/stats/ClientSocketStatsTest.java
View
@@ -0,0 +1,174 @@
+/*
+ * Copyright 2008-2011 LinkedIn, Inc
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License"); you may not
+ * use this file except in compliance with the License. You may obtain a copy of
+ * the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package voldemort.store.stats;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertNotNull;
+
+import org.junit.Before;
+import org.junit.Test;
+
+import voldemort.ServerTestUtils;
+import voldemort.client.protocol.RequestFormatType;
+import voldemort.store.socket.SocketDestination;
+import voldemort.store.socket.clientrequest.ClientRequestExecutor;
+import voldemort.utils.pool.KeyedResourcePool;
+
+public class ClientSocketStatsTest {
+
+ private ClientSocketStats masterStats;
+ private int port;
+ private SocketDestination dest1;
+ private SocketDestination dest2;
+ private KeyedResourcePool<SocketDestination, ClientRequestExecutor> pool;
+
+ @Before
+ public void setUp() throws Exception {
+ this.port = ServerTestUtils.findFreePort();
+ this.dest1 = new SocketDestination("localhost", port, RequestFormatType.VOLDEMORT_V1);
+ this.dest2 = new SocketDestination("localhost", port + 1, RequestFormatType.VOLDEMORT_V1);
+ this.masterStats = new ClientSocketStats(null);
+ pool = null;
+ }
+
+ @Test
+ public void testNewNodeStatsObject() {
+ ClientSocketStats stats = new ClientSocketStats(masterStats, dest1, pool);
+ assertNotNull(stats);
+ }
+
+ @Test
+ public void testNewAggrNodeStatsObject() {
+ ClientSocketStats stats = masterStats;
+ assertNotNull(stats);
+ assertEquals(0, stats.getConnectionsCreated());
+ assertEquals(0, stats.getConnectionsDestroyed());
+ assertEquals(0, stats.getConnectionsCheckedout());
+ assertEquals(-1, stats.getAveWaitUs());
+ }
+
+ @Test
+ public void testConnectionCreate() {
+ ClientSocketStats stats = masterStats;
+ stats.connectionCreate(dest1);
+ stats.connectionCreate(dest2);
+ stats.connectionCreate(dest1);
+ assertEquals(3, stats.getConnectionsCreated());
+ assertEquals(2, stats.getStatsMap().get(dest1).getConnectionsCreated());
+ assertEquals(1, stats.getStatsMap().get(dest2).getConnectionsCreated());
+ }
+
+ @Test
+ public void testConnectionDestroy() {
+ ClientSocketStats stats = masterStats;
+ stats.connectionDestroy(dest1);
+ stats.connectionDestroy(dest2);
+ stats.connectionDestroy(dest1);
+ assertEquals(3, stats.getConnectionsDestroyed());
+ assertEquals(2, stats.getStatsMap().get(dest1).getConnectionsDestroyed());
+ assertEquals(1, stats.getStatsMap().get(dest2).getConnectionsDestroyed());
+ }
+
+ @Test
+ public void testRecordCheckoutTimeNsOnce() {
+ ClientSocketStats stats = masterStats;
+ assertEquals(0, stats.getConnectionsCheckedout());
+
+ stats.recordCheckoutTimeUs(dest1, 100);
+ // check parent
+ assertEquals(1, stats.getConnectionsCheckedout());
+ assertEquals(100, stats.getWaitHistogram().getQuantile(0.99));
+
+ // check child
+ ClientSocketStats child = stats.getStatsMap().get(dest1);
+ assertNotNull(child);
+ assertEquals(1, child.getConnectionsCheckedout());
+ assertEquals(100, child.getWaitHistogram().getQuantile(0.99));
+ }
+
+ @Test
+ public void testRecordCheckoutTimeNsMultiple() {
+ ClientSocketStats stats = masterStats;
+ assertEquals(0, stats.getConnectionsCheckedout());
+
+ stats.recordCheckoutTimeUs(dest1, 100);
+ stats.recordCheckoutTimeUs(dest1, 200);
+ stats.recordCheckoutTimeUs(dest1, 300);
+ stats.recordCheckoutTimeUs(dest1, 400);
+ stats.recordCheckoutTimeUs(dest2, 500);
+ stats.recordCheckoutTimeUs(dest2, 600);
+ stats.recordCheckoutTimeUs(dest1, 700);
+ stats.recordCheckoutTimeUs(dest1, 800);
+ stats.recordCheckoutTimeUs(dest2, 900);
+
+ // check parent
+ assertEquals(9, stats.getConnectionsCheckedout());
+ assertEquals(900, stats.getWaitHistogram().getQuantile(0.99));
+
+ // check child1
+ ClientSocketStats child1 = stats.getStatsMap().get(dest1);
+ assertNotNull(child1);
+ assertEquals(6, child1.getConnectionsCheckedout());
+ assertEquals(100, child1.getWaitHistogram().getQuantile(0.1));
+ assertEquals(300, child1.getWaitHistogram().getQuantile(0.5));
+ assertEquals(800, child1.getWaitHistogram().getQuantile(0.99));
+
+ // check child2
+ ClientSocketStats child2 = stats.getStatsMap().get(dest2);
+ assertNotNull(child2);
+ assertEquals(3, child2.getConnectionsCheckedout());
+ assertEquals(500, child2.getWaitHistogram().getQuantile(0.1));
+ assertEquals(600, child2.getWaitHistogram().getQuantile(0.5));
+ assertEquals(900, child2.getWaitHistogram().getQuantile(0.99));
+ }
+
+ @Test
+ public void testSetMonitoringInterval() {
+ ClientSocketStats stats = masterStats;
+ stats.setMonitoringInterval(9);
+ stats.recordCheckoutTimeUs(dest1, 100);
+ stats.recordCheckoutTimeUs(dest1, 200);
+ stats.recordCheckoutTimeUs(dest1, 300);
+ stats.recordCheckoutTimeUs(dest1, 400);
+ stats.recordCheckoutTimeUs(dest1, 500);
+ stats.recordCheckoutTimeUs(dest1, 600);
+ stats.recordCheckoutTimeUs(dest2, 700);
+ stats.recordCheckoutTimeUs(dest2, 800);
+ // before interval based reset
+ // check parent
+ assertEquals(8, stats.getConnectionsCheckedout());
+ assertEquals(450, stats.getAveWaitUs());
+ // check child
+ ClientSocketStats child1 = stats.getStatsMap().get(dest1);
+ ClientSocketStats child2 = stats.getStatsMap().get(dest2);
+ assertEquals(6, child1.getConnectionsCheckedout());
+ assertEquals(2, child2.getConnectionsCheckedout());
+ assertEquals(350, child1.getAveWaitUs());
+ assertEquals(750, child2.getAveWaitUs());
+
+ // after interval based reset
+ stats.recordCheckoutTimeUs(dest2, 900000);
+ // check parent
+ assertEquals(-1, stats.getAveWaitUs());
+ assertEquals(0, stats.getConnectionsCheckedout());
+ // check child
+ assertEquals(-1, child1.getAveWaitUs());
+ assertEquals(0, child1.getConnectionsCheckedout());
+ assertEquals(-1, child2.getAveWaitUs());
+ assertEquals(0, child2.getConnectionsCheckedout());
+ }
+}
Please sign in to comment.
Something went wrong with that request. Please try again.