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.regionserver;
019
020import java.io.IOException;
021import java.util.ArrayList;
022import java.util.Collection;
023import java.util.List;
024import java.util.Map;
025import java.util.OptionalDouble;
026import java.util.OptionalLong;
027import java.util.concurrent.ConcurrentHashMap;
028import java.util.concurrent.ScheduledExecutorService;
029import java.util.concurrent.TimeUnit;
030
031import org.apache.commons.lang3.StringUtils;
032import org.apache.hadoop.hbase.CompatibilitySingletonFactory;
033import org.apache.hadoop.hbase.HConstants;
034import org.apache.hadoop.hbase.HDFSBlocksDistribution;
035import org.apache.hadoop.hbase.HRegionInfo;
036import org.apache.hadoop.hbase.ServerName;
037import org.apache.hadoop.hbase.io.hfile.BlockCache;
038import org.apache.hadoop.hbase.io.hfile.CacheStats;
039import org.apache.hadoop.hbase.io.hfile.CombinedBlockCache;
040import org.apache.hadoop.hbase.mob.MobFileCache;
041import org.apache.hadoop.hbase.regionserver.wal.MetricsWALSource;
042import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
043import org.apache.hadoop.hbase.util.FSUtils;
044import org.apache.hadoop.hbase.wal.WALProvider;
045import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
046import org.apache.hadoop.hdfs.DFSHedgedReadMetrics;
047import org.apache.hadoop.metrics2.MetricsExecutor;
048import org.apache.yetus.audience.InterfaceAudience;
049import org.slf4j.Logger;
050import org.slf4j.LoggerFactory;
051
052/**
053 * Impl for exposing HRegionServer Information through Hadoop's metrics 2 system.
054 */
055@InterfaceAudience.Private
056class MetricsRegionServerWrapperImpl
057    implements MetricsRegionServerWrapper {
058
059  private static final Logger LOG = LoggerFactory.getLogger(MetricsRegionServerWrapperImpl.class);
060
061  private final HRegionServer regionServer;
062  private final MetricsWALSource metricsWALSource;
063
064  private BlockCache blockCache;
065  private MobFileCache mobFileCache;
066  private CacheStats cacheStats;
067  private CacheStats l1Stats = null;
068  private CacheStats l2Stats = null;
069
070  private volatile long numStores = 0;
071  private volatile long numWALFiles = 0;
072  private volatile long walFileSize = 0;
073  private volatile long numStoreFiles = 0;
074  private volatile long memstoreSize = 0;
075  private volatile long storeFileSize = 0;
076  private volatile long maxStoreFileAge = 0;
077  private volatile long minStoreFileAge = 0;
078  private volatile long avgStoreFileAge = 0;
079  private volatile long numReferenceFiles = 0;
080  private volatile double requestsPerSecond = 0.0;
081  private volatile long readRequestsCount = 0;
082  private volatile double readRequestsRatePerSecond = 0;
083  private volatile long filteredReadRequestsCount = 0;
084  private volatile long writeRequestsCount = 0;
085  private volatile double writeRequestsRatePerSecond = 0;
086  private volatile long checkAndMutateChecksFailed = 0;
087  private volatile long checkAndMutateChecksPassed = 0;
088  private volatile long storefileIndexSize = 0;
089  private volatile long totalStaticIndexSize = 0;
090  private volatile long totalStaticBloomSize = 0;
091  private volatile long numMutationsWithoutWAL = 0;
092  private volatile long dataInMemoryWithoutWAL = 0;
093  private volatile double percentFileLocal = 0;
094  private volatile double percentFileLocalSecondaryRegions = 0;
095  private volatile long flushedCellsCount = 0;
096  private volatile long compactedCellsCount = 0;
097  private volatile long majorCompactedCellsCount = 0;
098  private volatile long flushedCellsSize = 0;
099  private volatile long compactedCellsSize = 0;
100  private volatile long majorCompactedCellsSize = 0;
101  private volatile long cellsCountCompactedToMob = 0;
102  private volatile long cellsCountCompactedFromMob = 0;
103  private volatile long cellsSizeCompactedToMob = 0;
104  private volatile long cellsSizeCompactedFromMob = 0;
105  private volatile long mobFlushCount = 0;
106  private volatile long mobFlushedCellsCount = 0;
107  private volatile long mobFlushedCellsSize = 0;
108  private volatile long mobScanCellsCount = 0;
109  private volatile long mobScanCellsSize = 0;
110  private volatile long mobFileCacheAccessCount = 0;
111  private volatile long mobFileCacheMissCount = 0;
112  private volatile double mobFileCacheHitRatio = 0;
113  private volatile long mobFileCacheEvictedCount = 0;
114  private volatile long mobFileCacheCount = 0;
115  private volatile long blockedRequestsCount = 0L;
116  private volatile long averageRegionSize = 0L;
117  protected final Map<String, ArrayList<Long>>
118      requestsCountCache = new ConcurrentHashMap<String, ArrayList<Long>>();
119
120  private ScheduledExecutorService executor;
121  private Runnable runnable;
122  private long period;
123
124  /**
125   * Can be null if not on hdfs.
126   */
127  private DFSHedgedReadMetrics dfsHedgedReadMetrics;
128
129  public MetricsRegionServerWrapperImpl(final HRegionServer regionServer) {
130    this.regionServer = regionServer;
131    initBlockCache();
132    initMobFileCache();
133
134    this.period =
135        regionServer.conf.getLong(HConstants.REGIONSERVER_METRICS_PERIOD,
136          HConstants.DEFAULT_REGIONSERVER_METRICS_PERIOD);
137
138    this.executor = CompatibilitySingletonFactory.getInstance(MetricsExecutor.class).getExecutor();
139    this.runnable = new RegionServerMetricsWrapperRunnable();
140    this.executor.scheduleWithFixedDelay(this.runnable, this.period, this.period,
141      TimeUnit.MILLISECONDS);
142    this.metricsWALSource = CompatibilitySingletonFactory.getInstance(MetricsWALSource.class);
143
144    try {
145      this.dfsHedgedReadMetrics = FSUtils.getDFSHedgedReadMetrics(regionServer.getConfiguration());
146    } catch (IOException e) {
147      LOG.warn("Failed to get hedged metrics", e);
148    }
149    if (LOG.isInfoEnabled()) {
150      LOG.info("Computing regionserver metrics every " + this.period + " milliseconds");
151    }
152  }
153
154  private void initBlockCache() {
155    this.blockCache = this.regionServer.getBlockCache().orElse(null);
156    this.cacheStats = this.blockCache != null ? this.blockCache.getStats() : null;
157    if (this.cacheStats != null) {
158      if (this.cacheStats instanceof CombinedBlockCache.CombinedCacheStats) {
159        l1Stats = ((CombinedBlockCache.CombinedCacheStats) this.cacheStats)
160          .getLruCacheStats();
161        l2Stats = ((CombinedBlockCache.CombinedCacheStats) this.cacheStats)
162          .getBucketCacheStats();
163      } else {
164        l1Stats = this.cacheStats;
165      }
166    }
167  }
168
169  /**
170   * Initializes the mob file cache.
171   */
172  private void initMobFileCache() {
173    this.mobFileCache = this.regionServer.getMobFileCache().orElse(null);
174  }
175
176  @Override
177  public String getClusterId() {
178    return regionServer.getClusterId();
179  }
180
181  @Override
182  public long getStartCode() {
183    return regionServer.getStartcode();
184  }
185
186  @Override
187  public String getZookeeperQuorum() {
188    ZKWatcher zk = regionServer.getZooKeeper();
189    if (zk == null) {
190      return "";
191    }
192    return zk.getQuorum();
193  }
194
195  @Override
196  public String getCoprocessors() {
197    String[] coprocessors = regionServer.getRegionServerCoprocessors();
198    if (coprocessors == null || coprocessors.length == 0) {
199      return "";
200    }
201    return StringUtils.join(coprocessors, ", ");
202  }
203
204  @Override
205  public String getServerName() {
206    ServerName serverName = regionServer.getServerName();
207    if (serverName == null) {
208      return "";
209    }
210    return serverName.getServerName();
211  }
212
213  @Override
214  public long getNumOnlineRegions() {
215    Collection<HRegion> onlineRegionsLocalContext = regionServer.getOnlineRegionsLocalContext();
216    if (onlineRegionsLocalContext == null) {
217      return 0;
218    }
219    return onlineRegionsLocalContext.size();
220  }
221
222  @Override
223  public long getTotalRequestCount() {
224    return regionServer.rpcServices.requestCount.sum();
225  }
226
227  @Override
228  public long getTotalRowActionRequestCount() {
229    return readRequestsCount + writeRequestsCount;
230  }
231
232  @Override
233  public int getSplitQueueSize() {
234    if (this.regionServer.compactSplitThread == null) {
235      return 0;
236    }
237    return this.regionServer.compactSplitThread.getSplitQueueSize();
238  }
239
240  @Override
241  public int getCompactionQueueSize() {
242    //The thread could be zero.  if so assume there is no queue.
243    if (this.regionServer.compactSplitThread == null) {
244      return 0;
245    }
246    return this.regionServer.compactSplitThread.getCompactionQueueSize();
247  }
248
249  @Override
250  public int getSmallCompactionQueueSize() {
251    //The thread could be zero.  if so assume there is no queue.
252    if (this.regionServer.compactSplitThread == null) {
253      return 0;
254    }
255    return this.regionServer.compactSplitThread.getSmallCompactionQueueSize();
256  }
257
258  @Override
259  public int getLargeCompactionQueueSize() {
260    //The thread could be zero.  if so assume there is no queue.
261    if (this.regionServer.compactSplitThread == null) {
262      return 0;
263    }
264    return this.regionServer.compactSplitThread.getLargeCompactionQueueSize();
265  }
266
267  @Override
268  public int getFlushQueueSize() {
269    //If there is no flusher there should be no queue.
270    if (this.regionServer.cacheFlusher == null) {
271      return 0;
272    }
273    return this.regionServer.cacheFlusher.getFlushQueueSize();
274  }
275
276  @Override
277  public long getBlockCacheCount() {
278    return this.blockCache != null ? this.blockCache.getBlockCount() : 0L;
279  }
280
281  @Override
282  public long getMemStoreLimit() {
283    return this.regionServer.getRegionServerAccounting().getGlobalMemStoreLimit();
284  }
285
286  @Override
287  public long getBlockCacheSize() {
288    return this.blockCache != null ? this.blockCache.getCurrentSize() : 0L;
289  }
290
291  @Override
292  public long getBlockCacheFreeSize() {
293    return this.blockCache != null ? this.blockCache.getFreeSize() : 0L;
294  }
295
296  @Override
297  public long getBlockCacheHitCount() {
298    return this.cacheStats != null ? this.cacheStats.getHitCount() : 0L;
299  }
300
301  @Override
302  public long getBlockCachePrimaryHitCount() {
303    return this.cacheStats != null ? this.cacheStats.getPrimaryHitCount() : 0L;
304  }
305
306  @Override
307  public long getBlockCacheMissCount() {
308    return this.cacheStats != null ? this.cacheStats.getMissCount() : 0L;
309  }
310
311  @Override
312  public long getBlockCachePrimaryMissCount() {
313    return this.cacheStats != null ? this.cacheStats.getPrimaryMissCount() : 0L;
314  }
315
316  @Override
317  public long getBlockCacheEvictedCount() {
318    return this.cacheStats != null ? this.cacheStats.getEvictedCount() : 0L;
319  }
320
321  @Override
322  public long getBlockCachePrimaryEvictedCount() {
323    return this.cacheStats != null ? this.cacheStats.getPrimaryEvictedCount() : 0L;
324  }
325
326  @Override
327  public double getBlockCacheHitPercent() {
328    double ratio = this.cacheStats != null ? this.cacheStats.getHitRatio() : 0.0;
329    if (Double.isNaN(ratio)) {
330      ratio = 0;
331    }
332    return (ratio * 100);
333  }
334
335  @Override
336  public double getBlockCacheHitCachingPercent() {
337    double ratio = this.cacheStats != null ? this.cacheStats.getHitCachingRatio() : 0.0;
338    if (Double.isNaN(ratio)) {
339      ratio = 0;
340    }
341    return (ratio * 100);
342  }
343
344  @Override
345  public long getBlockCacheFailedInsertions() {
346    return this.cacheStats != null ? this.cacheStats.getFailedInserts() : 0L;
347  }
348
349  @Override
350  public long getL1CacheHitCount() {
351    return this.l1Stats != null ? this.l1Stats.getHitCount() : 0L;
352  }
353
354  @Override
355  public long getL1CacheMissCount() {
356    return this.l1Stats != null ? this.l1Stats.getMissCount() : 0L;
357  }
358
359  @Override
360  public double getL1CacheHitRatio() {
361    return this.l1Stats != null ? this.l1Stats.getHitRatio() : 0.0;
362  }
363
364  @Override
365  public double getL1CacheMissRatio() {
366    return this.l1Stats != null ? this.l1Stats.getMissRatio() : 0.0;
367  }
368
369  @Override
370  public long getL2CacheHitCount() {
371    return this.l2Stats != null ? this.l2Stats.getHitCount() : 0L;
372  }
373
374  @Override
375  public long getL2CacheMissCount() {
376    return this.l2Stats != null ? this.l2Stats.getMissCount() : 0L;
377  }
378
379  @Override
380  public double getL2CacheHitRatio() {
381    return this.l2Stats != null ? this.l2Stats.getHitRatio() : 0.0;
382  }
383
384  @Override
385  public double getL2CacheMissRatio() {
386    return this.l2Stats != null ? this.l2Stats.getMissRatio() : 0.0;
387  }
388
389  @Override public void forceRecompute() {
390    this.runnable.run();
391  }
392
393  @Override
394  public long getNumStores() {
395    return numStores;
396  }
397
398  @Override
399  public long getNumWALFiles() {
400    return numWALFiles;
401  }
402
403  @Override
404  public long getWALFileSize() {
405    return walFileSize;
406  }
407
408  @Override
409  public long getNumWALSlowAppend() {
410    return metricsWALSource.getSlowAppendCount();
411  }
412
413  @Override
414  public long getNumStoreFiles() {
415    return numStoreFiles;
416  }
417
418  @Override
419  public long getMaxStoreFileAge() {
420    return maxStoreFileAge;
421  }
422
423  @Override
424  public long getMinStoreFileAge() {
425    return minStoreFileAge;
426  }
427
428  @Override
429  public long getAvgStoreFileAge() {
430    return avgStoreFileAge;
431  }
432
433  @Override
434  public long getNumReferenceFiles() {
435    return numReferenceFiles;
436  }
437
438  @Override
439  public long getMemStoreSize() {
440    return memstoreSize;
441  }
442
443  @Override
444  public long getStoreFileSize() {
445    return storeFileSize;
446  }
447
448  @Override public double getRequestsPerSecond() {
449    return requestsPerSecond;
450  }
451
452  @Override
453  public long getReadRequestsCount() {
454    return readRequestsCount;
455  }
456
457  @Override
458  public double getReadRequestsRatePerSecond() {
459    return readRequestsRatePerSecond;
460  }
461
462  @Override
463  public long getFilteredReadRequestsCount() {
464    return filteredReadRequestsCount;
465  }
466
467  @Override
468  public long getWriteRequestsCount() {
469    return writeRequestsCount;
470  }
471
472  @Override
473  public double getWriteRequestsRatePerSecond() {
474    return writeRequestsRatePerSecond;
475  }
476
477  @Override
478  public long getRpcGetRequestsCount() {
479    return regionServer.rpcServices.rpcGetRequestCount.sum();
480  }
481
482  @Override
483  public long getRpcScanRequestsCount() {
484    return regionServer.rpcServices.rpcScanRequestCount.sum();
485  }
486
487  @Override
488  public long getRpcMultiRequestsCount() {
489    return regionServer.rpcServices.rpcMultiRequestCount.sum();
490  }
491
492  @Override
493  public long getRpcMutateRequestsCount() {
494    return regionServer.rpcServices.rpcMutateRequestCount.sum();
495  }
496
497  @Override
498  public long getCheckAndMutateChecksFailed() {
499    return checkAndMutateChecksFailed;
500  }
501
502  @Override
503  public long getCheckAndMutateChecksPassed() {
504    return checkAndMutateChecksPassed;
505  }
506
507  @Override
508  public long getStoreFileIndexSize() {
509    return storefileIndexSize;
510  }
511
512  @Override
513  public long getTotalStaticIndexSize() {
514    return totalStaticIndexSize;
515  }
516
517  @Override
518  public long getTotalStaticBloomSize() {
519    return totalStaticBloomSize;
520  }
521
522  @Override
523  public long getNumMutationsWithoutWAL() {
524    return numMutationsWithoutWAL;
525  }
526
527  @Override
528  public long getDataInMemoryWithoutWAL() {
529    return dataInMemoryWithoutWAL;
530  }
531
532  @Override
533  public double getPercentFileLocal() {
534    return percentFileLocal;
535  }
536
537  @Override
538  public double getPercentFileLocalSecondaryRegions() {
539    return percentFileLocalSecondaryRegions;
540  }
541
542  @Override
543  public long getUpdatesBlockedTime() {
544    if (this.regionServer.cacheFlusher == null) {
545      return 0;
546    }
547    return this.regionServer.cacheFlusher.getUpdatesBlockedMsHighWater().sum();
548  }
549
550  @Override
551  public long getFlushedCellsCount() {
552    return flushedCellsCount;
553  }
554
555  @Override
556  public long getCompactedCellsCount() {
557    return compactedCellsCount;
558  }
559
560  @Override
561  public long getMajorCompactedCellsCount() {
562    return majorCompactedCellsCount;
563  }
564
565  @Override
566  public long getFlushedCellsSize() {
567    return flushedCellsSize;
568  }
569
570  @Override
571  public long getCompactedCellsSize() {
572    return compactedCellsSize;
573  }
574
575  @Override
576  public long getMajorCompactedCellsSize() {
577    return majorCompactedCellsSize;
578  }
579
580  @Override
581  public long getCellsCountCompactedFromMob() {
582    return cellsCountCompactedFromMob;
583  }
584
585  @Override
586  public long getCellsCountCompactedToMob() {
587    return cellsCountCompactedToMob;
588  }
589
590  @Override
591  public long getCellsSizeCompactedFromMob() {
592    return cellsSizeCompactedFromMob;
593  }
594
595  @Override
596  public long getCellsSizeCompactedToMob() {
597    return cellsSizeCompactedToMob;
598  }
599
600  @Override
601  public long getMobFlushCount() {
602    return mobFlushCount;
603  }
604
605  @Override
606  public long getMobFlushedCellsCount() {
607    return mobFlushedCellsCount;
608  }
609
610  @Override
611  public long getMobFlushedCellsSize() {
612    return mobFlushedCellsSize;
613  }
614
615  @Override
616  public long getMobScanCellsCount() {
617    return mobScanCellsCount;
618  }
619
620  @Override
621  public long getMobScanCellsSize() {
622    return mobScanCellsSize;
623  }
624
625  @Override
626  public long getMobFileCacheAccessCount() {
627    return mobFileCacheAccessCount;
628  }
629
630  @Override
631  public long getMobFileCacheMissCount() {
632    return mobFileCacheMissCount;
633  }
634
635  @Override
636  public long getMobFileCacheCount() {
637    return mobFileCacheCount;
638  }
639
640  @Override
641  public long getMobFileCacheEvictedCount() {
642    return mobFileCacheEvictedCount;
643  }
644
645  @Override
646  public double getMobFileCacheHitPercent() {
647    return mobFileCacheHitRatio * 100;
648  }
649
650  /**
651   * This is the runnable that will be executed on the executor every PERIOD number of seconds
652   * It will take metrics/numbers from all of the regions and use them to compute point in
653   * time metrics.
654   */
655  public class RegionServerMetricsWrapperRunnable implements Runnable {
656
657    private long lastRan = 0;
658
659    @Override
660    synchronized public void run() {
661      try {
662        HDFSBlocksDistribution hdfsBlocksDistribution =
663            new HDFSBlocksDistribution();
664        HDFSBlocksDistribution hdfsBlocksDistributionSecondaryRegions =
665            new HDFSBlocksDistribution();
666
667        long tempNumStores = 0, tempNumStoreFiles = 0, tempMemstoreSize = 0, tempStoreFileSize = 0;
668        long tempMaxStoreFileAge = 0, tempNumReferenceFiles = 0;
669        long avgAgeNumerator = 0, numHFiles = 0;
670        long tempMinStoreFileAge = Long.MAX_VALUE;
671        long tempReadRequestsCount = 0, tempFilteredReadRequestsCount = 0,
672          tempWriteRequestsCount = 0;
673        long tempCheckAndMutateChecksFailed = 0;
674        long tempCheckAndMutateChecksPassed = 0;
675        long tempStorefileIndexSize = 0;
676        long tempTotalStaticIndexSize = 0;
677        long tempTotalStaticBloomSize = 0;
678        long tempNumMutationsWithoutWAL = 0;
679        long tempDataInMemoryWithoutWAL = 0;
680        double tempPercentFileLocal = 0;
681        double tempPercentFileLocalSecondaryRegions = 0;
682        long tempFlushedCellsCount = 0;
683        long tempCompactedCellsCount = 0;
684        long tempMajorCompactedCellsCount = 0;
685        long tempFlushedCellsSize = 0;
686        long tempCompactedCellsSize = 0;
687        long tempMajorCompactedCellsSize = 0;
688        long tempCellsCountCompactedToMob = 0;
689        long tempCellsCountCompactedFromMob = 0;
690        long tempCellsSizeCompactedToMob = 0;
691        long tempCellsSizeCompactedFromMob = 0;
692        long tempMobFlushCount = 0;
693        long tempMobFlushedCellsCount = 0;
694        long tempMobFlushedCellsSize = 0;
695        long tempMobScanCellsCount = 0;
696        long tempMobScanCellsSize = 0;
697        long tempBlockedRequestsCount = 0;
698        int regionCount = 0;
699
700        long currentReadRequestsCount = 0;
701        long currentWriteRequestsCount = 0;
702        long lastReadRequestsCount = 0;
703        long lastWriteRequestsCount = 0;
704        long readRequestsDelta = 0;
705        long writeRequestsDelta = 0;
706        long totalReadRequestsDelta = 0;
707        long totalWriteRequestsDelta = 0;
708        String encodedRegionName;
709        for (HRegion r : regionServer.getOnlineRegionsLocalContext()) {
710          encodedRegionName = r.getRegionInfo().getEncodedName();
711          currentReadRequestsCount = r.getReadRequestsCount();
712          currentWriteRequestsCount = r.getWriteRequestsCount();
713          if (requestsCountCache.containsKey(encodedRegionName)) {
714            lastReadRequestsCount = requestsCountCache.get(encodedRegionName).get(0);
715            lastWriteRequestsCount = requestsCountCache.get(encodedRegionName).get(1);
716            readRequestsDelta = currentReadRequestsCount - lastReadRequestsCount;
717            writeRequestsDelta = currentWriteRequestsCount - lastWriteRequestsCount;
718            totalReadRequestsDelta += readRequestsDelta;
719            totalWriteRequestsDelta += writeRequestsDelta;
720            //Update cache for our next comparision
721            requestsCountCache.get(encodedRegionName).set(0,currentReadRequestsCount);
722            requestsCountCache.get(encodedRegionName).set(1,currentWriteRequestsCount);
723          } else {
724            // List[0] -> readRequestCount
725            // List[1] -> writeRequestCount
726            ArrayList<Long> requests = new ArrayList<Long>(2);
727            requests.add(currentReadRequestsCount);
728            requests.add(currentWriteRequestsCount);
729            requestsCountCache.put(encodedRegionName, requests);
730            totalReadRequestsDelta += currentReadRequestsCount;
731            totalWriteRequestsDelta += currentWriteRequestsCount;
732          }
733          tempNumMutationsWithoutWAL += r.getNumMutationsWithoutWAL();
734          tempDataInMemoryWithoutWAL += r.getDataInMemoryWithoutWAL();
735          tempReadRequestsCount += r.getReadRequestsCount();
736          tempFilteredReadRequestsCount += r.getFilteredReadRequestsCount();
737          tempWriteRequestsCount += r.getWriteRequestsCount();
738          tempCheckAndMutateChecksFailed += r.getCheckAndMutateChecksFailed();
739          tempCheckAndMutateChecksPassed += r.getCheckAndMutateChecksPassed();
740          tempBlockedRequestsCount += r.getBlockedRequestsCount();
741          List<? extends Store> storeList = r.getStores();
742          tempNumStores += storeList.size();
743          for (Store store : storeList) {
744            tempNumStoreFiles += store.getStorefilesCount();
745            tempMemstoreSize += store.getMemStoreSize().getDataSize();
746            tempStoreFileSize += store.getStorefilesSize();
747
748            OptionalLong storeMaxStoreFileAge = store.getMaxStoreFileAge();
749            if (storeMaxStoreFileAge.isPresent() &&
750                storeMaxStoreFileAge.getAsLong() > tempMaxStoreFileAge) {
751              tempMaxStoreFileAge = storeMaxStoreFileAge.getAsLong();
752            }
753
754            OptionalLong storeMinStoreFileAge = store.getMinStoreFileAge();
755            if (storeMinStoreFileAge.isPresent() &&
756                storeMinStoreFileAge.getAsLong() < tempMinStoreFileAge) {
757              tempMinStoreFileAge = storeMinStoreFileAge.getAsLong();
758            }
759
760            long storeHFiles = store.getNumHFiles();
761            numHFiles += storeHFiles;
762            tempNumReferenceFiles += store.getNumReferenceFiles();
763
764            OptionalDouble storeAvgStoreFileAge = store.getAvgStoreFileAge();
765            if (storeAvgStoreFileAge.isPresent()) {
766              avgAgeNumerator =
767                  (long) (avgAgeNumerator + storeAvgStoreFileAge.getAsDouble() * storeHFiles);
768            }
769
770            tempStorefileIndexSize += store.getStorefilesRootLevelIndexSize();
771            tempTotalStaticBloomSize += store.getTotalStaticBloomSize();
772            tempTotalStaticIndexSize += store.getTotalStaticIndexSize();
773            tempFlushedCellsCount += store.getFlushedCellsCount();
774            tempCompactedCellsCount += store.getCompactedCellsCount();
775            tempMajorCompactedCellsCount += store.getMajorCompactedCellsCount();
776            tempFlushedCellsSize += store.getFlushedCellsSize();
777            tempCompactedCellsSize += store.getCompactedCellsSize();
778            tempMajorCompactedCellsSize += store.getMajorCompactedCellsSize();
779            if (store instanceof HMobStore) {
780              HMobStore mobStore = (HMobStore) store;
781              tempCellsCountCompactedToMob += mobStore.getCellsCountCompactedToMob();
782              tempCellsCountCompactedFromMob += mobStore.getCellsCountCompactedFromMob();
783              tempCellsSizeCompactedToMob += mobStore.getCellsSizeCompactedToMob();
784              tempCellsSizeCompactedFromMob += mobStore.getCellsSizeCompactedFromMob();
785              tempMobFlushCount += mobStore.getMobFlushCount();
786              tempMobFlushedCellsCount += mobStore.getMobFlushedCellsCount();
787              tempMobFlushedCellsSize += mobStore.getMobFlushedCellsSize();
788              tempMobScanCellsCount += mobStore.getMobScanCellsCount();
789              tempMobScanCellsSize += mobStore.getMobScanCellsSize();
790            }
791          }
792
793          HDFSBlocksDistribution distro = r.getHDFSBlocksDistribution();
794          hdfsBlocksDistribution.add(distro);
795          if (r.getRegionInfo().getReplicaId() != HRegionInfo.DEFAULT_REPLICA_ID) {
796            hdfsBlocksDistributionSecondaryRegions.add(distro);
797          }
798          regionCount++;
799        }
800        float localityIndex = hdfsBlocksDistribution.getBlockLocalityIndex(
801            regionServer.getServerName().getHostname());
802        tempPercentFileLocal = Double.isNaN(tempBlockedRequestsCount) ? 0 : (localityIndex * 100);
803
804        float localityIndexSecondaryRegions = hdfsBlocksDistributionSecondaryRegions
805            .getBlockLocalityIndex(regionServer.getServerName().getHostname());
806        tempPercentFileLocalSecondaryRegions = Double.
807            isNaN(localityIndexSecondaryRegions) ? 0 : (localityIndexSecondaryRegions * 100);
808
809        // Compute the number of requests per second
810        long currentTime = EnvironmentEdgeManager.currentTime();
811
812        // assume that it took PERIOD seconds to start the executor.
813        // this is a guess but it's a pretty good one.
814        if (lastRan == 0) {
815          lastRan = currentTime - period;
816        }
817        // If we've time traveled keep the last requests per second.
818        if ((currentTime - lastRan) > 0) {
819          requestsPerSecond = (totalReadRequestsDelta + totalWriteRequestsDelta) /
820              ((currentTime - lastRan) / 1000.0);
821
822          double readRequestsRatePerMilliSecond = (double)totalReadRequestsDelta / period;
823          double writeRequestsRatePerMilliSecond = (double)totalWriteRequestsDelta / period;
824
825          readRequestsRatePerSecond = readRequestsRatePerMilliSecond * 1000.0;
826          writeRequestsRatePerSecond = writeRequestsRatePerMilliSecond * 1000.0;
827        }
828        lastRan = currentTime;
829
830        WALProvider provider = regionServer.walFactory.getWALProvider();
831        WALProvider metaProvider = regionServer.walFactory.getMetaWALProvider();
832        numWALFiles = (provider == null ? 0 : provider.getNumLogFiles()) +
833            (metaProvider == null ? 0 : metaProvider.getNumLogFiles());
834        walFileSize = (provider == null ? 0 : provider.getLogFileSize()) +
835            (provider == null ? 0 : provider.getLogFileSize());
836        // Copy over computed values so that no thread sees half computed values.
837        numStores = tempNumStores;
838        numStoreFiles = tempNumStoreFiles;
839        memstoreSize = tempMemstoreSize;
840        storeFileSize = tempStoreFileSize;
841        maxStoreFileAge = tempMaxStoreFileAge;
842        if (regionCount > 0) {
843          averageRegionSize = (memstoreSize + storeFileSize) / regionCount;
844        }
845        if (tempMinStoreFileAge != Long.MAX_VALUE) {
846          minStoreFileAge = tempMinStoreFileAge;
847        }
848
849        if (numHFiles != 0) {
850          avgStoreFileAge = avgAgeNumerator / numHFiles;
851        }
852
853        numReferenceFiles= tempNumReferenceFiles;
854        readRequestsCount = tempReadRequestsCount;
855        filteredReadRequestsCount = tempFilteredReadRequestsCount;
856        writeRequestsCount = tempWriteRequestsCount;
857        checkAndMutateChecksFailed = tempCheckAndMutateChecksFailed;
858        checkAndMutateChecksPassed = tempCheckAndMutateChecksPassed;
859        storefileIndexSize = tempStorefileIndexSize;
860        totalStaticIndexSize = tempTotalStaticIndexSize;
861        totalStaticBloomSize = tempTotalStaticBloomSize;
862        numMutationsWithoutWAL = tempNumMutationsWithoutWAL;
863        dataInMemoryWithoutWAL = tempDataInMemoryWithoutWAL;
864        percentFileLocal = tempPercentFileLocal;
865        percentFileLocalSecondaryRegions = tempPercentFileLocalSecondaryRegions;
866        flushedCellsCount = tempFlushedCellsCount;
867        compactedCellsCount = tempCompactedCellsCount;
868        majorCompactedCellsCount = tempMajorCompactedCellsCount;
869        flushedCellsSize = tempFlushedCellsSize;
870        compactedCellsSize = tempCompactedCellsSize;
871        majorCompactedCellsSize = tempMajorCompactedCellsSize;
872        cellsCountCompactedToMob = tempCellsCountCompactedToMob;
873        cellsCountCompactedFromMob = tempCellsCountCompactedFromMob;
874        cellsSizeCompactedToMob = tempCellsSizeCompactedToMob;
875        cellsSizeCompactedFromMob = tempCellsSizeCompactedFromMob;
876        mobFlushCount = tempMobFlushCount;
877        mobFlushedCellsCount = tempMobFlushedCellsCount;
878        mobFlushedCellsSize = tempMobFlushedCellsSize;
879        mobScanCellsCount = tempMobScanCellsCount;
880        mobScanCellsSize = tempMobScanCellsSize;
881        mobFileCacheAccessCount = mobFileCache != null ? mobFileCache.getAccessCount() : 0L;
882        mobFileCacheMissCount = mobFileCache != null ? mobFileCache.getMissCount() : 0L;
883        mobFileCacheHitRatio = mobFileCache != null ? mobFileCache.getHitRatio() : 0.0;
884        if (Double.isNaN(mobFileCacheHitRatio)) {
885          mobFileCacheHitRatio = 0.0;
886        }
887        mobFileCacheEvictedCount = mobFileCache != null ? mobFileCache.getEvictedFileCount() : 0L;
888        mobFileCacheCount = mobFileCache != null ? mobFileCache.getCacheSize() : 0;
889        blockedRequestsCount = tempBlockedRequestsCount;
890      } catch (Throwable e) {
891        LOG.warn("Caught exception! Will suppress and retry.", e);
892      }
893    }
894  }
895
896  @Override
897  public long getHedgedReadOps() {
898    return this.dfsHedgedReadMetrics == null? 0: this.dfsHedgedReadMetrics.getHedgedReadOps();
899  }
900
901  @Override
902  public long getHedgedReadWins() {
903    return this.dfsHedgedReadMetrics == null? 0: this.dfsHedgedReadMetrics.getHedgedReadWins();
904  }
905
906  @Override
907  public long getBlockedRequestsCount() {
908    return blockedRequestsCount;
909  }
910
911  @Override
912  public long getAverageRegionSize() {
913    return averageRegionSize;
914  }
915
916  @Override
917  public long getDataMissCount() {
918    return this.cacheStats != null ? this.cacheStats.getDataMissCount() : 0L;
919  }
920
921  @Override
922  public long getLeafIndexMissCount() {
923    return this.cacheStats != null ? this.cacheStats.getLeafIndexMissCount() : 0L;
924  }
925
926  @Override
927  public long getBloomChunkMissCount() {
928    return this.cacheStats != null ? this.cacheStats.getBloomChunkMissCount() : 0L;
929  }
930
931  @Override
932  public long getMetaMissCount() {
933    return this.cacheStats != null ? this.cacheStats.getMetaMissCount() : 0L;
934  }
935
936  @Override
937  public long getRootIndexMissCount() {
938    return this.cacheStats != null ? this.cacheStats.getRootIndexMissCount() : 0L;
939  }
940
941  @Override
942  public long getIntermediateIndexMissCount() {
943    return this.cacheStats != null ? this.cacheStats.getIntermediateIndexMissCount() : 0L;
944  }
945
946  @Override
947  public long getFileInfoMissCount() {
948    return this.cacheStats != null ? this.cacheStats.getFileInfoMissCount() : 0L;
949  }
950
951  @Override
952  public long getGeneralBloomMetaMissCount() {
953    return this.cacheStats != null ? this.cacheStats.getGeneralBloomMetaMissCount() : 0L;
954  }
955
956  @Override
957  public long getDeleteFamilyBloomMissCount() {
958    return this.cacheStats != null ? this.cacheStats.getDeleteFamilyBloomMissCount() : 0L;
959  }
960
961  @Override
962  public long getTrailerMissCount() {
963    return this.cacheStats != null ? this.cacheStats.getTrailerMissCount() : 0L;
964  }
965
966  @Override
967  public long getDataHitCount() {
968    return this.cacheStats != null ? this.cacheStats.getDataHitCount() : 0L;
969  }
970
971  @Override
972  public long getLeafIndexHitCount() {
973    return this.cacheStats != null ? this.cacheStats.getLeafIndexHitCount() : 0L;
974  }
975
976  @Override
977  public long getBloomChunkHitCount() {
978    return this.cacheStats != null ? this.cacheStats.getBloomChunkHitCount() : 0L;
979  }
980
981  @Override
982  public long getMetaHitCount() {
983    return this.cacheStats != null ? this.cacheStats.getMetaHitCount() : 0L;
984  }
985
986  @Override
987  public long getRootIndexHitCount() {
988    return this.cacheStats != null ? this.cacheStats.getRootIndexHitCount() : 0L;
989  }
990
991  @Override
992  public long getIntermediateIndexHitCount() {
993    return this.cacheStats != null ? this.cacheStats.getIntermediateIndexHitCount() : 0L;
994  }
995
996  @Override
997  public long getFileInfoHitCount() {
998    return this.cacheStats != null ? this.cacheStats.getFileInfoHitCount() : 0L;
999  }
1000
1001  @Override
1002  public long getGeneralBloomMetaHitCount() {
1003    return this.cacheStats != null ? this.cacheStats.getGeneralBloomMetaHitCount() : 0L;
1004  }
1005
1006  @Override
1007  public long getDeleteFamilyBloomHitCount() {
1008    return this.cacheStats != null ? this.cacheStats.getDeleteFamilyBloomHitCount() : 0L;
1009  }
1010
1011  @Override
1012  public long getTrailerHitCount() {
1013    return this.cacheStats != null ? this.cacheStats.getTrailerHitCount() : 0L;
1014  }
1015}