View Javadoc

1   /**
2    *
3    * Licensed to the Apache Software Foundation (ASF) under one
4    * or more contributor license agreements.  See the NOTICE file
5    * distributed with this work for additional information
6    * regarding copyright ownership.  The ASF licenses this file
7    * to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance
9    * with the License.  You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS,
15   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16   * See the License for the specific language governing permissions and
17   * limitations under the License.
18   */
19  package org.apache.hadoop.hbase.io.hfile;
20  
21  import java.util.concurrent.atomic.AtomicLong;
22  
23  import org.apache.hadoop.hbase.classification.InterfaceAudience;
24  
25  import com.yammer.metrics.core.Histogram;
26  import com.yammer.metrics.core.MetricsRegistry;
27  
28  /**
29   * Class that implements cache metrics.
30   */
31  @InterfaceAudience.Private
32  public class CacheStats {
33    /**
34     * Needed making histograms.
35     */
36    private static final MetricsRegistry METRICS = new MetricsRegistry();
37  
38    /** Sliding window statistics. The number of metric periods to include in
39     * sliding window hit ratio calculations.
40     */
41    static final int DEFAULT_WINDOW_PERIODS = 5;
42  
43    /** The number of getBlock requests that were cache hits */
44    private final AtomicLong hitCount = new AtomicLong(0);
45  
46    /** The number of getBlock requests that were cache hits from primary replica */
47    private final AtomicLong primaryHitCount = new AtomicLong(0);
48    
49    /**
50     * The number of getBlock requests that were cache hits, but only from
51     * requests that were set to use the block cache.  This is because all reads
52     * attempt to read from the block cache even if they will not put new blocks
53     * into the block cache.  See HBASE-2253 for more information.
54     */
55    private final AtomicLong hitCachingCount = new AtomicLong(0);
56  
57    /** The number of getBlock requests that were cache misses */
58    private final AtomicLong missCount = new AtomicLong(0);
59  
60    /** The number of getBlock requests for primary replica that were cache misses */
61    private final AtomicLong primaryMissCount = new AtomicLong(0);
62    /**
63     * The number of getBlock requests that were cache misses, but only from
64     * requests that were set to use the block cache.
65     */
66    private final AtomicLong missCachingCount = new AtomicLong(0);
67  
68    /** The number of times an eviction has occurred */
69    private final AtomicLong evictionCount = new AtomicLong(0);
70  
71    /** The total number of blocks that have been evicted */
72    private final AtomicLong evictedBlockCount = new AtomicLong(0);
73  
74    /** The total number of blocks for primary replica that have been evicted */
75    private final AtomicLong primaryEvictedBlockCount = new AtomicLong(0);
76  
77    /** The total number of blocks that were not inserted. */
78    private final AtomicLong failedInserts = new AtomicLong(0);
79  
80    /** The number of metrics periods to include in window */
81    private final int numPeriodsInWindow;
82    /** Hit counts for each period in window */
83    private final long [] hitCounts;
84    /** Caching hit counts for each period in window */
85    private final long [] hitCachingCounts;
86    /** Access counts for each period in window */
87    private final long [] requestCounts;
88    /** Caching access counts for each period in window */
89    private final long [] requestCachingCounts;
90    /** Last hit count read */
91    private long lastHitCount = 0;
92    /** Last hit caching count read */
93    private long lastHitCachingCount = 0;
94    /** Last request count read */
95    private long lastRequestCount = 0;
96    /** Last request caching count read */
97    private long lastRequestCachingCount = 0;
98    /** Current window index (next to be updated) */
99    private int windowIndex = 0;
100   /**
101    * Keep running age at eviction time
102    */
103   private Histogram ageAtEviction;
104   private long startTime = System.nanoTime();
105 
106   public CacheStats(final String name) {
107     this(name, DEFAULT_WINDOW_PERIODS);
108   }
109 
110   public CacheStats(final String name, int numPeriodsInWindow) {
111     this.numPeriodsInWindow = numPeriodsInWindow;
112     this.hitCounts = initializeZeros(numPeriodsInWindow);
113     this.hitCachingCounts = initializeZeros(numPeriodsInWindow);
114     this.requestCounts = initializeZeros(numPeriodsInWindow);
115     this.requestCachingCounts = initializeZeros(numPeriodsInWindow);
116     this.ageAtEviction = METRICS.newHistogram(CacheStats.class, name + ".ageAtEviction");
117   }
118 
119   @Override
120   public String toString() {
121     AgeSnapshot snapshot = getAgeAtEvictionSnapshot();
122     return "hitCount=" + getHitCount() + ", hitCachingCount=" + getHitCachingCount() +
123       ", missCount=" + getMissCount() + ", missCachingCount=" + getMissCachingCount() +
124       ", evictionCount=" + getEvictionCount() +
125       ", evictedBlockCount=" + getEvictedCount() +
126       ", primaryMissCount=" + getPrimaryMissCount() +
127       ", primaryHitCount=" + getPrimaryHitCount() +
128       ", evictedAgeMean=" + snapshot.getMean() +
129       ", evictedAgeStdDev=" + snapshot.getStdDev();
130   }
131 
132   public void miss(boolean caching, boolean primary) {
133     missCount.incrementAndGet();
134     if (primary) primaryMissCount.incrementAndGet();
135     if (caching) missCachingCount.incrementAndGet();
136   }
137 
138   public void hit(boolean caching) {
139     hit(caching, true);
140   }
141 
142   public void hit(boolean caching, boolean primary) {
143     hitCount.incrementAndGet();
144     if (primary) primaryHitCount.incrementAndGet();
145     if (caching) hitCachingCount.incrementAndGet();
146   }
147 
148   public void evict() {
149     evictionCount.incrementAndGet();
150   }
151 
152   public void evicted(final long t, boolean primary) {
153     if (t > this.startTime) {
154       this.ageAtEviction.update((t - this.startTime)/BlockCacheUtil.NANOS_PER_SECOND);
155     }
156     this.evictedBlockCount.incrementAndGet();
157     if (primary) {
158       primaryEvictedBlockCount.incrementAndGet();
159     }
160   }
161 
162   public long failInsert() {
163     return failedInserts.incrementAndGet();
164   }
165 
166   public long getRequestCount() {
167     return getHitCount() + getMissCount();
168   }
169 
170   public long getRequestCachingCount() {
171     return getHitCachingCount() + getMissCachingCount();
172   }
173 
174   public long getMissCount() {
175     return missCount.get();
176   }
177 
178   public long getPrimaryMissCount() {
179     return primaryMissCount.get();
180   }
181 
182   public long getMissCachingCount() {
183     return missCachingCount.get();
184   }
185 
186   public long getHitCount() {
187     return hitCount.get();
188   }
189 
190   public long getPrimaryHitCount() {
191     return primaryHitCount.get();
192   }
193 
194   public long getHitCachingCount() {
195     return hitCachingCount.get();
196   }
197 
198   public long getEvictionCount() {
199     return evictionCount.get();
200   }
201 
202   public long getEvictedCount() {
203     return this.evictedBlockCount.get();
204   }
205 
206   public long getPrimaryEvictedCount() {
207     return primaryEvictedBlockCount.get();
208   }
209 
210   public double getHitRatio() {
211     return ((float)getHitCount()/(float)getRequestCount());
212   }
213 
214   public double getHitCachingRatio() {
215     return ((float)getHitCachingCount()/(float)getRequestCachingCount());
216   }
217 
218   public double getMissRatio() {
219     return ((float)getMissCount()/(float)getRequestCount());
220   }
221 
222   public double getMissCachingRatio() {
223     return ((float)getMissCachingCount()/(float)getRequestCachingCount());
224   }
225 
226   public double evictedPerEviction() {
227     return ((float)getEvictedCount()/(float)getEvictionCount());
228   }
229 
230   public long getFailedInserts() {
231     return failedInserts.get();
232   }
233 
234   public void rollMetricsPeriod() {
235     hitCounts[windowIndex] = getHitCount() - lastHitCount;
236     lastHitCount = getHitCount();
237     hitCachingCounts[windowIndex] =
238       getHitCachingCount() - lastHitCachingCount;
239     lastHitCachingCount = getHitCachingCount();
240     requestCounts[windowIndex] = getRequestCount() - lastRequestCount;
241     lastRequestCount = getRequestCount();
242     requestCachingCounts[windowIndex] =
243       getRequestCachingCount() - lastRequestCachingCount;
244     lastRequestCachingCount = getRequestCachingCount();
245     windowIndex = (windowIndex + 1) % numPeriodsInWindow;
246   }
247 
248   public long getSumHitCountsPastNPeriods() {
249     return sum(hitCounts);
250   }
251 
252   public long getSumRequestCountsPastNPeriods() {
253     return sum(requestCounts);
254   }
255 
256   public long getSumHitCachingCountsPastNPeriods() {
257     return sum(hitCachingCounts);
258   }
259 
260   public long getSumRequestCachingCountsPastNPeriods() {
261     return sum(requestCachingCounts);
262   }
263 
264   public double getHitRatioPastNPeriods() {
265     double ratio = ((double)getSumHitCountsPastNPeriods() /
266         (double)getSumRequestCountsPastNPeriods());
267     return Double.isNaN(ratio) ? 0 : ratio;
268   }
269 
270   public double getHitCachingRatioPastNPeriods() {
271     double ratio = ((double)getSumHitCachingCountsPastNPeriods() /
272         (double)getSumRequestCachingCountsPastNPeriods());
273     return Double.isNaN(ratio) ? 0 : ratio;
274   }
275 
276   public AgeSnapshot getAgeAtEvictionSnapshot() {
277     return new AgeSnapshot(this.ageAtEviction);
278   }
279 
280   private static long sum(long [] counts) {
281     long sum = 0;
282     for (long count : counts) sum += count;
283     return sum;
284   }
285 
286   private static long [] initializeZeros(int n) {
287     long [] zeros = new long [n];
288     for (int i=0; i<n; i++) {
289       zeros[i] = 0L;
290     }
291     return zeros;
292   }
293 }