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.io.hfile;
019
020import java.util.Arrays;
021import java.util.concurrent.atomic.AtomicLong;
022import java.util.concurrent.atomic.LongAdder;
023import org.apache.hadoop.hbase.metrics.impl.FastLongHistogram;
024import org.apache.yetus.audience.InterfaceAudience;
025
026/**
027 * Class that implements cache metrics.
028 */
029@InterfaceAudience.Private
030public class CacheStats {
031
032  /**
033   * Sliding window statistics. The number of metric periods to include in sliding window hit ratio
034   * calculations.
035   */
036  static final int DEFAULT_WINDOW_PERIODS = 5;
037
038  /** The number of getBlock requests that were cache hits */
039  private final LongAdder hitCount = new LongAdder();
040
041  /** The number of getBlock requests that were cache hits from primary replica */
042  private final LongAdder primaryHitCount = new LongAdder();
043
044  /**
045   * The number of getBlock requests that were cache hits, but only from requests that were set to
046   * use the block cache. This is because all reads attempt to read from the block cache even if
047   * they will not put new blocks into the block cache. See HBASE-2253 for more information.
048   */
049  private final LongAdder hitCachingCount = new LongAdder();
050
051  /** The number of getBlock requests that were cache misses */
052  private final LongAdder missCount = new LongAdder();
053
054  /** The number of getBlock requests for primary replica that were cache misses */
055  private final LongAdder primaryMissCount = new LongAdder();
056  /**
057   * The number of getBlock requests that were cache misses, but only from requests that were set to
058   * use the block cache.
059   */
060  private final LongAdder missCachingCount = new LongAdder();
061
062  /** The number of times an eviction has occurred */
063  private final LongAdder evictionCount = new LongAdder();
064
065  /** The total number of blocks that have been evicted */
066  private final LongAdder evictedBlockCount = new LongAdder();
067
068  /** The total number of blocks for primary replica that have been evicted */
069  private final LongAdder primaryEvictedBlockCount = new LongAdder();
070
071  /** The total number of blocks that were not inserted. */
072  private final AtomicLong failedInserts = new AtomicLong(0);
073
074  /** Per Block Type Counts */
075  private final LongAdder dataMissCount = new LongAdder();
076  private final LongAdder leafIndexMissCount = new LongAdder();
077  private final LongAdder bloomChunkMissCount = new LongAdder();
078  private final LongAdder metaMissCount = new LongAdder();
079  private final LongAdder rootIndexMissCount = new LongAdder();
080  private final LongAdder intermediateIndexMissCount = new LongAdder();
081  private final LongAdder fileInfoMissCount = new LongAdder();
082  private final LongAdder generalBloomMetaMissCount = new LongAdder();
083  private final LongAdder deleteFamilyBloomMissCount = new LongAdder();
084  private final LongAdder trailerMissCount = new LongAdder();
085
086  private final LongAdder dataHitCount = new LongAdder();
087  private final LongAdder leafIndexHitCount = new LongAdder();
088  private final LongAdder bloomChunkHitCount = new LongAdder();
089  private final LongAdder metaHitCount = new LongAdder();
090  private final LongAdder rootIndexHitCount = new LongAdder();
091  private final LongAdder intermediateIndexHitCount = new LongAdder();
092  private final LongAdder fileInfoHitCount = new LongAdder();
093  private final LongAdder generalBloomMetaHitCount = new LongAdder();
094  private final LongAdder deleteFamilyBloomHitCount = new LongAdder();
095  private final LongAdder trailerHitCount = new LongAdder();
096
097  /** The number of metrics periods to include in window */
098  private final int numPeriodsInWindow;
099  /** Hit counts for each period in window */
100  private final long[] hitCounts;
101  /** Caching hit counts for each period in window */
102  private final long[] hitCachingCounts;
103  /** Access counts for each period in window */
104  private final long[] requestCounts;
105  /** Caching access counts for each period in window */
106  private final long[] requestCachingCounts;
107  /** Last hit count read */
108  private long lastHitCount = 0;
109  /** Last hit caching count read */
110  private long lastHitCachingCount = 0;
111  /** Last request count read */
112  private long lastRequestCount = 0;
113  /** Last request caching count read */
114  private long lastRequestCachingCount = 0;
115  /** Current window index (next to be updated) */
116  private int windowIndex = 0;
117  /**
118   * Keep running age at eviction time
119   */
120  private FastLongHistogram ageAtEviction;
121  private long startTime = System.nanoTime();
122
123  public CacheStats(final String name) {
124    this(name, DEFAULT_WINDOW_PERIODS);
125  }
126
127  public CacheStats(final String name, int numPeriodsInWindow) {
128    this.numPeriodsInWindow = numPeriodsInWindow;
129    this.hitCounts = new long[numPeriodsInWindow];
130    this.hitCachingCounts = new long[numPeriodsInWindow];
131    this.requestCounts = new long[numPeriodsInWindow];
132    this.requestCachingCounts = new long[numPeriodsInWindow];
133    this.ageAtEviction = new FastLongHistogram();
134  }
135
136  @Override
137  public String toString() {
138    AgeSnapshot snapshot = getAgeAtEvictionSnapshot();
139    return "hitCount=" + getHitCount() + ", hitCachingCount=" + getHitCachingCount()
140      + ", missCount=" + getMissCount() + ", missCachingCount=" + getMissCachingCount()
141      + ", evictionCount=" + getEvictionCount() + ", evictedBlockCount=" + getEvictedCount()
142      + ", primaryMissCount=" + getPrimaryMissCount() + ", primaryHitCount=" + getPrimaryHitCount()
143      + ", evictedAgeMean=" + snapshot.getMean();
144  }
145
146  public void miss(boolean caching, boolean primary, BlockType type) {
147    missCount.increment();
148    if (primary) primaryMissCount.increment();
149    if (caching) missCachingCount.increment();
150    if (type == null) {
151      return;
152    }
153    switch (type) {
154      case DATA:
155      case ENCODED_DATA:
156        dataMissCount.increment();
157        break;
158      case LEAF_INDEX:
159        leafIndexMissCount.increment();
160        break;
161      case BLOOM_CHUNK:
162        bloomChunkMissCount.increment();
163        break;
164      case META:
165        metaMissCount.increment();
166        break;
167      case INTERMEDIATE_INDEX:
168        intermediateIndexMissCount.increment();
169        break;
170      case ROOT_INDEX:
171        rootIndexMissCount.increment();
172        break;
173      case FILE_INFO:
174        fileInfoMissCount.increment();
175        break;
176      case GENERAL_BLOOM_META:
177        generalBloomMetaMissCount.increment();
178        break;
179      case DELETE_FAMILY_BLOOM_META:
180        deleteFamilyBloomMissCount.increment();
181        break;
182      case TRAILER:
183        trailerMissCount.increment();
184        break;
185      default:
186        // If there's a new type that's fine
187        // Ignore it for now. This is metrics don't exception.
188        break;
189    }
190  }
191
192  public void hit(boolean caching, boolean primary, BlockType type) {
193    hitCount.increment();
194    if (primary) primaryHitCount.increment();
195    if (caching) hitCachingCount.increment();
196
197    if (type == null) {
198      return;
199    }
200    switch (type) {
201      case DATA:
202      case ENCODED_DATA:
203        dataHitCount.increment();
204        break;
205      case LEAF_INDEX:
206        leafIndexHitCount.increment();
207        break;
208      case BLOOM_CHUNK:
209        bloomChunkHitCount.increment();
210        break;
211      case META:
212        metaHitCount.increment();
213        break;
214      case INTERMEDIATE_INDEX:
215        intermediateIndexHitCount.increment();
216        break;
217      case ROOT_INDEX:
218        rootIndexHitCount.increment();
219        break;
220      case FILE_INFO:
221        fileInfoHitCount.increment();
222        break;
223      case GENERAL_BLOOM_META:
224        generalBloomMetaHitCount.increment();
225        break;
226      case DELETE_FAMILY_BLOOM_META:
227        deleteFamilyBloomHitCount.increment();
228        break;
229      case TRAILER:
230        trailerHitCount.increment();
231        break;
232      default:
233        // If there's a new type that's fine
234        // Ignore it for now. This is metrics don't exception.
235        break;
236    }
237  }
238
239  public void evict() {
240    evictionCount.increment();
241  }
242
243  public void evicted(final long t, boolean primary) {
244    if (t > this.startTime) {
245      this.ageAtEviction.add((t - this.startTime) / BlockCacheUtil.NANOS_PER_SECOND, 1);
246    }
247    this.evictedBlockCount.increment();
248    if (primary) {
249      primaryEvictedBlockCount.increment();
250    }
251  }
252
253  public long failInsert() {
254    return failedInserts.incrementAndGet();
255  }
256
257  // All of the counts of misses and hits.
258  public long getDataMissCount() {
259    return dataMissCount.sum();
260  }
261
262  public long getLeafIndexMissCount() {
263    return leafIndexMissCount.sum();
264  }
265
266  public long getBloomChunkMissCount() {
267    return bloomChunkMissCount.sum();
268  }
269
270  public long getMetaMissCount() {
271    return metaMissCount.sum();
272  }
273
274  public long getRootIndexMissCount() {
275    return rootIndexMissCount.sum();
276  }
277
278  public long getIntermediateIndexMissCount() {
279    return intermediateIndexMissCount.sum();
280  }
281
282  public long getFileInfoMissCount() {
283    return fileInfoMissCount.sum();
284  }
285
286  public long getGeneralBloomMetaMissCount() {
287    return generalBloomMetaMissCount.sum();
288  }
289
290  public long getDeleteFamilyBloomMissCount() {
291    return deleteFamilyBloomMissCount.sum();
292  }
293
294  public long getTrailerMissCount() {
295    return trailerMissCount.sum();
296  }
297
298  public long getDataHitCount() {
299    return dataHitCount.sum();
300  }
301
302  public long getLeafIndexHitCount() {
303    return leafIndexHitCount.sum();
304  }
305
306  public long getBloomChunkHitCount() {
307    return bloomChunkHitCount.sum();
308  }
309
310  public long getMetaHitCount() {
311    return metaHitCount.sum();
312  }
313
314  public long getRootIndexHitCount() {
315    return rootIndexHitCount.sum();
316  }
317
318  public long getIntermediateIndexHitCount() {
319    return intermediateIndexHitCount.sum();
320  }
321
322  public long getFileInfoHitCount() {
323    return fileInfoHitCount.sum();
324  }
325
326  public long getGeneralBloomMetaHitCount() {
327    return generalBloomMetaHitCount.sum();
328  }
329
330  public long getDeleteFamilyBloomHitCount() {
331    return deleteFamilyBloomHitCount.sum();
332  }
333
334  public long getTrailerHitCount() {
335    return trailerHitCount.sum();
336  }
337
338  public long getRequestCount() {
339    return getHitCount() + getMissCount();
340  }
341
342  public long getRequestCachingCount() {
343    return getHitCachingCount() + getMissCachingCount();
344  }
345
346  public long getMissCount() {
347    return missCount.sum();
348  }
349
350  public long getPrimaryMissCount() {
351    return primaryMissCount.sum();
352  }
353
354  public long getMissCachingCount() {
355    return missCachingCount.sum();
356  }
357
358  public long getHitCount() {
359    return hitCount.sum();
360  }
361
362  public long getPrimaryHitCount() {
363    return primaryHitCount.sum();
364  }
365
366  public long getHitCachingCount() {
367    return hitCachingCount.sum();
368  }
369
370  public long getEvictionCount() {
371    return evictionCount.sum();
372  }
373
374  public long getEvictedCount() {
375    return this.evictedBlockCount.sum();
376  }
377
378  public long getPrimaryEvictedCount() {
379    return primaryEvictedBlockCount.sum();
380  }
381
382  public double getHitRatio() {
383    double requestCount = getRequestCount();
384
385    if (requestCount == 0) {
386      return 0;
387    }
388
389    return getHitCount() / requestCount;
390  }
391
392  public double getHitCachingRatio() {
393    double requestCachingCount = getRequestCachingCount();
394
395    if (requestCachingCount == 0) {
396      return 0;
397    }
398
399    return getHitCachingCount() / requestCachingCount;
400  }
401
402  public double getMissRatio() {
403    double requestCount = getRequestCount();
404
405    if (requestCount == 0) {
406      return 0;
407    }
408
409    return getMissCount() / requestCount;
410  }
411
412  public double getMissCachingRatio() {
413    double requestCachingCount = getRequestCachingCount();
414
415    if (requestCachingCount == 0) {
416      return 0;
417    }
418
419    return getMissCachingCount() / requestCachingCount;
420  }
421
422  public double evictedPerEviction() {
423    double evictionCount = getEvictionCount();
424
425    if (evictionCount == 0) {
426      return 0;
427    }
428
429    return getEvictedCount() / evictionCount;
430  }
431
432  public long getFailedInserts() {
433    return failedInserts.get();
434  }
435
436  public void rollMetricsPeriod() {
437    hitCounts[windowIndex] = getHitCount() - lastHitCount;
438    lastHitCount = getHitCount();
439    hitCachingCounts[windowIndex] = getHitCachingCount() - lastHitCachingCount;
440    lastHitCachingCount = getHitCachingCount();
441    requestCounts[windowIndex] = getRequestCount() - lastRequestCount;
442    lastRequestCount = getRequestCount();
443    requestCachingCounts[windowIndex] = getRequestCachingCount() - lastRequestCachingCount;
444    lastRequestCachingCount = getRequestCachingCount();
445    windowIndex = (windowIndex + 1) % numPeriodsInWindow;
446  }
447
448  public long getSumHitCountsPastNPeriods() {
449    return sum(hitCounts);
450  }
451
452  public long getSumRequestCountsPastNPeriods() {
453    return sum(requestCounts);
454  }
455
456  public long getSumHitCachingCountsPastNPeriods() {
457    return sum(hitCachingCounts);
458  }
459
460  public long getSumRequestCachingCountsPastNPeriods() {
461    return sum(requestCachingCounts);
462  }
463
464  public double getHitRatioPastNPeriods() {
465    double ratio =
466      ((double) getSumHitCountsPastNPeriods() / (double) getSumRequestCountsPastNPeriods());
467    return Double.isNaN(ratio) ? 0 : ratio;
468  }
469
470  public double getHitCachingRatioPastNPeriods() {
471    double ratio = ((double) getSumHitCachingCountsPastNPeriods()
472      / (double) getSumRequestCachingCountsPastNPeriods());
473    return Double.isNaN(ratio) ? 0 : ratio;
474  }
475
476  public AgeSnapshot getAgeAtEvictionSnapshot() {
477    return new AgeSnapshot(this.ageAtEviction);
478  }
479
480  private static long sum(long[] counts) {
481    return Arrays.stream(counts).sum();
482  }
483}