This is an automated email from the ASF dual-hosted git repository. domgarguilo pushed a commit to branch 2.1 in repository https://gitbox.apache.org/repos/asf/accumulo.git
The following commit(s) were added to refs/heads/2.1 by this push: new 41fd482771 Replace OpTimer with new Timer class (#4784) 41fd482771 is described below commit 41fd482771d5e85992f830d0e94fa3083fd64a10 Author: Dom G. <domgargu...@apache.org> AuthorDate: Fri Aug 2 14:08:11 2024 -0400 Replace OpTimer with new Timer class (#4784) * Replace OpTimer with new Timer class --- .../accumulo/core/client/ZooKeeperInstance.java | 11 +- .../accumulo/core/clientImpl/ClientContext.java | 16 +- .../core/clientImpl/NamespaceOperationsImpl.java | 16 +- .../core/clientImpl/RootTabletLocator.java | 10 +- .../core/clientImpl/TableOperationsImpl.java | 24 ++- .../core/clientImpl/TabletLocatorImpl.java | 24 ++- .../TabletServerBatchReaderIterator.java | 14 +- .../accumulo/core/clientImpl/ThriftScanner.java | 31 ++-- .../core/metadata/MetadataLocationObtainer.java | 12 +- .../org/apache/accumulo/core/util/OpTimer.java | 131 ---------------- .../java/org/apache/accumulo/core/util/Timer.java | 91 +++++++++++ .../org/apache/accumulo/core/util/OpTimerTest.java | 168 --------------------- .../org/apache/accumulo/core/util/TimerTest.java | 109 +++++++++++++ 13 files changed, 272 insertions(+), 385 deletions(-) diff --git a/core/src/main/java/org/apache/accumulo/core/client/ZooKeeperInstance.java b/core/src/main/java/org/apache/accumulo/core/client/ZooKeeperInstance.java index aa96404a80..fbe17c6386 100644 --- a/core/src/main/java/org/apache/accumulo/core/client/ZooKeeperInstance.java +++ b/core/src/main/java/org/apache/accumulo/core/client/ZooKeeperInstance.java @@ -19,7 +19,7 @@ package org.apache.accumulo.core.client; import static com.google.common.base.Preconditions.checkArgument; -import static java.util.concurrent.TimeUnit.SECONDS; +import static java.util.concurrent.TimeUnit.MILLISECONDS; import java.util.List; import java.util.Properties; @@ -43,7 +43,7 @@ import org.apache.accumulo.core.metadata.schema.TabletsMetadata; import org.apache.accumulo.core.singletons.SingletonManager; import org.apache.accumulo.core.singletons.SingletonManager.Mode; import org.apache.accumulo.core.singletons.SingletonReservation; -import org.apache.accumulo.core.util.OpTimer; +import org.apache.accumulo.core.util.Timer; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -138,21 +138,20 @@ public class ZooKeeperInstance implements Instance { @Override public String getRootTabletLocation() { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Looking up root tablet location in zookeeper.", Thread.currentThread().getId()); - timer = new OpTimer().start(); + timer = Timer.startNew(); } Location loc = TabletsMetadata .getRootMetadata(Constants.ZROOT + "/" + getInstanceID(), zooCache).getLocation(); if (timer != null) { - timer.stop(); log.trace("tid={} Found root tablet at {} in {}", Thread.currentThread().getId(), loc, - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } if (loc == null || loc.getType() != LocationType.CURRENT) { diff --git a/core/src/main/java/org/apache/accumulo/core/clientImpl/ClientContext.java b/core/src/main/java/org/apache/accumulo/core/clientImpl/ClientContext.java index 204665d0fc..b6c30891b6 100644 --- a/core/src/main/java/org/apache/accumulo/core/clientImpl/ClientContext.java +++ b/core/src/main/java/org/apache/accumulo/core/clientImpl/ClientContext.java @@ -99,8 +99,8 @@ import org.apache.accumulo.core.singletons.SingletonReservation; import org.apache.accumulo.core.spi.common.ServiceEnvironment; import org.apache.accumulo.core.spi.scan.ScanServerInfo; import org.apache.accumulo.core.spi.scan.ScanServerSelector; -import org.apache.accumulo.core.util.OpTimer; import org.apache.accumulo.core.util.Pair; +import org.apache.accumulo.core.util.Timer; import org.apache.accumulo.core.util.tables.TableZooHelper; import org.apache.accumulo.core.util.threads.ThreadPools; import org.apache.accumulo.core.util.threads.Threads; @@ -478,21 +478,20 @@ public class ClientContext implements AccumuloClient { public String getRootTabletLocation() { ensureOpen(); - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Looking up root tablet location in zookeeper.", Thread.currentThread().getId()); - timer = new OpTimer().start(); + timer = Timer.startNew(); } Location loc = getAmple().readTablet(RootTable.EXTENT, ReadConsistency.EVENTUAL, LOCATION).getLocation(); if (timer != null) { - timer.stop(); log.trace("tid={} Found root tablet at {} in {}", Thread.currentThread().getId(), loc, - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } if (loc == null || loc.getType() != LocationType.CURRENT) { @@ -517,20 +516,19 @@ public class ClientContext implements AccumuloClient { var zLockManagerPath = ServiceLock.path(Constants.ZROOT + "/" + instanceId + Constants.ZMANAGER_LOCK); - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Looking up manager location in zookeeper.", Thread.currentThread().getId()); - timer = new OpTimer().start(); + timer = Timer.startNew(); } byte[] loc = zooCache.getLockData(zLockManagerPath); if (timer != null) { - timer.stop(); log.trace("tid={} Found manager at {} in {}", Thread.currentThread().getId(), (loc == null ? "null" : new String(loc, UTF_8)), - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } if (loc == null) { diff --git a/core/src/main/java/org/apache/accumulo/core/clientImpl/NamespaceOperationsImpl.java b/core/src/main/java/org/apache/accumulo/core/clientImpl/NamespaceOperationsImpl.java index 0b1d496223..adcd02e4f4 100644 --- a/core/src/main/java/org/apache/accumulo/core/clientImpl/NamespaceOperationsImpl.java +++ b/core/src/main/java/org/apache/accumulo/core/clientImpl/NamespaceOperationsImpl.java @@ -62,8 +62,8 @@ import org.apache.accumulo.core.rpc.clients.ThriftClientTypes; import org.apache.accumulo.core.trace.TraceUtil; import org.apache.accumulo.core.util.LocalityGroupUtil; import org.apache.accumulo.core.util.LocalityGroupUtil.LocalityGroupConfigurationError; -import org.apache.accumulo.core.util.OpTimer; import org.apache.accumulo.core.util.Retry; +import org.apache.accumulo.core.util.Timer; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -82,19 +82,18 @@ public class NamespaceOperationsImpl extends NamespaceOperationsHelper { @Override public SortedSet<String> list() { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Fetching list of namespaces...", Thread.currentThread().getId()); - timer = new OpTimer().start(); + timer = Timer.startNew(); } TreeSet<String> namespaces = new TreeSet<>(Namespaces.getNameToIdMap(context).keySet()); if (timer != null) { - timer.stop(); log.trace("tid={} Fetched {} namespaces in {}", Thread.currentThread().getId(), - namespaces.size(), String.format("%.3f secs", timer.scale(SECONDS))); + namespaces.size(), String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } return namespaces; @@ -104,20 +103,19 @@ public class NamespaceOperationsImpl extends NamespaceOperationsHelper { public boolean exists(String namespace) { EXISTING_NAMESPACE_NAME.validate(namespace); - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Checking if namespace {} exists", Thread.currentThread().getId(), namespace); - timer = new OpTimer().start(); + timer = Timer.startNew(); } boolean exists = Namespaces.namespaceNameExists(context, namespace); if (timer != null) { - timer.stop(); log.trace("tid={} Checked existence of {} in {}", Thread.currentThread().getId(), exists, - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } return exists; diff --git a/core/src/main/java/org/apache/accumulo/core/clientImpl/RootTabletLocator.java b/core/src/main/java/org/apache/accumulo/core/clientImpl/RootTabletLocator.java index 28aa5b9870..7fa49e5794 100644 --- a/core/src/main/java/org/apache/accumulo/core/clientImpl/RootTabletLocator.java +++ b/core/src/main/java/org/apache/accumulo/core/clientImpl/RootTabletLocator.java @@ -19,7 +19,6 @@ package org.apache.accumulo.core.clientImpl; import static java.util.concurrent.TimeUnit.MILLISECONDS; -import static java.util.concurrent.TimeUnit.SECONDS; import static org.apache.accumulo.core.metadata.schema.TabletMetadata.ColumnType.LOCATION; import static org.apache.accumulo.core.util.UtilWaitThread.sleepUninterruptibly; @@ -38,7 +37,7 @@ import org.apache.accumulo.core.metadata.RootTable; import org.apache.accumulo.core.metadata.schema.Ample.ReadConsistency; import org.apache.accumulo.core.metadata.schema.TabletMetadata.Location; import org.apache.accumulo.core.metadata.schema.TabletMetadata.LocationType; -import org.apache.accumulo.core.util.OpTimer; +import org.apache.accumulo.core.util.Timer; import org.apache.hadoop.io.Text; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -100,21 +99,20 @@ public class RootTabletLocator extends TabletLocator { protected TabletLocation getRootTabletLocation(ClientContext context) { Logger log = LoggerFactory.getLogger(this.getClass()); - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Looking up root tablet location in zookeeper.", Thread.currentThread().getId()); - timer = new OpTimer().start(); + timer = Timer.startNew(); } Location loc = context.getAmple() .readTablet(RootTable.EXTENT, ReadConsistency.EVENTUAL, LOCATION).getLocation(); if (timer != null) { - timer.stop(); log.trace("tid={} Found root tablet at {} in {}", Thread.currentThread().getId(), loc, - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } if (loc == null || loc.getType() != LocationType.CURRENT) { diff --git a/core/src/main/java/org/apache/accumulo/core/clientImpl/TableOperationsImpl.java b/core/src/main/java/org/apache/accumulo/core/clientImpl/TableOperationsImpl.java index c3b51237b6..0d69699164 100644 --- a/core/src/main/java/org/apache/accumulo/core/clientImpl/TableOperationsImpl.java +++ b/core/src/main/java/org/apache/accumulo/core/clientImpl/TableOperationsImpl.java @@ -140,10 +140,10 @@ import org.apache.accumulo.core.util.HostAndPort; import org.apache.accumulo.core.util.LocalityGroupUtil; import org.apache.accumulo.core.util.LocalityGroupUtil.LocalityGroupConfigurationError; import org.apache.accumulo.core.util.MapCounter; -import org.apache.accumulo.core.util.OpTimer; import org.apache.accumulo.core.util.Pair; import org.apache.accumulo.core.util.Retry; import org.apache.accumulo.core.util.TextUtil; +import org.apache.accumulo.core.util.Timer; import org.apache.accumulo.core.volume.VolumeConfiguration; import org.apache.hadoop.fs.FileStatus; import org.apache.hadoop.fs.FileSystem; @@ -177,19 +177,18 @@ public class TableOperationsImpl extends TableOperationsHelper { @Override public SortedSet<String> list() { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Fetching list of tables...", Thread.currentThread().getId()); - timer = new OpTimer().start(); + timer = Timer.startNew(); } TreeSet<String> tableNames = new TreeSet<>(context.getTableNameToIdMap().keySet()); if (timer != null) { - timer.stop(); log.trace("tid={} Fetched {} table names in {}", Thread.currentThread().getId(), - tableNames.size(), String.format("%.3f secs", timer.scale(SECONDS))); + tableNames.size(), String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } return tableNames; @@ -203,19 +202,18 @@ public class TableOperationsImpl extends TableOperationsHelper { return true; } - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Checking if table {} exists...", Thread.currentThread().getId(), tableName); - timer = new OpTimer().start(); + timer = Timer.startNew(); } boolean exists = context.getTableNameToIdMap().containsKey(tableName); if (timer != null) { - timer.stop(); log.trace("tid={} Checked existence of {} in {}", Thread.currentThread().getId(), exists, - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } return exists; @@ -570,12 +568,12 @@ public class TableOperationsImpl extends TableOperationsHelper { ThriftUtil.getClient(ThriftClientTypes.TABLET_SERVER, address, context); try { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Splitting tablet {} on {} at {}", Thread.currentThread().getId(), tl.tablet_extent, address, split); - timer = new OpTimer().start(); + timer = Timer.startNew(); } client.splitTablet(TraceUtil.traceInfo(), context.rpcCreds(), @@ -585,8 +583,8 @@ public class TableOperationsImpl extends TableOperationsHelper { tabLocator.invalidateCache(tl.tablet_extent); if (timer != null) { - timer.stop(); - log.trace("Split tablet in {}", String.format("%.3f secs", timer.scale(SECONDS))); + log.trace("Split tablet in {}", + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } } finally { diff --git a/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletLocatorImpl.java b/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletLocatorImpl.java index 02844db972..d1e935577b 100644 --- a/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletLocatorImpl.java +++ b/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletLocatorImpl.java @@ -19,7 +19,6 @@ package org.apache.accumulo.core.clientImpl; import static java.util.concurrent.TimeUnit.MILLISECONDS; -import static java.util.concurrent.TimeUnit.SECONDS; import static org.apache.accumulo.core.util.UtilWaitThread.sleepUninterruptibly; import java.util.ArrayList; @@ -47,9 +46,9 @@ import org.apache.accumulo.core.data.PartialKey; import org.apache.accumulo.core.data.Range; import org.apache.accumulo.core.data.TableId; import org.apache.accumulo.core.dataImpl.KeyExtent; -import org.apache.accumulo.core.util.OpTimer; import org.apache.accumulo.core.util.Pair; import org.apache.accumulo.core.util.TextUtil; +import org.apache.accumulo.core.util.Timer; import org.apache.hadoop.io.Text; import org.apache.hadoop.io.WritableComparator; import org.slf4j.Logger; @@ -167,12 +166,12 @@ public class TabletLocatorImpl extends TabletLocator { Map<String,TabletServerMutations<T>> binnedMutations, List<T> failures) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Binning {} mutations for table {}", Thread.currentThread().getId(), mutations.size(), tableId); - timer = new OpTimer().start(); + timer = Timer.startNew(); } ArrayList<T> notInCache = new ArrayList<>(); @@ -233,10 +232,9 @@ public class TabletLocatorImpl extends TabletLocator { } if (timer != null) { - timer.stop(); log.trace("tid={} Binned {} mutations for table {} to {} tservers in {}", Thread.currentThread().getId(), mutations.size(), tableId, binnedMutations.size(), - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } } @@ -373,12 +371,12 @@ public class TabletLocatorImpl extends TabletLocator { * logging. Therefore methods called by this are not synchronized and should not log. */ - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Binning {} ranges for table {}", Thread.currentThread().getId(), ranges.size(), tableId); - timer = new OpTimer().start(); + timer = Timer.startNew(); } LockCheckerSession lcSession = new LockCheckerSession(); @@ -412,10 +410,9 @@ public class TabletLocatorImpl extends TabletLocator { } if (timer != null) { - timer.stop(); log.trace("tid={} Binned {} ranges for table {} to {} tservers in {}", Thread.currentThread().getId(), ranges.size(), tableId, binnedRanges.size(), - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } return failures; @@ -491,12 +488,12 @@ public class TabletLocatorImpl extends TabletLocator { public TabletLocation locateTablet(ClientContext context, Text row, boolean skipRow, boolean retry) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Locating tablet table={} row={} skipRow={} retry={}", Thread.currentThread().getId(), tableId, TextUtil.truncate(row), skipRow, retry); - timer = new OpTimer().start(); + timer = Timer.startNew(); } while (true) { @@ -514,10 +511,9 @@ public class TabletLocatorImpl extends TabletLocator { } if (timer != null) { - timer.stop(); log.trace("tid={} Located tablet {} at {} in {}", Thread.currentThread().getId(), (tl == null ? "null" : tl.tablet_extent), (tl == null ? "null" : tl.tablet_location), - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } return tl; diff --git a/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletServerBatchReaderIterator.java b/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletServerBatchReaderIterator.java index 322fd0f098..79fd934861 100644 --- a/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletServerBatchReaderIterator.java +++ b/core/src/main/java/org/apache/accumulo/core/clientImpl/TabletServerBatchReaderIterator.java @@ -80,8 +80,8 @@ import org.apache.accumulo.core.tabletserver.thrift.TabletScanClientService; import org.apache.accumulo.core.trace.TraceUtil; import org.apache.accumulo.core.util.ByteBufferUtil; import org.apache.accumulo.core.util.HostAndPort; -import org.apache.accumulo.core.util.OpTimer; import org.apache.accumulo.core.util.Retry; +import org.apache.accumulo.core.util.Timer; import org.apache.thrift.TApplicationException; import org.apache.thrift.TException; import org.apache.thrift.transport.TTransportException; @@ -819,7 +819,7 @@ public class TabletServerBatchReaderIterator implements Iterator<Entry<Key,Value try { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace( @@ -828,7 +828,7 @@ public class TabletServerBatchReaderIterator implements Iterator<Entry<Key,Value sumSizes(requested.values()), options.serverSideIteratorList, options.serverSideIteratorOptions); - timer = new OpTimer().start(); + timer = Timer.startNew(); } TabletType ttype = TabletType.type(requested.keySet()); @@ -857,11 +857,10 @@ public class TabletServerBatchReaderIterator implements Iterator<Entry<Key,Value MultiScanResult scanResult = imsr.result; if (timer != null) { - timer.stop(); log.trace("tid={} Got 1st multi scan results, #results={} {} in {}", Thread.currentThread().getId(), scanResult.results.size(), (scanResult.more ? "scanID=" + imsr.scanID : ""), - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } ArrayList<Entry<Key,Value>> entries = new ArrayList<>(scanResult.results.size()); @@ -888,17 +887,16 @@ public class TabletServerBatchReaderIterator implements Iterator<Entry<Key,Value if (timer != null) { log.trace("tid={} oid={} Continuing multi scan, scanid={}", Thread.currentThread().getId(), nextOpid.get(), imsr.scanID); - timer.reset().start(); + timer.restart(); } scanResult = client.continueMultiScan(TraceUtil.traceInfo(), imsr.scanID, busyTimeout); if (timer != null) { - timer.stop(); log.trace("tid={} oid={} Got more multi scan results, #results={} {} in {}", Thread.currentThread().getId(), nextOpid.getAndIncrement(), scanResult.results.size(), (scanResult.more ? " scanID=" + imsr.scanID : ""), - String.format("%.3f secs", timer.scale(SECONDS))); + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } entries = new ArrayList<>(scanResult.results.size()); diff --git a/core/src/main/java/org/apache/accumulo/core/clientImpl/ThriftScanner.java b/core/src/main/java/org/apache/accumulo/core/clientImpl/ThriftScanner.java index c76277f79b..fc49f486f7 100644 --- a/core/src/main/java/org/apache/accumulo/core/clientImpl/ThriftScanner.java +++ b/core/src/main/java/org/apache/accumulo/core/clientImpl/ThriftScanner.java @@ -18,6 +18,7 @@ */ package org.apache.accumulo.core.clientImpl; +import static java.util.concurrent.TimeUnit.MILLISECONDS; import static java.util.concurrent.TimeUnit.SECONDS; import java.io.IOException; @@ -78,8 +79,8 @@ import org.apache.accumulo.core.tabletserver.thrift.TooManyFilesException; import org.apache.accumulo.core.trace.TraceUtil; import org.apache.accumulo.core.trace.thrift.TInfo; import org.apache.accumulo.core.util.HostAndPort; -import org.apache.accumulo.core.util.OpTimer; import org.apache.accumulo.core.util.Retry; +import org.apache.accumulo.core.util.Timer; import org.apache.hadoop.io.Text; import org.apache.thrift.TApplicationException; import org.apache.thrift.TException; @@ -270,8 +271,8 @@ public class ThriftScanner { static <T> Optional<T> waitUntil(Supplier<Optional<T>> condition, Duration maxWaitTime, String description, Duration timeoutLeft, ClientContext context, TableId tableId, Logger log) { - Retry retry = Retry.builder().infiniteRetries().retryAfter(100, TimeUnit.MILLISECONDS) - .incrementBy(100, TimeUnit.MILLISECONDS).maxWait(1, SECONDS).backOffFactor(1.5) + Retry retry = Retry.builder().infiniteRetries().retryAfter(100, MILLISECONDS) + .incrementBy(100, MILLISECONDS).maxWait(1, SECONDS).backOffFactor(1.5) .logInterval(3, TimeUnit.MINUTES).createRetry(); long startTime = System.nanoTime(); @@ -662,7 +663,7 @@ public class ThriftScanner { NotServingTabletException, TException, NoSuchScanIDException, TooManyFilesException, TSampleNotPresentException, ScanServerBusyException { - OpTimer timer = null; + Timer timer = null; final TInfo tinfo = TraceUtil.traceInfo(); @@ -690,7 +691,7 @@ public class ThriftScanner { + scanState.serverSideIteratorList + " ssio=" + scanState.serverSideIteratorOptions + " context=" + scanState.classLoaderContext; log.trace("tid={} {}", Thread.currentThread().getId(), msg); - timer = new OpTimer().start(); + timer = Timer.startNew(); } TabletType ttype = TabletType.type(loc.tablet_extent); @@ -724,7 +725,7 @@ public class ThriftScanner { if (log.isTraceEnabled()) { log.trace("tid={} {}", Thread.currentThread().getId(), msg); - timer = new OpTimer().start(); + timer = Timer.startNew(); } sr = client.continueScan(tinfo, scanState.scanID, busyTimeout); @@ -736,10 +737,10 @@ public class ThriftScanner { if (sr.more) { if (timer != null) { - timer.stop(); log.trace("tid={} Finished scan in {} #results={} scanid={}", - Thread.currentThread().getId(), String.format("%.3f secs", timer.scale(SECONDS)), - sr.results.size(), scanState.scanID); + Thread.currentThread().getId(), + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0), sr.results.size(), + scanState.scanID); } } else { // log.debug("No more : tab end row = "+loc.tablet_extent.getEndRow()+" range = @@ -748,9 +749,9 @@ public class ThriftScanner { scanState.finished = true; if (timer != null) { - timer.stop(); log.trace("tid={} Completely finished scan in {} #results={}", - Thread.currentThread().getId(), String.format("%.3f secs", timer.scale(SECONDS)), + Thread.currentThread().getId(), + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0), sr.results.size()); } @@ -760,17 +761,17 @@ public class ThriftScanner { scanState.skipStartRow = true; if (timer != null) { - timer.stop(); log.trace("tid={} Finished scanning tablet in {} #results={}", - Thread.currentThread().getId(), String.format("%.3f secs", timer.scale(SECONDS)), + Thread.currentThread().getId(), + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0), sr.results.size()); } } else { scanState.finished = true; if (timer != null) { - timer.stop(); log.trace("tid={} Completely finished in {} #results={}", - Thread.currentThread().getId(), String.format("%.3f secs", timer.scale(SECONDS)), + Thread.currentThread().getId(), + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0), sr.results.size()); } } diff --git a/core/src/main/java/org/apache/accumulo/core/metadata/MetadataLocationObtainer.java b/core/src/main/java/org/apache/accumulo/core/metadata/MetadataLocationObtainer.java index 9ac2658f63..e81ec2b485 100644 --- a/core/src/main/java/org/apache/accumulo/core/metadata/MetadataLocationObtainer.java +++ b/core/src/main/java/org/apache/accumulo/core/metadata/MetadataLocationObtainer.java @@ -18,7 +18,7 @@ */ package org.apache.accumulo.core.metadata; -import static java.util.concurrent.TimeUnit.SECONDS; +import static java.util.concurrent.TimeUnit.MILLISECONDS; import java.io.IOException; import java.util.ArrayList; @@ -58,8 +58,8 @@ import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.Cu import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.FutureLocationColumnFamily; import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.TabletColumnFamily; import org.apache.accumulo.core.security.Authorizations; -import org.apache.accumulo.core.util.OpTimer; import org.apache.accumulo.core.util.TextUtil; +import org.apache.accumulo.core.util.Timer; import org.apache.hadoop.io.Text; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -84,13 +84,13 @@ public class MetadataLocationObtainer implements TabletLocationObtainer { try { - OpTimer timer = null; + Timer timer = null; if (log.isTraceEnabled()) { log.trace("tid={} Looking up in {} row={} extent={} tserver={}", Thread.currentThread().getId(), src.tablet_extent.tableId(), TextUtil.truncate(row), src.tablet_extent, src.tablet_location); - timer = new OpTimer().start(); + timer = Timer.startNew(); } Range range = new Range(row, true, stopRow, true); @@ -123,9 +123,9 @@ public class MetadataLocationObtainer implements TabletLocationObtainer { } if (timer != null) { - timer.stop(); log.trace("tid={} Got {} results from {} in {}", Thread.currentThread().getId(), - results.size(), src.tablet_extent, String.format("%.3f secs", timer.scale(SECONDS))); + results.size(), src.tablet_extent, + String.format("%.3f secs", timer.elapsed(MILLISECONDS) / 1000.0)); } // if (log.isTraceEnabled()) log.trace("results "+results); diff --git a/core/src/main/java/org/apache/accumulo/core/util/OpTimer.java b/core/src/main/java/org/apache/accumulo/core/util/OpTimer.java deleted file mode 100644 index 6653c20d31..0000000000 --- a/core/src/main/java/org/apache/accumulo/core/util/OpTimer.java +++ /dev/null @@ -1,131 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you 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 - * - * https://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 org.apache.accumulo.core.util; - -import static java.util.concurrent.TimeUnit.NANOSECONDS; - -import java.util.concurrent.TimeUnit; - -/** - * Provides a stop watch for timing a single type of event. This code is based on the - * org.apache.hadoop.util.StopWatch available in hadoop 2.7.0 - */ -public class OpTimer { - - private boolean isStarted; - private long startNanos; - private long currentElapsedNanos; - - /** - * Returns timer running state - * - * @return true if timer is running - */ - public boolean isRunning() { - return isStarted; - } - - /** - * Start the timer instance. - * - * @return this instance for fluent chaining. - * @throws IllegalStateException if start is called on running instance. - */ - public OpTimer start() throws IllegalStateException { - if (isStarted) { - throw new IllegalStateException("OpTimer is already running"); - } - isStarted = true; - startNanos = System.nanoTime(); - return this; - } - - /** - * Stop the timer instance. - * - * @return this instance for fluent chaining. - * @throws IllegalStateException if stop is called on instance that is not running. - */ - public OpTimer stop() throws IllegalStateException { - if (!isStarted) { - throw new IllegalStateException("OpTimer is already stopped"); - } - long now = System.nanoTime(); - isStarted = false; - currentElapsedNanos += now - startNanos; - return this; - } - - /** - * Stops timer instance and current elapsed time to 0. - * - * @return this instance for fluent chaining - */ - public OpTimer reset() { - currentElapsedNanos = 0; - isStarted = false; - return this; - } - - /** - * Converts current timer value to specific unit. The conversion to courser granularities truncate - * with loss of precision. - * - * @param timeUnit the time unit that will converted to. - * @return truncated time in unit of specified time unit. - */ - public long now(TimeUnit timeUnit) { - return timeUnit.convert(now(), NANOSECONDS); - } - - /** - * Returns the current elapsed time scaled to the provided time unit. This method does not - * truncate like {@link #now(TimeUnit)} but returns the value as a double. - * - * <p> - * Note: this method is not included in the hadoop 2.7 org.apache.hadoop.util.StopWatch class. If - * that class is adopted, then provisions will be required to replace this method. - * - * @param timeUnit the time unit to scale the elapsed time to. - * @return the elapsed time of this instance scaled to the provided time unit. - */ - public double scale(TimeUnit timeUnit) { - return (double) now() / NANOSECONDS.convert(1L, timeUnit); - } - - /** - * Returns current timer elapsed time as nanoseconds. - * - * @return elapsed time in nanoseconds. - */ - public long now() { - return isStarted ? System.nanoTime() - startNanos + currentElapsedNanos : currentElapsedNanos; - } - - /** - * Return the current elapsed time in nanoseconds as a string. - * - * @return timer elapsed time as nanoseconds. - */ - @Override - public String toString() { - return String.valueOf(now()); - } - -} diff --git a/core/src/main/java/org/apache/accumulo/core/util/Timer.java b/core/src/main/java/org/apache/accumulo/core/util/Timer.java new file mode 100644 index 0000000000..cf06789993 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/util/Timer.java @@ -0,0 +1,91 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you 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 + * + * https://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 org.apache.accumulo.core.util; + +import java.time.Duration; +import java.util.concurrent.TimeUnit; + +/** + * This class provides a timer for measuring elapsed time. + */ +public final class Timer { + + private long startNanos; + + private Timer() { + this.startNanos = System.nanoTime(); + } + + /** + * Creates and starts a new Timer instance. + * + * @return a new Timer instance that is already started. + */ + public static Timer startNew() { + return new Timer(); + } + + /** + * Resets the start point for this timer. + */ + public void restart() { + this.startNanos = System.nanoTime(); + } + + private long getElapsedNanos() { + return System.nanoTime() - startNanos; + } + + /** + * Checks if the specified duration has elapsed since the timer was started. + * + * @param duration the duration to check. + * @return true if the specified duration has elapsed, false otherwise. + */ + public boolean hasElapsed(Duration duration) { + return getElapsedNanos() >= duration.toNanos(); + } + + /** + * Checks if the specified duration has elapsed since the timer was started. + * + * @param duration the duration to check. + * @param unit the TimeUnit of the duration. + * @return true if the specified duration has elapsed, false otherwise. + */ + public boolean hasElapsed(long duration, TimeUnit unit) { + return getElapsedNanos() >= unit.toNanos(duration); + } + + /** + * @return the elapsed time as a Duration. + */ + public Duration elapsed() { + return Duration.ofNanos(getElapsedNanos()); + } + + /** + * @param unit the TimeUnit to return the elapsed time in. + * @return the elapsed time in the specified TimeUnit. + */ + public long elapsed(TimeUnit unit) { + return unit.convert(getElapsedNanos(), TimeUnit.NANOSECONDS); + } + +} diff --git a/core/src/test/java/org/apache/accumulo/core/util/OpTimerTest.java b/core/src/test/java/org/apache/accumulo/core/util/OpTimerTest.java deleted file mode 100644 index 7f11ffb80d..0000000000 --- a/core/src/test/java/org/apache/accumulo/core/util/OpTimerTest.java +++ /dev/null @@ -1,168 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you 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 - * - * https://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 org.apache.accumulo.core.util; - -import static java.util.concurrent.TimeUnit.MILLISECONDS; -import static java.util.concurrent.TimeUnit.SECONDS; -import static org.junit.jupiter.api.Assertions.assertEquals; -import static org.junit.jupiter.api.Assertions.assertFalse; -import static org.junit.jupiter.api.Assertions.assertThrows; -import static org.junit.jupiter.api.Assertions.assertTrue; - -import org.junit.jupiter.api.Test; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -/** - * Exercise basic timer (org.apache.hadoop.util.StopWatch) functionality. Current usage requires - * ability to reset timer. - */ -public class OpTimerTest { - - private static final Logger log = LoggerFactory.getLogger(OpTimerTest.class); - - /** - * Validate reset functionality - */ - @Test - public void verifyReset() throws InterruptedException { - - OpTimer timer = new OpTimer().start(); - Thread.sleep(50); - - timer.stop(); - - long tValue = timer.now(); - - log.debug("Time value before reset {}", String.format("%.3f ms", timer.scale(MILLISECONDS))); - - timer.reset().start(); - Thread.sleep(1); - - timer.stop(); - - assertTrue(timer.now() > 0); - - assertTrue(tValue > timer.now()); - - timer.reset(); - - log.debug("Time value after reset {}", String.format("%.3f ms", timer.scale(MILLISECONDS))); - - assertEquals(0, timer.now()); - - } - - /** - * Verify that IllegalStateException is thrown when calling stop when timer has not been started. - */ - @Test - public void verifyExceptionCallingStopWhenNotStarted() { - OpTimer timer = new OpTimer(); - - assertFalse(timer.isRunning()); - - // should throw exception - not running - assertThrows(IllegalStateException.class, timer::stop, - "Should not be able to call stop on a timer that is not running"); - } - - /** - * Verify that IllegalStateException is thrown when calling start on running timer. - */ - @Test - public void verifyExceptionCallingStartWhenRunning() throws InterruptedException { - OpTimer timer = new OpTimer().start(); - - Thread.sleep(50); - - assertTrue(timer.isRunning()); - - // should throw exception - already running - assertThrows(IllegalStateException.class, timer::start, - "Should not be able to call start on a timer that is already running"); - } - - /** - * Verify that IllegalStateException is thrown when calling stop when not running. - */ - @Test - public void verifyExceptionCallingStopWhenNotRunning() throws InterruptedException { - OpTimer timer = new OpTimer().start(); - - Thread.sleep(50); - - assertTrue(timer.isRunning()); - - timer.stop(); - - assertFalse(timer.isRunning()); - - assertThrows(IllegalStateException.class, timer::stop, - "Should not be able to call stop on a timer that is not running"); - } - - /** - * Validate that start / stop accumulates time. - */ - @Test - public void verifyElapsed() throws InterruptedException { - OpTimer timer = new OpTimer().start(); - - Thread.sleep(50); - - timer.stop(); - - long tValue = timer.now(); - - log.debug("Time value after first stop {}", - String.format("%.3f ms", timer.scale(MILLISECONDS))); - - timer.start(); - - Thread.sleep(10); - - timer.stop(); - - log.debug("Time value after second stop {}", - String.format("%.3f ms", timer.scale(MILLISECONDS))); - - assertTrue(tValue < timer.now(), "The timer did not increase in value over time"); - } - - /** - * Validate that scale returns correct values. - */ - @Test - public void scale() throws InterruptedException { - OpTimer timer = new OpTimer().start(); - - Thread.sleep(50); - - timer.stop(); - - long tValue = timer.now(); - - double nanosPerMillisecond = 1_000_000.0; - assertEquals(tValue / nanosPerMillisecond, timer.scale(MILLISECONDS), 0.00000001); - - double nanosPerSecond = 1_000_000_000.0; - assertEquals(tValue / nanosPerSecond, timer.scale(SECONDS), 0.00000001); - } -} diff --git a/core/src/test/java/org/apache/accumulo/core/util/TimerTest.java b/core/src/test/java/org/apache/accumulo/core/util/TimerTest.java new file mode 100644 index 0000000000..b090214126 --- /dev/null +++ b/core/src/test/java/org/apache/accumulo/core/util/TimerTest.java @@ -0,0 +1,109 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you 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 + * + * https://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 org.apache.accumulo.core.util; + +import static java.util.concurrent.TimeUnit.MILLISECONDS; +import static org.junit.jupiter.api.Assertions.assertEquals; +import static org.junit.jupiter.api.Assertions.assertFalse; +import static org.junit.jupiter.api.Assertions.assertTrue; + +import java.time.Duration; +import java.util.concurrent.TimeUnit; + +import org.junit.jupiter.api.Test; + +public class TimerTest { + + @Test + public void testRestart() throws InterruptedException { + Timer timer = Timer.startNew(); + + // Perform a longer sleep initially + Thread.sleep(100); + + Duration firstElapsed = timer.elapsed(); + + assertTrue(timer.hasElapsed(Duration.ofMillis(100)), + "Should see at least the sleep time has elapsed."); + + timer.restart(); + + // Perform a shorter sleep + Thread.sleep(50); + + Duration secondElapsed = timer.elapsed(); + + // Assert that the elapsed time after restart is greater than 0 + assertFalse(secondElapsed.isNegative(), + "Elapsed time should be greater than 0 after restarting the timer."); + assertTrue(secondElapsed.compareTo(firstElapsed) < 0, + "Elapsed time after restart should be less than the initial elapsed time."); + + } + + @Test + public void testHasElapsed() throws InterruptedException { + Timer timer = Timer.startNew(); + + Thread.sleep(50); + + assertTrue(timer.hasElapsed(Duration.ofMillis(50)), + "The timer should indicate that 50 milliseconds have elapsed."); + assertFalse(timer.hasElapsed(Duration.ofMillis(100)), + "The timer should not indicate that 100 milliseconds have elapsed."); + } + + @Test + public void testHasElapsedWithTimeUnit() throws InterruptedException { + Timer timer = Timer.startNew(); + + Thread.sleep(50); + + assertTrue(timer.hasElapsed(50, MILLISECONDS), + "The timer should indicate that 50 milliseconds have elapsed."); + assertFalse(timer.hasElapsed(100, MILLISECONDS), + "The timer should not indicate that 100 milliseconds have elapsed."); + } + + @Test + public void testElapsedPrecision() throws InterruptedException { + Timer timer = Timer.startNew(); + + final int sleepMillis = 50; + Thread.sleep(sleepMillis); + + long elapsedMillis = timer.elapsed(MILLISECONDS); + assertEquals(sleepMillis, elapsedMillis, 5, "Elapsed time in milliseconds is not accurate."); + } + + @Test + public void testElapsedWithTimeUnit() throws InterruptedException { + Timer timer = Timer.startNew(); + + Thread.sleep(50); + + long elapsedMillis = timer.elapsed(MILLISECONDS); + assertEquals(50, elapsedMillis, 5, "Elapsed time in milliseconds is not accurate."); + + long elapsedSeconds = timer.elapsed(TimeUnit.SECONDS); + assertEquals(0, elapsedSeconds, + "Elapsed time in seconds should be 0 for 50 milliseconds of sleep."); + } + +}