This is an automated email from the ASF dual-hosted git repository.

ycai pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/cassandra.git


The following commit(s) were added to refs/heads/trunk by this push:
     new 845077f  Make JMXTimer expose attributes using consistent time unit
845077f is described below

commit 845077fbc8d7102f303372eb8cb3299a155a436e
Author: Yifan Cai <y...@apache.org>
AuthorDate: Thu Jul 15 12:43:43 2021 -0700

    Make JMXTimer expose attributes using consistent time unit
    
    patch by Yifan Cai; reviewed by Caleb Rackliffe for CASSANDRA-16760
---
 CHANGES.txt                                        |  1 +
 .../metrics/CassandraMetricsRegistry.java          | 60 +++++++++++++++-----
 .../apache/cassandra/metrics/LatencyMetrics.java   |  2 +-
 .../apache/cassandra/metrics/ScalingReservoir.java | 65 ++++++++++++++++++++++
 .../apache/cassandra/utils/EstimatedHistogram.java |  4 +-
 .../metrics/CassandraMetricsRegistryTest.java      | 46 +++++++++++++++
 .../apache/cassandra/net/MessagingServiceTest.java |  4 +-
 7 files changed, 163 insertions(+), 19 deletions(-)

diff --git a/CHANGES.txt b/CHANGES.txt
index 9672441..955ab8c 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 4.1
+ * Make JMXTimer expose attributes using consistent time unit (CASSANDRA-16760)
  * Remove check on gossip status from DynamicEndpointSnitch::updateScores 
(CASSANDRA-11671)
  * Fix AbstractReadQuery::toCQLString not returning valid CQL (CASSANDRA-16510)
  * Log when compacting many tombstones (CASSANDRA-16780)
diff --git 
a/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java 
b/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java
index 1ae2455..90bbe15 100644
--- a/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java
+++ b/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java
@@ -31,6 +31,7 @@ import javax.management.ObjectName;
 import com.google.common.annotations.VisibleForTesting;
 
 import com.codahale.metrics.*;
+import org.apache.cassandra.utils.EstimatedHistogram;
 import org.apache.cassandra.utils.MBeanWrapper;
 
 /**
@@ -98,19 +99,50 @@ public class CassandraMetricsRegistry extends MetricRegistry
 
     public Timer timer(MetricName name)
     {
-        Timer timer = register(name, new Timer(new 
DecayingEstimatedHistogramReservoir()));
+        return timer(name, TimeUnit.MICROSECONDS);
+    }
+
+    public Timer timer(MetricName name, MetricName alias)
+    {
+        return timer(name, alias, TimeUnit.MICROSECONDS);
+    }
+
+    public Timer timer(MetricName name, TimeUnit durationUnit)
+    {
+        Timer timer = register(name, new 
Timer(CassandraMetricsRegistry.createReservoir(TimeUnit.MICROSECONDS)));
         registerMBean(timer, name.getMBeanName());
 
         return timer;
     }
 
-    public Timer timer(MetricName name, MetricName alias)
+    public Timer timer(MetricName name, MetricName alias, TimeUnit 
durationUnit)
     {
-        Timer timer = timer(name);
+        Timer timer = timer(name, durationUnit);
         registerAlias(name, alias);
         return timer;
     }
 
+    public static Reservoir createReservoir(TimeUnit durationUnit)
+    {
+        Reservoir reservoir;
+        if (durationUnit != TimeUnit.NANOSECONDS)
+        {
+            Reservoir underlying = new 
DecayingEstimatedHistogramReservoir(DecayingEstimatedHistogramReservoir.DEFAULT_ZERO_CONSIDERATION,
+                                                                           
EstimatedHistogram.DEFAULT_BUCKET_COUNT,
+                                                                           
DecayingEstimatedHistogramReservoir.DEFAULT_STRIPE_COUNT);
+            // less buckets (90) should suffice if timer is not based on nanos
+            reservoir = new ScalingReservoir(underlying,
+                                             // timer update values in nanos.
+                                             v -> durationUnit.convert(v, 
TimeUnit.NANOSECONDS));
+        }
+        else
+        {
+            // Use more buckets if timer is created with nanos resolution.
+            reservoir = new DecayingEstimatedHistogramReservoir();
+        }
+        return reservoir;
+    }
+
     public <T extends Metric> T register(MetricName name, T metric)
     {
         try
@@ -532,7 +564,6 @@ public class CassandraMetricsRegistry extends MetricRegistry
     static class JmxTimer extends JmxMeter implements JmxTimerMBean
     {
         private final Timer metric;
-        private final double durationFactor;
         private final String durationUnit;
         private long[] last = null;
 
@@ -543,68 +574,67 @@ public class CassandraMetricsRegistry extends 
MetricRegistry
         {
             super(metric, objectName, rateUnit);
             this.metric = metric;
-            this.durationFactor = 1.0 / durationUnit.toNanos(1);
             this.durationUnit = durationUnit.toString().toLowerCase(Locale.US);
         }
 
         @Override
         public double get50thPercentile()
         {
-            return metric.getSnapshot().getMedian() * durationFactor;
+            return metric.getSnapshot().getMedian();
         }
 
         @Override
         public double getMin()
         {
-            return metric.getSnapshot().getMin() * durationFactor;
+            return metric.getSnapshot().getMin();
         }
 
         @Override
         public double getMax()
         {
-            return metric.getSnapshot().getMax() * durationFactor;
+            return metric.getSnapshot().getMax();
         }
 
         @Override
         public double getMean()
         {
-            return metric.getSnapshot().getMean() * durationFactor;
+            return metric.getSnapshot().getMean();
         }
 
         @Override
         public double getStdDev()
         {
-            return metric.getSnapshot().getStdDev() * durationFactor;
+            return metric.getSnapshot().getStdDev();
         }
 
         @Override
         public double get75thPercentile()
         {
-            return metric.getSnapshot().get75thPercentile() * durationFactor;
+            return metric.getSnapshot().get75thPercentile();
         }
 
         @Override
         public double get95thPercentile()
         {
-            return metric.getSnapshot().get95thPercentile() * durationFactor;
+            return metric.getSnapshot().get95thPercentile();
         }
 
         @Override
         public double get98thPercentile()
         {
-            return metric.getSnapshot().get98thPercentile() * durationFactor;
+            return metric.getSnapshot().get98thPercentile();
         }
 
         @Override
         public double get99thPercentile()
         {
-            return metric.getSnapshot().get99thPercentile() * durationFactor;
+            return metric.getSnapshot().get99thPercentile();
         }
 
         @Override
         public double get999thPercentile()
         {
-            return metric.getSnapshot().get999thPercentile() * durationFactor;
+            return metric.getSnapshot().get999thPercentile();
         }
 
         @Override
diff --git a/src/java/org/apache/cassandra/metrics/LatencyMetrics.java 
b/src/java/org/apache/cassandra/metrics/LatencyMetrics.java
index bf0bc1f..af8ad71 100644
--- a/src/java/org/apache/cassandra/metrics/LatencyMetrics.java
+++ b/src/java/org/apache/cassandra/metrics/LatencyMetrics.java
@@ -89,7 +89,7 @@ public class LatencyMetrics
         this.aliasFactory = aliasFactory;
         this.namePrefix = namePrefix;
 
-        LatencyMetricsTimer timer = new LatencyMetrics.LatencyMetricsTimer(new 
DecayingEstimatedHistogramReservoir());
+        LatencyMetricsTimer timer = new 
LatencyMetrics.LatencyMetricsTimer(CassandraMetricsRegistry.createReservoir(TimeUnit.MICROSECONDS));
         Counter counter = new LatencyMetricsCounter();
 
         if (aliasFactory == null)
diff --git a/src/java/org/apache/cassandra/metrics/ScalingReservoir.java 
b/src/java/org/apache/cassandra/metrics/ScalingReservoir.java
new file mode 100644
index 0000000..31e7744
--- /dev/null
+++ b/src/java/org/apache/cassandra/metrics/ScalingReservoir.java
@@ -0,0 +1,65 @@
+/*
+ * 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
+ *
+ *     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 org.apache.cassandra.metrics;
+
+import com.codahale.metrics.Reservoir;
+import com.codahale.metrics.Snapshot;
+
+/**
+ * A reservoir that scales the values before updating.
+ */
+public class ScalingReservoir implements Reservoir
+{
+    private final Reservoir delegate;
+    private final ScaleFunction scaleFunc;
+
+    public ScalingReservoir(Reservoir reservoir, ScaleFunction scaleFunc)
+    {
+        this.delegate = reservoir;
+        this.scaleFunc = scaleFunc;
+    }
+
+    @Override
+    public int size()
+    {
+        return delegate.size();
+    }
+
+    @Override
+    public void update(long value)
+    {
+        delegate.update(scaleFunc.apply(value));
+    }
+
+    @Override
+    public Snapshot getSnapshot()
+    {
+        return delegate.getSnapshot();
+    }
+
+    /**
+     * Scale the input value.
+     *
+     * Not using {@linkplain java.util.function.Function<Long, Long>} to avoid 
auto-boxing.
+     */
+    @FunctionalInterface
+    public static interface ScaleFunction
+    {
+        long apply(long value);
+    }
+}
diff --git a/src/java/org/apache/cassandra/utils/EstimatedHistogram.java 
b/src/java/org/apache/cassandra/utils/EstimatedHistogram.java
index a494b3a..ed3dccc 100644
--- a/src/java/org/apache/cassandra/utils/EstimatedHistogram.java
+++ b/src/java/org/apache/cassandra/utils/EstimatedHistogram.java
@@ -34,6 +34,8 @@ public class EstimatedHistogram
 {
     public static final EstimatedHistogramSerializer serializer = new 
EstimatedHistogramSerializer();
 
+    public static final int DEFAULT_BUCKET_COUNT = 90;
+
     /**
      * The series of values to which the counts in `buckets` correspond:
      * 1, 2, 3, 4, 5, 6, 7, 8, 10, 12, 14, 17, 20, etc.
@@ -52,7 +54,7 @@ public class EstimatedHistogram
 
     public EstimatedHistogram()
     {
-        this(90);
+        this(DEFAULT_BUCKET_COUNT);
     }
 
     public EstimatedHistogram(int bucketCount)
diff --git 
a/test/unit/org/apache/cassandra/metrics/CassandraMetricsRegistryTest.java 
b/test/unit/org/apache/cassandra/metrics/CassandraMetricsRegistryTest.java
index cd9866c..0dd7eab 100644
--- a/test/unit/org/apache/cassandra/metrics/CassandraMetricsRegistryTest.java
+++ b/test/unit/org/apache/cassandra/metrics/CassandraMetricsRegistryTest.java
@@ -24,13 +24,17 @@ import static org.junit.Assert.*;
 
 import java.lang.management.ManagementFactory;
 import java.util.Collection;
+import java.util.concurrent.TimeUnit;
 
+import com.codahale.metrics.Timer;
 import org.apache.cassandra.metrics.CassandraMetricsRegistry.MetricName;
+
 import org.junit.Test;
 
 import com.codahale.metrics.jvm.BufferPoolMetricSet;
 import com.codahale.metrics.jvm.GarbageCollectorMetricSet;
 import com.codahale.metrics.jvm.MemoryUsageGaugeSet;
+import org.apache.cassandra.utils.EstimatedHistogram;
 
 
 public class CassandraMetricsRegistryTest
@@ -107,4 +111,46 @@ public class CassandraMetricsRegistryTest
         assertArrayEquals(count, CassandraMetricsRegistry.delta(count, new 
long[3]));
         assertArrayEquals(new long[6], CassandraMetricsRegistry.delta(count, 
new long[]{0, 1, 2, 3, 4, 5, 6, 7, 8, 9}));
     }
+
+    /**
+     * Test the updated timer values are estimated correctly (i.e., in the 
valid range, 1.2) in the micros based histogram.
+     */
+    @Test
+    public void testTimer()
+    {
+        long[] offsets = new EstimatedHistogram().getBucketOffsets();
+        Timer timer = new 
Timer(CassandraMetricsRegistry.createReservoir(TimeUnit.MICROSECONDS));
+        timer.update(42, TimeUnit.NANOSECONDS);
+        timer.update(100, TimeUnit.NANOSECONDS);
+        timer.update(42, TimeUnit.MICROSECONDS);
+        timer.update(100, TimeUnit.MICROSECONDS);
+        timer.update(42, TimeUnit.MILLISECONDS);
+        timer.update(100, TimeUnit.MILLISECONDS);
+        long[] counts = timer.getSnapshot().getValues();
+        int expectedBucketsWithValues = 5;
+        int bucketsWithValues = 0;
+        for (int i = 0; i < counts.length; i++)
+        {
+            if (counts[i] != 0)
+            {
+                bucketsWithValues ++;
+                assertTrue(
+                inRange(offsets[i], TimeUnit.NANOSECONDS.toMicros(42), 1.2)
+                || inRange(offsets[i], TimeUnit.NANOSECONDS.toMicros(100), 1.2)
+                || inRange(offsets[i], TimeUnit.MICROSECONDS.toMicros(42), 1.2)
+                || inRange(offsets[i], TimeUnit.MICROSECONDS.toMicros(100), 
1.2)
+                || inRange(offsets[i], TimeUnit.MILLISECONDS.toMicros(42), 1.2)
+                || inRange(offsets[i], TimeUnit.MILLISECONDS.toMicros(100), 
1.2)
+                );
+            }
+        }
+        assertEquals("42 and 100 nanos should both be put in the first bucket",
+                            2, counts[0]);
+        assertEquals(expectedBucketsWithValues, bucketsWithValues);
+    }
+
+    private boolean inRange(long anchor, long input, double range)
+    {
+        return input / ((double) anchor) < range;
+    }
 }
diff --git a/test/unit/org/apache/cassandra/net/MessagingServiceTest.java 
b/test/unit/org/apache/cassandra/net/MessagingServiceTest.java
index fd78e2a..8870a4e 100644
--- a/test/unit/org/apache/cassandra/net/MessagingServiceTest.java
+++ b/test/unit/org/apache/cassandra/net/MessagingServiceTest.java
@@ -156,7 +156,7 @@ public class MessagingServiceTest
         addDCLatency(sentAt, now);
         assertNotNull(dcLatency.get("datacenter1"));
         assertEquals(1, dcLatency.get("datacenter1").dcLatency.getCount());
-        long expectedBucket = 
bucketOffsets[Math.abs(Arrays.binarySearch(bucketOffsets, 
MILLISECONDS.toNanos(latency))) - 1];
+        long expectedBucket = 
bucketOffsets[Math.abs(Arrays.binarySearch(bucketOffsets, 
MILLISECONDS.toMicros(latency))) - 1];
         assertEquals(expectedBucket, 
dcLatency.get("datacenter1").dcLatency.getSnapshot().getMax());
     }
 
@@ -186,7 +186,7 @@ public class MessagingServiceTest
         Map<Verb, Timer> queueWaitLatency = 
MessagingService.instance().metrics.internalLatency;
         MessagingService.instance().metrics.recordInternalLatency(verb, 
latency, MILLISECONDS);
         assertEquals(1, queueWaitLatency.get(verb).getCount());
-        long expectedBucket = 
bucketOffsets[Math.abs(Arrays.binarySearch(bucketOffsets, 
MILLISECONDS.toNanos(latency))) - 1];
+        long expectedBucket = 
bucketOffsets[Math.abs(Arrays.binarySearch(bucketOffsets, 
MILLISECONDS.toMicros(latency))) - 1];
         assertEquals(expectedBucket, 
queueWaitLatency.get(verb).getSnapshot().getMax());
     }
 

---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscr...@cassandra.apache.org
For additional commands, e-mail: commits-h...@cassandra.apache.org

Reply via email to