You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by jm...@apache.org on 2015/12/04 19:20:27 UTC

[10/22] cassandra git commit: 10585-2.2 patch

10585-2.2 patch


Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo
Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/c8136b98
Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/c8136b98
Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/c8136b98

Branch: refs/heads/cassandra-3.0
Commit: c8136b98f20f5bb93bae81444ac4ad9247d9042a
Parents: d1e93e9
Author: Joshua McKenzie <jm...@apache.org>
Authored: Fri Dec 4 13:18:10 2015 -0500
Committer: Joshua McKenzie <jm...@apache.org>
Committed: Fri Dec 4 13:18:10 2015 -0500

----------------------------------------------------------------------
 .../apache/cassandra/db/ColumnFamilyStore.java  |  4 +-
 .../metrics/CASClientRequestMetrics.java        |  2 +-
 .../metrics/CassandraMetricsRegistry.java       |  6 +-
 .../cassandra/metrics/ColumnFamilyMetrics.java  | 16 +++---
 .../metrics/EstimatedHistogramReservoir.java    | 10 ++--
 .../cassandra/metrics/KeyspaceMetrics.java      |  8 +--
 .../cassandra/utils/EstimatedHistogram.java     | 31 +++++++---
 .../org/apache/cassandra/db/RowCacheTest.java   | 50 +++++++++++++++++
 .../cassandra/utils/EstimatedHistogramTest.java | 59 ++++++++++++++++++--
 9 files changed, 151 insertions(+), 35 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/ColumnFamilyStore.java b/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
index 5325c8a..7e0fbcb 100644
--- a/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
+++ b/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
@@ -1654,7 +1654,9 @@ public class ColumnFamilyStore implements ColumnFamilyStoreMBean
             {
                 metric.rowCacheHit.inc();
                 Tracing.trace("Row cache hit");
-                return filterColumnFamily(cachedCf, filter);
+                ColumnFamily result = filterColumnFamily(cachedCf, filter);
+                metric.updateSSTableIterated(0);
+                return result;
             }
 
             metric.rowCacheHitOutOfRange.inc();

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/src/java/org/apache/cassandra/metrics/CASClientRequestMetrics.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/metrics/CASClientRequestMetrics.java b/src/java/org/apache/cassandra/metrics/CASClientRequestMetrics.java
index e6f2b81..4e64cff 100644
--- a/src/java/org/apache/cassandra/metrics/CASClientRequestMetrics.java
+++ b/src/java/org/apache/cassandra/metrics/CASClientRequestMetrics.java
@@ -34,7 +34,7 @@ public class CASClientRequestMetrics extends ClientRequestMetrics
 
     public CASClientRequestMetrics(String scope) {
         super(scope);
-        contention = Metrics.histogram(factory.createMetricName("ContentionHistogram"));
+        contention = Metrics.histogram(factory.createMetricName("ContentionHistogram"), false);
         conditionNotMet =  Metrics.counter(factory.createMetricName("ConditionNotMet"));
         unfinishedCommit =  Metrics.counter(factory.createMetricName("UnfinishedCommit"));
     }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java b/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java
index 26b9c07..6fdb2ff 100644
--- a/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java
+++ b/src/java/org/apache/cassandra/metrics/CassandraMetricsRegistry.java
@@ -58,9 +58,9 @@ public class CassandraMetricsRegistry extends MetricRegistry
         return meter;
     }
 
-    public Histogram histogram(MetricName name)
+    public Histogram histogram(MetricName name, boolean considerZeroes)
     {
-        Histogram histogram = register(name, new ClearableHistogram(new EstimatedHistogramReservoir()));
+        Histogram histogram = register(name, new ClearableHistogram(new EstimatedHistogramReservoir(considerZeroes)));
         registerMBean(histogram, name.getMBeanName());
 
         return histogram;
@@ -68,7 +68,7 @@ public class CassandraMetricsRegistry extends MetricRegistry
 
     public Timer timer(MetricName name)
     {
-        Timer timer = register(name, new Timer(new EstimatedHistogramReservoir()));
+        Timer timer = register(name, new Timer(new EstimatedHistogramReservoir(false)));
         registerMBean(timer, name.getMBeanName());
 
         return timer;

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/src/java/org/apache/cassandra/metrics/ColumnFamilyMetrics.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/metrics/ColumnFamilyMetrics.java b/src/java/org/apache/cassandra/metrics/ColumnFamilyMetrics.java
index 4ab4446..40ed2e4 100644
--- a/src/java/org/apache/cassandra/metrics/ColumnFamilyMetrics.java
+++ b/src/java/org/apache/cassandra/metrics/ColumnFamilyMetrics.java
@@ -306,7 +306,7 @@ public class ColumnFamilyMetrics
                 });
             }
         });
-        sstablesPerReadHistogram = createColumnFamilyHistogram("SSTablesPerReadHistogram", cfs.keyspace.metric.sstablesPerReadHistogram);
+        sstablesPerReadHistogram = createColumnFamilyHistogram("SSTablesPerReadHistogram", cfs.keyspace.metric.sstablesPerReadHistogram, true);
         compressionRatio = createColumnFamilyGauge("CompressionRatio", new Gauge<Double>()
         {
             public Double getValue()
@@ -596,12 +596,12 @@ public class ColumnFamilyMetrics
                 return Math.max(requests, 1); // to avoid NaN.
             }
         });
-        tombstoneScannedHistogram = createColumnFamilyHistogram("TombstoneScannedHistogram", cfs.keyspace.metric.tombstoneScannedHistogram);
-        liveScannedHistogram = createColumnFamilyHistogram("LiveScannedHistogram", cfs.keyspace.metric.liveScannedHistogram);
-        colUpdateTimeDeltaHistogram = createColumnFamilyHistogram("ColUpdateTimeDeltaHistogram", cfs.keyspace.metric.colUpdateTimeDeltaHistogram);
+        tombstoneScannedHistogram = createColumnFamilyHistogram("TombstoneScannedHistogram", cfs.keyspace.metric.tombstoneScannedHistogram, false);
+        liveScannedHistogram = createColumnFamilyHistogram("LiveScannedHistogram", cfs.keyspace.metric.liveScannedHistogram, false);
+        colUpdateTimeDeltaHistogram = createColumnFamilyHistogram("ColUpdateTimeDeltaHistogram", cfs.keyspace.metric.colUpdateTimeDeltaHistogram, false);
         coordinatorReadLatency = Metrics.timer(factory.createMetricName("CoordinatorReadLatency"));
         coordinatorScanLatency = Metrics.timer(factory.createMetricName("CoordinatorScanLatency"));
-        waitingOnFreeMemtableSpace = Metrics.histogram(factory.createMetricName("WaitingOnFreeMemtableSpace"));
+        waitingOnFreeMemtableSpace = Metrics.histogram(factory.createMetricName("WaitingOnFreeMemtableSpace"), false);
 
         trueSnapshotsSize = createColumnFamilyGauge("SnapshotsSize", new Gauge<Long>()
         {
@@ -710,11 +710,11 @@ public class ColumnFamilyMetrics
      * Create a histogram-like interface that will register both a CF, keyspace and global level
      * histogram and forward any updates to both
      */
-    protected ColumnFamilyHistogram createColumnFamilyHistogram(String name, Histogram keyspaceHistogram)
+    protected ColumnFamilyHistogram createColumnFamilyHistogram(String name, Histogram keyspaceHistogram, boolean considerZeroes)
     {
-        Histogram cfHistogram = Metrics.histogram(factory.createMetricName(name));
+        Histogram cfHistogram = Metrics.histogram(factory.createMetricName(name), considerZeroes);
         register(name, cfHistogram);
-        return new ColumnFamilyHistogram(cfHistogram, keyspaceHistogram, Metrics.histogram(globalNameFactory.createMetricName(name)));
+        return new ColumnFamilyHistogram(cfHistogram, keyspaceHistogram, Metrics.histogram(globalNameFactory.createMetricName(name), considerZeroes));
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/src/java/org/apache/cassandra/metrics/EstimatedHistogramReservoir.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/metrics/EstimatedHistogramReservoir.java b/src/java/org/apache/cassandra/metrics/EstimatedHistogramReservoir.java
index 3051711..3658f3a 100644
--- a/src/java/org/apache/cassandra/metrics/EstimatedHistogramReservoir.java
+++ b/src/java/org/apache/cassandra/metrics/EstimatedHistogramReservoir.java
@@ -34,14 +34,14 @@ public class EstimatedHistogramReservoir implements Reservoir
     EstimatedHistogram histogram;
 
     // Default to >4 hours of in nanoseconds of buckets
-    public EstimatedHistogramReservoir()
+    public EstimatedHistogramReservoir(boolean considerZeroes)
     {
-        this(164);
+        this(164, considerZeroes);
     }
 
-    public EstimatedHistogramReservoir(int numBuckets)
+    public EstimatedHistogramReservoir(int numBuckets, boolean considerZeroes)
     {
-        histogram = new EstimatedHistogram(numBuckets);
+        histogram = new EstimatedHistogram(numBuckets, considerZeroes);
     }
 
     @Override
@@ -100,7 +100,7 @@ public class EstimatedHistogramReservoir implements Reservoir
         @Override
         public double getMean()
         {
-            return histogram.mean();
+            return histogram.rawMean();
         }
     }
 }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/src/java/org/apache/cassandra/metrics/KeyspaceMetrics.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/metrics/KeyspaceMetrics.java b/src/java/org/apache/cassandra/metrics/KeyspaceMetrics.java
index e9ce49c..369f323 100644
--- a/src/java/org/apache/cassandra/metrics/KeyspaceMetrics.java
+++ b/src/java/org/apache/cassandra/metrics/KeyspaceMetrics.java
@@ -221,10 +221,10 @@ public class KeyspaceMetrics
         writeLatency = new LatencyMetrics(factory, "Write");
         rangeLatency = new LatencyMetrics(factory, "Range");
         // create histograms for ColumnFamilyMetrics to replicate updates to
-        sstablesPerReadHistogram = Metrics.histogram(factory.createMetricName("SSTablesPerReadHistogram"));
-        tombstoneScannedHistogram = Metrics.histogram(factory.createMetricName("TombstoneScannedHistogram"));
-        liveScannedHistogram = Metrics.histogram(factory.createMetricName("LiveScannedHistogram"));
-        colUpdateTimeDeltaHistogram = Metrics.histogram(factory.createMetricName("ColUpdateTimeDeltaHistogram"));
+        sstablesPerReadHistogram = Metrics.histogram(factory.createMetricName("SSTablesPerReadHistogram"), true);
+        tombstoneScannedHistogram = Metrics.histogram(factory.createMetricName("TombstoneScannedHistogram"), false);
+        liveScannedHistogram = Metrics.histogram(factory.createMetricName("LiveScannedHistogram"), false);
+        colUpdateTimeDeltaHistogram = Metrics.histogram(factory.createMetricName("ColUpdateTimeDeltaHistogram"), false);
         // add manually since histograms do not use createKeyspaceGauge method
         allMetrics.addAll(Lists.newArrayList("SSTablesPerReadHistogram", "TombstoneScannedHistogram", "LiveScannedHistogram"));
 

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/src/java/org/apache/cassandra/utils/EstimatedHistogram.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/utils/EstimatedHistogram.java b/src/java/org/apache/cassandra/utils/EstimatedHistogram.java
index 93d142a..6c929df 100644
--- a/src/java/org/apache/cassandra/utils/EstimatedHistogram.java
+++ b/src/java/org/apache/cassandra/utils/EstimatedHistogram.java
@@ -57,7 +57,12 @@ public class EstimatedHistogram
 
     public EstimatedHistogram(int bucketCount)
     {
-        bucketOffsets = newOffsets(bucketCount);
+        this(bucketCount, false);
+    }
+
+    public EstimatedHistogram(int bucketCount, boolean considerZeroes)
+    {
+        bucketOffsets = newOffsets(bucketCount, considerZeroes);
         buckets = new AtomicLongArray(bucketOffsets.length + 1);
     }
 
@@ -68,12 +73,15 @@ public class EstimatedHistogram
         buckets = new AtomicLongArray(bucketData);
     }
 
-    private static long[] newOffsets(int size)
+    private static long[] newOffsets(int size, boolean considerZeroes)
     {
-        long[] result = new long[size];
+        long[] result = new long[size + (considerZeroes ? 1 : 0)];
+        int i = 0;
+        if (considerZeroes)
+            result[i++] = 0;
         long last = 1;
-        result[0] = last;
-        for (int i = 1; i < size; i++)
+        result[i++] = last;
+        for (; i < result.length; i++)
         {
             long next = Math.round(last * 1.2);
             if (next == last)
@@ -193,11 +201,20 @@ public class EstimatedHistogram
     }
 
     /**
-     * @return the mean histogram value (average of bucket offsets, weighted by count)
+     * @return the ceil of mean histogram value (average of bucket offsets, weighted by count)
      * @throws IllegalStateException if any values were greater than the largest bucket threshold
      */
     public long mean()
     {
+        return (long) Math.ceil(rawMean());
+    }
+
+    /**
+     * @return the mean histogram value (average of bucket offsets, weighted by count)
+     * @throws IllegalStateException if any values were greater than the largest bucket threshold
+     */
+    public double rawMean()
+    {
         int lastBucket = buckets.length() - 1;
         if (buckets.get(lastBucket) > 0)
             throw new IllegalStateException("Unable to compute ceiling for max when histogram overflowed");
@@ -211,7 +228,7 @@ public class EstimatedHistogram
             sum += bCount * bucketOffsets[i];
         }
 
-        return (long) Math.ceil((double) sum / elements);
+        return (double) sum / elements;
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/test/unit/org/apache/cassandra/db/RowCacheTest.java
----------------------------------------------------------------------
diff --git a/test/unit/org/apache/cassandra/db/RowCacheTest.java b/test/unit/org/apache/cassandra/db/RowCacheTest.java
index 9fb322b..0c0396c 100644
--- a/test/unit/org/apache/cassandra/db/RowCacheTest.java
+++ b/test/unit/org/apache/cassandra/db/RowCacheTest.java
@@ -46,10 +46,12 @@ import org.apache.cassandra.exceptions.ConfigurationException;
 import org.apache.cassandra.dht.ByteOrderedPartitioner.BytesToken;
 import org.apache.cassandra.locator.TokenMetadata;
 import org.apache.cassandra.locator.SimpleStrategy;
+import org.apache.cassandra.metrics.ClearableHistogram;
 import org.apache.cassandra.service.CacheService;
 import org.apache.cassandra.service.StorageService;
 import org.apache.cassandra.utils.ByteBufferUtil;
 import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
 
 public class RowCacheTest
 {
@@ -339,6 +341,54 @@ public class RowCacheTest
         }
     }
 
+    @Test
+    public void testSSTablesPerReadHistogramWhenRowCache()
+    {
+        CompactionManager.instance.disableAutoCompaction();
+
+        Keyspace keyspace = Keyspace.open(KEYSPACE_CACHED);
+        ColumnFamilyStore cachedStore  = keyspace.getColumnFamilyStore(CF_CACHED);
+
+        // empty the row cache
+        CacheService.instance.invalidateRowCache();
+
+        // set global row cache size to 1 MB
+        CacheService.instance.setRowCacheCapacityInMB(1);
+
+        // inserting 100 rows into both column families
+        SchemaLoader.insertData(KEYSPACE_CACHED, CF_CACHED, 0, 100);
+
+        //force flush for confidence that SSTables exists
+        cachedStore.forceBlockingFlush();
+
+        ((ClearableHistogram)cachedStore.metric.sstablesPerReadHistogram.cf).clear();
+
+        for (int i = 0; i < 100; i++)
+        {
+            DecoratedKey key = Util.dk("key" + i);
+
+            cachedStore.getColumnFamily(key, Composites.EMPTY, Composites.EMPTY, false, 1, System.currentTimeMillis());
+
+            long count_before = cachedStore.metric.sstablesPerReadHistogram.cf.getCount();
+            cachedStore.getColumnFamily(key, Composites.EMPTY, Composites.EMPTY, false, 1, System.currentTimeMillis());
+
+            // check that SSTablePerReadHistogram has been updated by zero,
+            // so count has been increased and in a 1/2 of requests there were zero read SSTables
+            long count_after = cachedStore.metric.sstablesPerReadHistogram.cf.getCount();
+            double belowMedian = cachedStore.metric.sstablesPerReadHistogram.cf.getSnapshot().getValue(0.49D);
+            double mean_after = cachedStore.metric.sstablesPerReadHistogram.cf.getSnapshot().getMean();
+            assertEquals("SSTablePerReadHistogram should be updated even key found in row cache", count_before + 1, count_after);
+            assertTrue("In half of requests we have not touched SSTables, " +
+                       "so 49 percentile (" + belowMedian + ") must be strongly less than 0.9", belowMedian < 0.9D);
+            assertTrue("In half of requests we have not touched SSTables, " +
+                       "so mean value (" + mean_after + ") must be strongly less than 1, but greater than 0", mean_after < 0.999D && mean_after > 0.001D);
+        }
+
+        assertEquals("Min value of SSTablesPerRead should be zero", 0, cachedStore.metric.sstablesPerReadHistogram.cf.getSnapshot().getMin());
+
+        CacheService.instance.setRowCacheCapacityInMB(0);
+    }
+
     public void rowCacheLoad(int totalKeys, int keysToSave, int offset) throws Exception
     {
         CompactionManager.instance.disableAutoCompaction();

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c8136b98/test/unit/org/apache/cassandra/utils/EstimatedHistogramTest.java
----------------------------------------------------------------------
diff --git a/test/unit/org/apache/cassandra/utils/EstimatedHistogramTest.java b/test/unit/org/apache/cassandra/utils/EstimatedHistogramTest.java
index eebaa25..b0e0e4f 100644
--- a/test/unit/org/apache/cassandra/utils/EstimatedHistogramTest.java
+++ b/test/unit/org/apache/cassandra/utils/EstimatedHistogramTest.java
@@ -28,12 +28,23 @@ public class EstimatedHistogramTest
     @Test
     public void testSimple()
     {
-        // 0 and 1 map to the same, first bucket
-        EstimatedHistogram histogram = new EstimatedHistogram();
-        histogram.add(0);
-        assertEquals(1, histogram.get(0));
-        histogram.add(1);
-        assertEquals(2, histogram.get(0));
+        {
+            // 0 and 1 map to the same, first bucket
+            EstimatedHistogram histogram = new EstimatedHistogram();
+            histogram.add(0);
+            assertEquals(1, histogram.get(0));
+            histogram.add(1);
+            assertEquals(2, histogram.get(0));
+        }
+        {
+            // 0 and 1 map to different buckets
+            EstimatedHistogram histogram = new EstimatedHistogram(90, true);
+            histogram.add(0);
+            assertEquals(1, histogram.get(0));
+            histogram.add(1);
+            assertEquals(1, histogram.get(0));
+            assertEquals(1, histogram.get(1));
+        }
     }
 
     @Test
@@ -55,6 +66,33 @@ public class EstimatedHistogramTest
     }
 
     @Test
+    public void testMean()
+    {
+        {
+            EstimatedHistogram histogram = new EstimatedHistogram();
+            for (int i = 0; i < 40; i++)
+                histogram.add(0);
+            for (int i = 0; i < 20; i++)
+                histogram.add(1);
+            for (int i = 0; i < 10; i++)
+                histogram.add(2);
+            assertEquals(70, histogram.count());
+            assertEquals(2, histogram.mean());
+        }
+        {
+            EstimatedHistogram histogram = new EstimatedHistogram(90, true);
+            for (int i = 0; i < 40; i++)
+                histogram.add(0);
+            for (int i = 0; i < 20; i++)
+                histogram.add(1);
+            for (int i = 0; i < 10; i++)
+                histogram.add(2);
+            assertEquals(70, histogram.count());
+            assertEquals(1, histogram.mean());
+        }
+    }
+
+    @Test
     public void testFindingCorrectBuckets()
     {
         EstimatedHistogram histogram = new EstimatedHistogram();
@@ -119,5 +157,14 @@ public class EstimatedHistogramTest
             assertEquals(17, histogram.percentile(0.60));
             assertEquals(20, histogram.percentile(0.80));
         }
+        {
+            EstimatedHistogram histogram = new EstimatedHistogram(90, true);
+            histogram.add(0);
+            histogram.add(0);
+            histogram.add(1);
+
+            assertEquals(0, histogram.percentile(0.5));
+            assertEquals(1, histogram.percentile(0.99));
+        }
     }
 }