001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.metrics.impl;
019
020import java.util.Arrays;
021import java.util.concurrent.atomic.AtomicLong;
022import java.util.concurrent.atomic.LongAdder;
023import java.util.stream.Stream;
024import org.apache.hadoop.hbase.metrics.Snapshot;
025import org.apache.hadoop.hbase.util.AtomicUtils;
026import org.apache.yetus.audience.InterfaceAudience;
027import org.apache.yetus.audience.InterfaceStability;
028
029/**
030 * FastLongHistogram is a thread-safe class that estimate distribution of data and computes the
031 * quantiles.
032 */
033@InterfaceAudience.Private
034@InterfaceStability.Evolving
035public class FastLongHistogram {
036
037  /**
038   * Default number of bins.
039   */
040  public static final int DEFAULT_NBINS = 255;
041
042  public static final double[] DEFAULT_QUANTILES =
043    new double[] { 0.25, 0.5, 0.75, 0.90, 0.95, 0.98, 0.99, 0.999 };
044
045  /**
046   * Bins is a class containing a list of buckets(or bins) for estimation histogram of some data.
047   */
048  private static class Bins {
049
050    private final LongAdder[] counts;
051    // inclusive
052    private final long binsMin;
053    // exclusive
054    private final long binsMax;
055    private final long bins10XMax;
056    private final AtomicLong min = new AtomicLong(Long.MAX_VALUE);
057    private final AtomicLong max = new AtomicLong(0L);
058
059    private final LongAdder count = new LongAdder();
060    private final LongAdder total = new LongAdder();
061
062    // set to true when any of data has been inserted to the Bins. It is set after the counts are
063    // updated.
064    private volatile boolean hasData = false;
065
066    /**
067     * The constructor for creating a Bins without any prior data.
068     */
069    public Bins(int numBins) {
070      counts = createCounters(numBins);
071      this.binsMin = 1L;
072
073      // These two numbers are total guesses
074      // and should be treated as highly suspect.
075      this.binsMax = 1000;
076      this.bins10XMax = binsMax * 10;
077    }
078
079    /**
080     * The constructor for creating a Bins with last Bins.
081     */
082    public Bins(Bins last, int numBins, double minQ, double maxQ) {
083      long[] values = last.getQuantiles(new double[] { minQ, maxQ });
084      long wd = values[1] - values[0] + 1;
085      // expand minQ and maxQ in two ends back assuming uniform distribution
086      this.binsMin = Math.max(0L, (long) (values[0] - wd * minQ));
087      long binsMax = (long) (values[1] + wd * (1 - maxQ)) + 1;
088      // make sure each of bins is at least of width 1
089      this.binsMax = Math.max(binsMax, this.binsMin + numBins);
090      this.bins10XMax = Math.max((long) (values[1] + (binsMax - 1) * 9), this.binsMax + 1);
091
092      this.counts = createCounters(numBins);
093    }
094
095    private LongAdder[] createCounters(int numBins) {
096      return Stream.generate(LongAdder::new).limit(numBins + 3).toArray(LongAdder[]::new);
097    }
098
099    private int getIndex(long value) {
100      if (value < this.binsMin) {
101        return 0;
102      } else if (value > this.bins10XMax) {
103        return this.counts.length - 1;
104      } else if (value >= this.binsMax) {
105        return this.counts.length - 2;
106      }
107      // compute the position
108      return 1
109        + (int) ((value - this.binsMin) * (this.counts.length - 3) / (this.binsMax - this.binsMin));
110
111    }
112
113    /**
114     * Adds a value to the histogram.
115     */
116    public void add(long value, long count) {
117      if (value < 0) {
118        // The whole computation is completely thrown off if there are negative numbers
119        //
120        // Normally we would throw an IllegalArgumentException however this is the metrics
121        // system and it should be completely safe at all times.
122        // So silently throw it away.
123        return;
124      }
125      AtomicUtils.updateMin(min, value);
126      AtomicUtils.updateMax(max, value);
127
128      this.count.add(count);
129      this.total.add(value * count);
130
131      int pos = getIndex(value);
132      this.counts[pos].add(count);
133
134      // hasData needs to be updated as last
135      this.hasData = true;
136    }
137
138    /**
139     * Computes the quantiles give the ratios.
140     */
141    @edu.umd.cs.findbugs.annotations.SuppressWarnings(value = "FL_FLOATS_AS_LOOP_COUNTERS",
142        justification = "valid usage")
143    public long[] getQuantiles(double[] quantiles) {
144      if (!hasData) {
145        // No data yet.
146        return new long[quantiles.length];
147      }
148
149      // Make a snapshot of lowerCounter, higherCounter and bins.counts to counts.
150      // This is not synchronized, but since the counter are accumulating, the result is a good
151      // estimation of a snapshot.
152      long[] counts = new long[this.counts.length];
153      long total = 0L;
154      for (int i = 0; i < this.counts.length; i++) {
155        counts[i] = this.counts[i].sum();
156        total += counts[i];
157      }
158
159      int rIndex = 0;
160      double qCount = total * quantiles[0];
161      long cum = 0L;
162
163      long[] res = new long[quantiles.length];
164      countsLoop: for (int i = 0; i < counts.length; i++) {
165        // mn and mx define a value range
166        long mn, mx;
167        if (i == 0) {
168          mn = this.min.get();
169          mx = this.binsMin;
170        } else if (i == counts.length - 1) {
171          mn = this.bins10XMax;
172          mx = this.max.get();
173        } else if (i == counts.length - 2) {
174          mn = this.binsMax;
175          mx = this.bins10XMax;
176        } else {
177          mn = this.binsMin + (i - 1) * (this.binsMax - this.binsMin) / (this.counts.length - 3);
178          mx = this.binsMin + i * (this.binsMax - this.binsMin) / (this.counts.length - 3);
179        }
180
181        if (mx < this.min.get()) {
182          continue;
183        }
184        if (mn > this.max.get()) {
185          break;
186        }
187        mn = Math.max(mn, this.min.get());
188        mx = Math.min(mx, this.max.get());
189
190        // lastCum/cum are the corresponding counts to mn/mx
191        double lastCum = cum;
192        cum += counts[i];
193
194        // fill the results for qCount is within current range.
195        while (qCount <= cum) {
196          if (cum == lastCum) {
197            res[rIndex] = mn;
198          } else {
199            res[rIndex] = (long) ((qCount - lastCum) * (mx - mn) / (cum - lastCum) + mn);
200          }
201
202          // move to next quantile
203          rIndex++;
204          if (rIndex >= quantiles.length) {
205            break countsLoop;
206          }
207          qCount = total * quantiles[rIndex];
208        }
209      }
210      // In case quantiles contains values >= 100%
211      for (; rIndex < quantiles.length; rIndex++) {
212        res[rIndex] = this.max.get();
213      }
214
215      return res;
216    }
217
218    long getNumAtOrBelow(long val) {
219      return Arrays.stream(counts).mapToLong(c -> c.sum()).limit(getIndex(val) + 1).sum();
220    }
221
222    public long getMin() {
223      long min = this.min.get();
224      return min == Long.MAX_VALUE ? 0 : min; // in case it is not initialized
225    }
226
227    public long getMean() {
228      long count = this.count.sum();
229      long total = this.total.sum();
230      if (count == 0) {
231        return 0;
232      }
233      return total / count;
234    }
235  }
236
237  // The bins counting values. It is replaced with a new one in calling of reset().
238  private volatile Bins bins;
239
240  /**
241   * Constructor.
242   */
243  public FastLongHistogram() {
244    this(DEFAULT_NBINS);
245  }
246
247  /**
248   * Constructor.
249   * @param numOfBins the number of bins for the histogram. A larger value results in more precise
250   *                  results but with lower efficiency, and vice versus.
251   */
252  public FastLongHistogram(int numOfBins) {
253    this.bins = new Bins(numOfBins);
254  }
255
256  /**
257   * Constructor setting the bins assuming a uniform distribution within a range.
258   * @param numOfBins the number of bins for the histogram. A larger value results in more precise
259   *                  results but with lower efficiency, and vice versus.
260   * @param min       lower bound of the region, inclusive.
261   * @param max       higher bound of the region, inclusive.
262   */
263  public FastLongHistogram(int numOfBins, long min, long max) {
264    this(numOfBins);
265    Bins bins = new Bins(numOfBins);
266    bins.add(min, 1);
267    bins.add(max, 1);
268    this.bins = new Bins(bins, numOfBins, 0.01, 0.999);
269  }
270
271  /**
272   * Adds a value to the histogram.
273   */
274  public void add(long value, long count) {
275    this.bins.add(value, count);
276  }
277
278  /**
279   * Computes the quantiles give the ratios.
280   */
281  public long[] getQuantiles(double[] quantiles) {
282    return this.bins.getQuantiles(quantiles);
283  }
284
285  public long[] getQuantiles() {
286    return this.bins.getQuantiles(DEFAULT_QUANTILES);
287  }
288
289  public long getMin() {
290    return this.bins.getMin();
291  }
292
293  public long getMax() {
294    return this.bins.max.get();
295  }
296
297  public long getCount() {
298    return this.bins.count.sum();
299  }
300
301  public long getMean() {
302    return this.bins.getMean();
303  }
304
305  public long getNumAtOrBelow(long value) {
306    return this.bins.getNumAtOrBelow(value);
307  }
308
309  /**
310   * Resets the histogram for new counting.
311   */
312  public Snapshot snapshotAndReset() {
313    final Bins oldBins = this.bins;
314    this.bins = new Bins(this.bins, this.bins.counts.length - 3, 0.01, 0.99);
315    final long[] percentiles = oldBins.getQuantiles(DEFAULT_QUANTILES);
316    final long count = oldBins.count.sum();
317
318    return new Snapshot() {
319      @Override
320      public long[] getQuantiles(double[] quantiles) {
321        return oldBins.getQuantiles(quantiles);
322      }
323
324      @Override
325      public long[] getQuantiles() {
326        return percentiles;
327      }
328
329      @Override
330      public long getCount() {
331        return count;
332      }
333
334      @Override
335      public long getCountAtOrBelow(long val) {
336        return oldBins.getNumAtOrBelow(val);
337      }
338
339      @Override
340      public long get25thPercentile() {
341        return percentiles[0];
342      }
343
344      @Override
345      public long get75thPercentile() {
346        return percentiles[2];
347      }
348
349      @Override
350      public long get90thPercentile() {
351        return percentiles[3];
352      }
353
354      @Override
355      public long get95thPercentile() {
356        return percentiles[4];
357      }
358
359      @Override
360      public long get98thPercentile() {
361        return percentiles[5];
362      }
363
364      @Override
365      public long get99thPercentile() {
366        return percentiles[6];
367      }
368
369      @Override
370      public long get999thPercentile() {
371        return percentiles[7];
372      }
373
374      @Override
375      public long getMedian() {
376        return percentiles[1];
377      }
378
379      @Override
380      public long getMax() {
381        return oldBins.max.get();
382      }
383
384      @Override
385      public long getMean() {
386        return oldBins.getMean();
387      }
388
389      @Override
390      public long getMin() {
391        return oldBins.getMin();
392      }
393    };
394  }
395}