Repository: cassandra Updated Branches: refs/heads/cassandra-3.0 1e80e3539 -> ef5ac1a4a refs/heads/cassandra-3.11 03155b021 -> d44a0d25b refs/heads/trunk 33ab4902a -> 733ea88fa
StreamingHistogram is not thread safe Patch by Jeff Jirsa; Reviewed by Jason Brown, Marcus Eriksson for CASSANDRA-13756 Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/ef5ac1a4 Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/ef5ac1a4 Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/ef5ac1a4 Branch: refs/heads/cassandra-3.0 Commit: ef5ac1a4abe4fb5f407c0a24f4bc808932c5d7a2 Parents: 1e80e35 Author: Jeff Jirsa <jji...@apple.com> Authored: Tue Aug 29 10:31:16 2017 -0700 Committer: Jeff Jirsa <jji...@apple.com> Committed: Tue Sep 5 09:54:27 2017 -0700 ---------------------------------------------------------------------- CHANGES.txt | 1 + .../apache/cassandra/db/ColumnFamilyStore.java | 2 +- .../io/sstable/metadata/MetadataCollector.java | 10 +- .../cassandra/utils/StreamingHistogram.java | 284 ++++++++++--------- .../microbench/StreamingHistogramBench.java | 192 +++++++------ .../cassandra/utils/StreamingHistogramTest.java | 31 +- 6 files changed, 263 insertions(+), 257 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/cassandra/blob/ef5ac1a4/CHANGES.txt ---------------------------------------------------------------------- diff --git a/CHANGES.txt b/CHANGES.txt index 1ed3fe6..853bf61 100644 --- a/CHANGES.txt +++ b/CHANGES.txt @@ -1,4 +1,5 @@ 3.0.15 + * StreamingHistogram is not thread safe (CASSANDRA-13756) * Fix MV timestamp issues (CASSANDRA-11500) * Better tolerate improperly formatted bcrypt hashes (CASSANDRA-13626) * Fix race condition in read command serialization (CASSANDRA-13363) http://git-wip-us.apache.org/repos/asf/cassandra/blob/ef5ac1a4/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 183176c..983d6b1 100644 --- a/src/java/org/apache/cassandra/db/ColumnFamilyStore.java +++ b/src/java/org/apache/cassandra/db/ColumnFamilyStore.java @@ -2415,7 +2415,7 @@ public class ColumnFamilyStore implements ColumnFamilyStoreMBean long allColumns = 0; int localTime = (int)(System.currentTimeMillis()/1000); - for (SSTableReader sstable : getSSTables(SSTableSet.LIVE)) + for (SSTableReader sstable : getSSTables(SSTableSet.CANONICAL)) { allDroppable += sstable.getDroppableTombstonesBefore(localTime - sstable.metadata.params.gcGraceSeconds); allColumns += sstable.getEstimatedColumnCount().mean() * sstable.getEstimatedColumnCount().count(); http://git-wip-us.apache.org/repos/asf/cassandra/blob/ef5ac1a4/src/java/org/apache/cassandra/io/sstable/metadata/MetadataCollector.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/io/sstable/metadata/MetadataCollector.java b/src/java/org/apache/cassandra/io/sstable/metadata/MetadataCollector.java index 3b13cf4..487a932 100644 --- a/src/java/org/apache/cassandra/io/sstable/metadata/MetadataCollector.java +++ b/src/java/org/apache/cassandra/io/sstable/metadata/MetadataCollector.java @@ -59,9 +59,9 @@ public class MetadataCollector implements PartitionStatisticsCollector return new EstimatedHistogram(150); } - static StreamingHistogram defaultTombstoneDropTimeHistogram() + static StreamingHistogram.StreamingHistogramBuilder defaultTombstoneDropTimeHistogramBuilder() { - return new StreamingHistogram(SSTable.TOMBSTONE_HISTOGRAM_BIN_SIZE, SSTable.TOMBSTONE_HISTOGRAM_SPOOL_SIZE, SSTable.TOMBSTONE_HISTOGRAM_TTL_ROUND_SECONDS); + return new StreamingHistogram.StreamingHistogramBuilder(SSTable.TOMBSTONE_HISTOGRAM_BIN_SIZE, SSTable.TOMBSTONE_HISTOGRAM_SPOOL_SIZE, SSTable.TOMBSTONE_HISTOGRAM_TTL_ROUND_SECONDS); } public static StatsMetadata defaultStatsMetadata() @@ -76,7 +76,7 @@ public class MetadataCollector implements PartitionStatisticsCollector 0, Integer.MAX_VALUE, NO_COMPRESSION_RATIO, - defaultTombstoneDropTimeHistogram(), + defaultTombstoneDropTimeHistogramBuilder().build(), 0, Collections.<ByteBuffer>emptyList(), Collections.<ByteBuffer>emptyList(), @@ -94,7 +94,7 @@ public class MetadataCollector implements PartitionStatisticsCollector protected final MinMaxIntTracker localDeletionTimeTracker = new MinMaxIntTracker(Cell.NO_DELETION_TIME, Cell.NO_DELETION_TIME); protected final MinMaxIntTracker ttlTracker = new MinMaxIntTracker(Cell.NO_TTL, Cell.NO_TTL); protected double compressionRatio = NO_COMPRESSION_RATIO; - protected StreamingHistogram estimatedTombstoneDropTime = defaultTombstoneDropTimeHistogram(); + protected StreamingHistogram.StreamingHistogramBuilder estimatedTombstoneDropTime = defaultTombstoneDropTimeHistogramBuilder(); protected int sstableLevel; protected ByteBuffer[] minClusteringValues; protected ByteBuffer[] maxClusteringValues; @@ -292,7 +292,7 @@ public class MetadataCollector implements PartitionStatisticsCollector ttlTracker.min(), ttlTracker.max(), compressionRatio, - estimatedTombstoneDropTime, + estimatedTombstoneDropTime.build(), sstableLevel, makeList(minClusteringValues), makeList(maxClusteringValues), http://git-wip-us.apache.org/repos/asf/cassandra/blob/ef5ac1a4/src/java/org/apache/cassandra/utils/StreamingHistogram.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/utils/StreamingHistogram.java b/src/java/org/apache/cassandra/utils/StreamingHistogram.java index f1752a9..6500a1a 100644 --- a/src/java/org/apache/cassandra/utils/StreamingHistogram.java +++ b/src/java/org/apache/cassandra/utils/StreamingHistogram.java @@ -40,145 +40,12 @@ public class StreamingHistogram // TreeMap to hold bins of histogram. private final TreeMap<Double, Long> bin; - - // Keep a second, larger buffer to spool data in, before finalizing it into `bin` - private final TreeMap<Double, Long> spool; - - // maximum bin size for this histogram private final int maxBinSize; - // maximum size of the spool - private final int maxSpoolSize; - - // voluntarily give up resolution for speed - private final int roundSeconds; - - /** - * Creates a new histogram with max bin size of maxBinSize - * @param maxBinSize maximum number of bins this histogram can have - */ - public StreamingHistogram(int maxBinSize, int maxSpoolSize, int roundSeconds) + private StreamingHistogram(int maxBinSize, Map<Double, Long> bin) { this.maxBinSize = maxBinSize; - this.maxSpoolSize = maxSpoolSize; - this.roundSeconds = roundSeconds; - bin = new TreeMap<>(); - spool = new TreeMap<>(); - } - - private StreamingHistogram(int maxBinSize, int maxSpoolSize, int roundSeconds, Map<Double, Long> bin) - { - this.maxBinSize = maxBinSize; - this.maxSpoolSize = maxSpoolSize; - this.roundSeconds = roundSeconds; this.bin = new TreeMap<>(bin); - this.spool = new TreeMap<>(); - } - - /** - * Adds new point p to this histogram. - * @param p - */ - public void update(double p) - { - update(p, 1); - } - - /** - * Adds new point p with value m to this histogram. - * @param p - * @param m - */ - public void update(double p, long m) - { - double d = p % this.roundSeconds; - if (d > 0) - p = p + (this.roundSeconds - d); - - Long mi = spool.get(p); - if (mi != null) - { - // we found the same p so increment that counter - spool.put(p, mi + m); - } - else - { - spool.put(p, m); - } - if(spool.size() > maxSpoolSize) - flushHistogram(); - } - - /** - * Drain the temporary spool into the final bins - */ - public void flushHistogram() - { - if(spool.size() > 0) - { - Long spoolValue; - Long binValue; - - // Iterate over the spool, copying the value into the primary bin map - // and compacting that map as necessary - for (Map.Entry<Double, Long> entry : spool.entrySet()) - { - Double key = entry.getKey(); - spoolValue = entry.getValue(); - binValue = bin.get(key); - - if (binValue != null) - { - binValue += spoolValue; - bin.put(key, binValue); - } else - { - bin.put(key, spoolValue); - } - - // if bin size exceeds maximum bin size then trim down to max size - if (bin.size() > maxBinSize) - { - // find points p1, p2 which have smallest difference - Iterator<Double> keys = bin.keySet().iterator(); - double p1 = keys.next(); - double p2 = keys.next(); - double smallestDiff = p2 - p1; - double q1 = p1, q2 = p2; - while (keys.hasNext()) { - p1 = p2; - p2 = keys.next(); - double diff = p2 - p1; - if (diff < smallestDiff) { - smallestDiff = diff; - q1 = p1; - q2 = p2; - } - } - // merge those two - long k1 = bin.remove(q1); - long k2 = bin.remove(q2); - bin.put((q1 * k1 + q2 * k2) / (k1 + k2), k1 + k2); - } - } - spool.clear(); - } - } - - /** - * Merges given histogram with this histogram. - * - * @param other histogram to merge - */ - public void merge(StreamingHistogram other) - { - if (other == null) - return; - - flushHistogram(); - - for (Map.Entry<Double, Long> entry : other.getAsMap().entrySet()) - update(entry.getKey(), entry.getValue()); } /** @@ -189,7 +56,6 @@ public class StreamingHistogram */ public double sum(double b) { - flushHistogram(); double sum = 0; // find the points pi, pnext which satisfy pi <= b < pnext Map.Entry<Double, Long> pnext = bin.higherEntry(b); @@ -219,10 +85,150 @@ public class StreamingHistogram public Map<Double, Long> getAsMap() { - flushHistogram(); return Collections.unmodifiableMap(bin); } + public static class StreamingHistogramBuilder + { + // TreeMap to hold bins of histogram. + private final TreeMap<Double, Long> bin; + + // Keep a second, larger buffer to spool data in, before finalizing it into `bin` + private final TreeMap<Double, Long> spool; + + // maximum bin size for this histogram + private final int maxBinSize; + + // maximum size of the spool + private final int maxSpoolSize; + + // voluntarily give up resolution for speed + private final int roundSeconds; + /** + * Creates a new histogram with max bin size of maxBinSize + * @param maxBinSize maximum number of bins this histogram can have + */ + public StreamingHistogramBuilder(int maxBinSize, int maxSpoolSize, int roundSeconds) + { + this.maxBinSize = maxBinSize; + this.maxSpoolSize = maxSpoolSize; + this.roundSeconds = roundSeconds; + bin = new TreeMap<>(); + spool = new TreeMap<>(); + } + + public StreamingHistogram build() + { + flushHistogram(); + return new StreamingHistogram(maxBinSize, bin); + } + /** + * Adds new point p to this histogram. + * @param p + */ + public void update(double p) + { + update(p, 1); + } + + /** + * Adds new point p with value m to this histogram. + * @param p + * @param m + */ + public void update(double p, long m) + { + double d = p % this.roundSeconds; + if (d > 0) + p = p + (this.roundSeconds - d); + + Long mi = spool.get(p); + if (mi != null) + { + // we found the same p so increment that counter + spool.put(p, mi + m); + } + else + { + spool.put(p, m); + } + if(spool.size() > maxSpoolSize) + flushHistogram(); + } + + /** + * Drain the temporary spool into the final bins + */ + public void flushHistogram() + { + if(spool.size() > 0) + { + Long spoolValue; + Long binValue; + + // Iterate over the spool, copying the value into the primary bin map + // and compacting that map as necessary + for (Map.Entry<Double, Long> entry : spool.entrySet()) + { + Double key = entry.getKey(); + spoolValue = entry.getValue(); + binValue = bin.get(key); + + if (binValue != null) + { + binValue += spoolValue; + bin.put(key, binValue); + } else + { + bin.put(key, spoolValue); + } + + // if bin size exceeds maximum bin size then trim down to max size + if (bin.size() > maxBinSize) + { + // find points p1, p2 which have smallest difference + Iterator<Double> keys = bin.keySet().iterator(); + double p1 = keys.next(); + double p2 = keys.next(); + double smallestDiff = p2 - p1; + double q1 = p1, q2 = p2; + while (keys.hasNext()) { + p1 = p2; + p2 = keys.next(); + double diff = p2 - p1; + if (diff < smallestDiff) { + smallestDiff = diff; + q1 = p1; + q2 = p2; + } + } + // merge those two + long k1 = bin.remove(q1); + long k2 = bin.remove(q2); + bin.put((q1 * k1 + q2 * k2) / (k1 + k2), k1 + k2); + } + } + spool.clear(); + } + } + + /** + * Merges given histogram with this histogram. + * + * @param other histogram to merge + */ + public void merge(StreamingHistogram other) + { + if (other == null) + return; + + flushHistogram(); + + for (Map.Entry<Double, Long> entry : other.getAsMap().entrySet()) + update(entry.getKey(), entry.getValue()); + } + } + public static class StreamingHistogramSerializer implements ISerializer<StreamingHistogram> { public void serialize(StreamingHistogram histogram, DataOutputPlus out) throws IOException @@ -247,7 +253,7 @@ public class StreamingHistogram tmp.put(in.readDouble(), in.readLong()); } - return new StreamingHistogram(maxBinSize, maxBinSize, 1, tmp); + return new StreamingHistogram(maxBinSize, tmp); } public long serializedSize(StreamingHistogram histogram) @@ -272,14 +278,12 @@ public class StreamingHistogram StreamingHistogram that = (StreamingHistogram) o; return maxBinSize == that.maxBinSize - && spool.equals(that.spool) && bin.equals(that.bin); } @Override public int hashCode() { - return Objects.hashCode(bin.hashCode(), spool.hashCode(), maxBinSize); + return Objects.hashCode(bin.hashCode(), maxBinSize); } - } http://git-wip-us.apache.org/repos/asf/cassandra/blob/ef5ac1a4/test/microbench/org/apache/cassandra/test/microbench/StreamingHistogramBench.java ---------------------------------------------------------------------- diff --git a/test/microbench/org/apache/cassandra/test/microbench/StreamingHistogramBench.java b/test/microbench/org/apache/cassandra/test/microbench/StreamingHistogramBench.java index 23e8f4e..c1ecf6d 100644 --- a/test/microbench/org/apache/cassandra/test/microbench/StreamingHistogramBench.java +++ b/test/microbench/org/apache/cassandra/test/microbench/StreamingHistogramBench.java @@ -36,39 +36,38 @@ import org.openjdk.jmh.annotations.*; @State(Scope.Benchmark) public class StreamingHistogramBench { - - StreamingHistogram streamingHistogram; - StreamingHistogram newStreamingHistogram; - StreamingHistogram newStreamingHistogram2; - StreamingHistogram newStreamingHistogram3; - StreamingHistogram newStreamingHistogram4; - StreamingHistogram newStreamingHistogram5; - StreamingHistogram newStreamingHistogram6; - StreamingHistogram streamingHistogram60; - StreamingHistogram newStreamingHistogram60; - StreamingHistogram newStreamingHistogram100x60; - - StreamingHistogram narrowstreamingHistogram; - StreamingHistogram narrownewStreamingHistogram; - StreamingHistogram narrownewStreamingHistogram2; - StreamingHistogram narrownewStreamingHistogram3; - StreamingHistogram narrownewStreamingHistogram4; - StreamingHistogram narrownewStreamingHistogram5; - StreamingHistogram narrownewStreamingHistogram6; - StreamingHistogram narrownewStreamingHistogram60; - StreamingHistogram narrowstreamingHistogram60; - StreamingHistogram narrownewStreamingHistogram100x60; - - StreamingHistogram sparsestreamingHistogram; - StreamingHistogram sparsenewStreamingHistogram; - StreamingHistogram sparsenewStreamingHistogram2; - StreamingHistogram sparsenewStreamingHistogram3; - StreamingHistogram sparsenewStreamingHistogram4; - StreamingHistogram sparsenewStreamingHistogram5; - StreamingHistogram sparsenewStreamingHistogram6; - StreamingHistogram sparsestreamingHistogram60; - StreamingHistogram sparsenewStreamingHistogram60; - StreamingHistogram sparsenewStreamingHistogram100x60; + StreamingHistogram.StreamingHistogramBuilder streamingHistogram; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram2; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram3; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram4; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram5; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram6; + StreamingHistogram.StreamingHistogramBuilder streamingHistogram60; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram60; + StreamingHistogram.StreamingHistogramBuilder newStreamingHistogram100x60; + + StreamingHistogram.StreamingHistogramBuilder narrowstreamingHistogram; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram2; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram3; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram4; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram5; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram6; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram60; + StreamingHistogram.StreamingHistogramBuilder narrowstreamingHistogram60; + StreamingHistogram.StreamingHistogramBuilder narrownewStreamingHistogram100x60; + + StreamingHistogram.StreamingHistogramBuilder sparsestreamingHistogram; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram2; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram3; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram4; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram5; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram6; + StreamingHistogram.StreamingHistogramBuilder sparsestreamingHistogram60; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram60; + StreamingHistogram.StreamingHistogramBuilder sparsenewStreamingHistogram100x60; static int[] ttls = new int[10000000]; static int[] narrowttls = new int[10000000]; @@ -91,39 +90,39 @@ public class StreamingHistogramBench public void setup() throws Throwable { - streamingHistogram = new StreamingHistogram(100, 0, 1); - newStreamingHistogram = new StreamingHistogram(100, 1000, 1); - newStreamingHistogram2 = new StreamingHistogram(100, 10000, 1); - newStreamingHistogram3 = new StreamingHistogram(100, 100000, 1); - newStreamingHistogram4 = new StreamingHistogram(50, 100000, 1); - newStreamingHistogram5 = new StreamingHistogram(50, 10000,1 ); - newStreamingHistogram6 = new StreamingHistogram(100, 1000000, 1); - streamingHistogram60 = new StreamingHistogram(100, 0, 60); - newStreamingHistogram60 = new StreamingHistogram(100, 100000, 60); - newStreamingHistogram100x60 = new StreamingHistogram(100, 10000, 60); - - narrowstreamingHistogram = new StreamingHistogram(100, 0, 1); - narrownewStreamingHistogram = new StreamingHistogram(100, 1000, 1); - narrownewStreamingHistogram2 = new StreamingHistogram(100, 10000, 1); - narrownewStreamingHistogram3 = new StreamingHistogram(100, 100000, 1); - narrownewStreamingHistogram4 = new StreamingHistogram(50, 100000, 1); - narrownewStreamingHistogram5 = new StreamingHistogram(50, 10000, 1); - narrownewStreamingHistogram6 = new StreamingHistogram(100, 1000000, 1); - narrowstreamingHistogram60 = new StreamingHistogram(100, 0, 60); - narrownewStreamingHistogram60 = new StreamingHistogram(100, 100000, 60); - narrownewStreamingHistogram100x60 = new StreamingHistogram(100, 10000, 60); - - - sparsestreamingHistogram = new StreamingHistogram(100, 0, 1); - sparsenewStreamingHistogram = new StreamingHistogram(100, 1000, 1); - sparsenewStreamingHistogram2 = new StreamingHistogram(100, 10000, 1); - sparsenewStreamingHistogram3 = new StreamingHistogram(100, 100000, 1); - sparsenewStreamingHistogram4 = new StreamingHistogram(50, 100000, 1); - sparsenewStreamingHistogram5 = new StreamingHistogram(50, 10000, 1); - sparsenewStreamingHistogram6 = new StreamingHistogram(100, 1000000, 1); - sparsestreamingHistogram60 = new StreamingHistogram(100, 0, 60); - sparsenewStreamingHistogram60 = new StreamingHistogram(100, 100000, 60); - sparsenewStreamingHistogram100x60 = new StreamingHistogram(100, 10000, 60); + streamingHistogram = new StreamingHistogram.StreamingHistogramBuilder(100, 0, 1); + newStreamingHistogram = new StreamingHistogram.StreamingHistogramBuilder(100, 1000, 1); + newStreamingHistogram2 = new StreamingHistogram.StreamingHistogramBuilder(100, 10000, 1); + newStreamingHistogram3 = new StreamingHistogram.StreamingHistogramBuilder(100, 100000, 1); + newStreamingHistogram4 = new StreamingHistogram.StreamingHistogramBuilder(50, 100000, 1); + newStreamingHistogram5 = new StreamingHistogram.StreamingHistogramBuilder(50, 10000,1 ); + newStreamingHistogram6 = new StreamingHistogram.StreamingHistogramBuilder(100, 1000000, 1); + streamingHistogram60 = new StreamingHistogram.StreamingHistogramBuilder(100, 0, 60); + newStreamingHistogram60 = new StreamingHistogram.StreamingHistogramBuilder(100, 100000, 60); + newStreamingHistogram100x60 = new StreamingHistogram.StreamingHistogramBuilder(100, 10000, 60); + + narrowstreamingHistogram = new StreamingHistogram.StreamingHistogramBuilder(100, 0, 1); + narrownewStreamingHistogram = new StreamingHistogram.StreamingHistogramBuilder(100, 1000, 1); + narrownewStreamingHistogram2 = new StreamingHistogram.StreamingHistogramBuilder(100, 10000, 1); + narrownewStreamingHistogram3 = new StreamingHistogram.StreamingHistogramBuilder(100, 100000, 1); + narrownewStreamingHistogram4 = new StreamingHistogram.StreamingHistogramBuilder(50, 100000, 1); + narrownewStreamingHistogram5 = new StreamingHistogram.StreamingHistogramBuilder(50, 10000, 1); + narrownewStreamingHistogram6 = new StreamingHistogram.StreamingHistogramBuilder(100, 1000000, 1); + narrowstreamingHistogram60 = new StreamingHistogram.StreamingHistogramBuilder(100, 0, 60); + narrownewStreamingHistogram60 = new StreamingHistogram.StreamingHistogramBuilder(100, 100000, 60); + narrownewStreamingHistogram100x60 = new StreamingHistogram.StreamingHistogramBuilder(100, 10000, 60); + + + sparsestreamingHistogram = new StreamingHistogram.StreamingHistogramBuilder(100, 0, 1); + sparsenewStreamingHistogram = new StreamingHistogram.StreamingHistogramBuilder(100, 1000, 1); + sparsenewStreamingHistogram2 = new StreamingHistogram.StreamingHistogramBuilder(100, 10000, 1); + sparsenewStreamingHistogram3 = new StreamingHistogram.StreamingHistogramBuilder(100, 100000, 1); + sparsenewStreamingHistogram4 = new StreamingHistogram.StreamingHistogramBuilder(50, 100000, 1); + sparsenewStreamingHistogram5 = new StreamingHistogram.StreamingHistogramBuilder(50, 10000, 1); + sparsenewStreamingHistogram6 = new StreamingHistogram.StreamingHistogramBuilder(100, 1000000, 1); + sparsestreamingHistogram60 = new StreamingHistogram.StreamingHistogramBuilder(100, 0, 60); + sparsenewStreamingHistogram60 = new StreamingHistogram.StreamingHistogramBuilder(100, 100000, 60); + sparsenewStreamingHistogram100x60 = new StreamingHistogram.StreamingHistogramBuilder(100, 10000, 60); } @@ -138,7 +137,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) streamingHistogram.update(ttls[i]); - streamingHistogram.flushHistogram(); + streamingHistogram.build(); } @Benchmark @@ -146,7 +145,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram.update(ttls[i]); - newStreamingHistogram.flushHistogram(); + newStreamingHistogram.build(); } @@ -155,7 +154,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram2.update(ttls[i]); - newStreamingHistogram2.flushHistogram(); + newStreamingHistogram2.build(); } @@ -164,7 +163,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram3.update(ttls[i]); - newStreamingHistogram3.flushHistogram(); + newStreamingHistogram3.build(); } @@ -173,7 +172,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram6.update(ttls[i]); - newStreamingHistogram6.flushHistogram(); + newStreamingHistogram6.build(); } @@ -183,7 +182,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram4.update(ttls[i]); - newStreamingHistogram4.flushHistogram(); + newStreamingHistogram4.build(); } @@ -192,7 +191,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram5.update(ttls[i]); - newStreamingHistogram5.flushHistogram(); + newStreamingHistogram5.build(); } @@ -201,7 +200,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) streamingHistogram60.update(sparsettls[i]); - streamingHistogram60.flushHistogram(); + streamingHistogram60.build(); } @@ -210,7 +209,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram60.update(sparsettls[i]); - newStreamingHistogram60.flushHistogram(); + newStreamingHistogram60.build(); } @Benchmark @@ -218,7 +217,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) newStreamingHistogram100x60.update(sparsettls[i]); - newStreamingHistogram100x60.flushHistogram(); + newStreamingHistogram100x60.build(); } @@ -227,7 +226,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrowstreamingHistogram.update(narrowttls[i]); - narrowstreamingHistogram.flushHistogram(); + narrowstreamingHistogram.build(); } @Benchmark @@ -235,7 +234,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram.update(narrowttls[i]); - narrownewStreamingHistogram.flushHistogram(); + narrownewStreamingHistogram.build(); } @@ -244,7 +243,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram2.update(narrowttls[i]); - narrownewStreamingHistogram2.flushHistogram(); + narrownewStreamingHistogram2.build(); } @@ -253,7 +252,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram3.update(narrowttls[i]); - narrownewStreamingHistogram3.flushHistogram(); + narrownewStreamingHistogram3.build(); } @@ -262,7 +261,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram6.update(ttls[i]); - narrownewStreamingHistogram6.flushHistogram(); + narrownewStreamingHistogram6.build(); } @@ -272,7 +271,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram4.update(narrowttls[i]); - narrownewStreamingHistogram4.flushHistogram(); + narrownewStreamingHistogram4.build(); } @@ -281,7 +280,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram5.update(narrowttls[i]); - narrownewStreamingHistogram5.flushHistogram(); + narrownewStreamingHistogram5.build(); } @@ -290,7 +289,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrowstreamingHistogram60.update(sparsettls[i]); - narrowstreamingHistogram60.flushHistogram(); + narrowstreamingHistogram60.build(); } @@ -299,7 +298,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram60.update(sparsettls[i]); - narrownewStreamingHistogram60.flushHistogram(); + narrownewStreamingHistogram60.build(); } @@ -308,7 +307,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) narrownewStreamingHistogram100x60.update(sparsettls[i]); - narrownewStreamingHistogram100x60.flushHistogram(); + narrownewStreamingHistogram100x60.build(); } @@ -318,7 +317,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsestreamingHistogram.update(sparsettls[i]); - sparsestreamingHistogram.flushHistogram(); + sparsestreamingHistogram.build(); } @Benchmark @@ -326,7 +325,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram.update(sparsettls[i]); - sparsenewStreamingHistogram.flushHistogram(); + sparsenewStreamingHistogram.build(); } @@ -335,7 +334,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram2.update(sparsettls[i]); - sparsenewStreamingHistogram2.flushHistogram(); + sparsenewStreamingHistogram2.build(); } @@ -344,7 +343,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram3.update(sparsettls[i]); - sparsenewStreamingHistogram3.flushHistogram(); + sparsenewStreamingHistogram3.build(); } @@ -353,7 +352,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram6.update(ttls[i]); - sparsenewStreamingHistogram6.flushHistogram(); + sparsenewStreamingHistogram6.build(); } @@ -362,7 +361,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram4.update(sparsettls[i]); - sparsenewStreamingHistogram4.flushHistogram(); + sparsenewStreamingHistogram4.build(); } @@ -371,7 +370,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram5.update(sparsettls[i]); - sparsenewStreamingHistogram5.flushHistogram(); + sparsenewStreamingHistogram5.build(); } @@ -380,7 +379,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsestreamingHistogram60.update(sparsettls[i]); - sparsestreamingHistogram60.flushHistogram(); + sparsestreamingHistogram60.build(); } @@ -389,7 +388,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram60.update(sparsettls[i]); - sparsenewStreamingHistogram60.flushHistogram(); + sparsenewStreamingHistogram60.build(); } @@ -398,8 +397,7 @@ public class StreamingHistogramBench { for(int i = 0 ; i < ttls.length; i++) sparsenewStreamingHistogram100x60.update(sparsettls[i]); - sparsenewStreamingHistogram100x60.flushHistogram(); + sparsenewStreamingHistogram100x60.build(); } - } http://git-wip-us.apache.org/repos/asf/cassandra/blob/ef5ac1a4/test/unit/org/apache/cassandra/utils/StreamingHistogramTest.java ---------------------------------------------------------------------- diff --git a/test/unit/org/apache/cassandra/utils/StreamingHistogramTest.java b/test/unit/org/apache/cassandra/utils/StreamingHistogramTest.java index 21c736e..b107600 100644 --- a/test/unit/org/apache/cassandra/utils/StreamingHistogramTest.java +++ b/test/unit/org/apache/cassandra/utils/StreamingHistogramTest.java @@ -32,7 +32,7 @@ public class StreamingHistogramTest @Test public void testFunction() throws Exception { - StreamingHistogram hist = new StreamingHistogram(5, 5, 1); + StreamingHistogram.StreamingHistogramBuilder hist = new StreamingHistogram.StreamingHistogramBuilder(5, 5, 1); long[] samples = new long[]{23, 19, 10, 16, 36, 2, 9, 32, 30, 45}; // add 7 points to histogram of 5 bins @@ -50,7 +50,7 @@ public class StreamingHistogramTest expected1.put(36.0, 1L); Iterator<Map.Entry<Double, Long>> expectedItr = expected1.entrySet().iterator(); - for (Map.Entry<Double, Long> actual : hist.getAsMap().entrySet()) + for (Map.Entry<Double, Long> actual : hist.build().getAsMap().entrySet()) { Map.Entry<Double, Long> entry = expectedItr.next(); assertEquals(entry.getKey(), actual.getKey(), 0.01); @@ -58,12 +58,13 @@ public class StreamingHistogramTest } // merge test - StreamingHistogram hist2 = new StreamingHistogram(3, 0, 1); + StreamingHistogram.StreamingHistogramBuilder hist2 = new StreamingHistogram.StreamingHistogramBuilder(3, 0, 1); for (int i = 7; i < samples.length; i++) { hist2.update(samples[i]); } - hist.merge(hist2); + hist.merge(hist2.build()); + StreamingHistogram histBuilt = hist.build(); // should end up (2,1),(9.5,2),(19.33,3),(32.67,3),(45,1) Map<Double, Long> expected2 = new LinkedHashMap<Double, Long>(5); expected2.put(2.0, 1L); @@ -72,7 +73,7 @@ public class StreamingHistogramTest expected2.put(32.67, 3L); expected2.put(45.0, 1L); expectedItr = expected2.entrySet().iterator(); - for (Map.Entry<Double, Long> actual : hist.getAsMap().entrySet()) + for (Map.Entry<Double, Long> actual : histBuilt.getAsMap().entrySet()) { Map.Entry<Double, Long> entry = expectedItr.next(); assertEquals(entry.getKey(), actual.getKey(), 0.01); @@ -80,22 +81,23 @@ public class StreamingHistogramTest } // sum test - assertEquals(3.28, hist.sum(15), 0.01); + assertEquals(3.28, histBuilt.sum(15), 0.01); // sum test (b > max(hist)) - assertEquals(10.0, hist.sum(50), 0.01); + assertEquals(10.0, histBuilt.sum(50), 0.01); } @Test public void testSerDe() throws Exception { - StreamingHistogram hist = new StreamingHistogram(5, 0, 1); + StreamingHistogram.StreamingHistogramBuilder histogramBuilder = new StreamingHistogram.StreamingHistogramBuilder(5, 0, 1); long[] samples = new long[]{23, 19, 10, 16, 36, 2, 9}; // add 7 points to histogram of 5 bins for (int i = 0; i < samples.length; i++) { - hist.update(samples[i]); + histogramBuilder.update(samples[i]); } + StreamingHistogram hist = histogramBuilder.build(); DataOutputBuffer out = new DataOutputBuffer(); StreamingHistogram.serializer.serialize(hist, out); @@ -123,7 +125,7 @@ public class StreamingHistogramTest @Test public void testOverflow() throws Exception { - StreamingHistogram hist = new StreamingHistogram(5, 10, 1); + StreamingHistogram.StreamingHistogramBuilder histogramBuilder = new StreamingHistogram.StreamingHistogramBuilder(5, 10, 1); long[] samples = new long[]{23, 19, 10, 16, 36, 2, 9, 32, 30, 45, 31, 32, 32, 33, 34, 35, 70, 78, 80, 90, 100, 32, 32, 33, 34, 35, 70, 78, 80, 90, 100 @@ -132,20 +134,21 @@ public class StreamingHistogramTest // Hit the spool cap, force it to make bins for (int i = 0; i < samples.length; i++) { - hist.update(samples[i]); + histogramBuilder.update(samples[i]); } - assertEquals(5, hist.getAsMap().keySet().size()); + assertEquals(5, histogramBuilder.build().getAsMap().keySet().size()); } @Test public void testRounding() throws Exception { - StreamingHistogram hist = new StreamingHistogram(5, 10, 60); + StreamingHistogram.StreamingHistogramBuilder histogramBuilder = new StreamingHistogram.StreamingHistogramBuilder(5, 10, 60); long[] samples = new long[] { 59, 60, 119, 180, 181, 300 }; // 60, 60, 120, 180, 240, 300 for (int i = 0 ; i < samples.length ; i++) - hist.update(samples[i]); + histogramBuilder.update(samples[i]); + StreamingHistogram hist = histogramBuilder.build(); assertEquals(hist.getAsMap().keySet().size(), (int) 5); assertEquals((long) hist.getAsMap().get(Double.valueOf(60)), (long) 2L); assertEquals((long) hist.getAsMap().get(Double.valueOf(120)), (long) 1L); --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@cassandra.apache.org For additional commands, e-mail: commits-h...@cassandra.apache.org