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.client; 019 020import java.io.IOException; 021import java.util.Optional; 022import java.util.concurrent.atomic.AtomicLong; 023import org.apache.hadoop.conf.Configuration; 024import org.apache.hadoop.hbase.coprocessor.ObserverContext; 025import org.apache.hadoop.hbase.coprocessor.RegionCoprocessor; 026import org.apache.hadoop.hbase.coprocessor.RegionCoprocessorEnvironment; 027import org.apache.hadoop.hbase.coprocessor.RegionObserver; 028import org.apache.hadoop.hbase.util.Threads; 029 030/** 031 * This coproceesor sleep 2s at first increment/append rpc call. 032 */ 033public class SleepAtFirstRpcCall implements RegionCoprocessor, RegionObserver { 034 static final AtomicLong ct = new AtomicLong(0); 035 static final String SLEEP_TIME_CONF_KEY = "hbase.coprocessor.SleepAtFirstRpcCall.sleepTime"; 036 static final long DEFAULT_SLEEP_TIME = 2000; 037 static final AtomicLong sleepTime = new AtomicLong(DEFAULT_SLEEP_TIME); 038 039 @Override 040 public Optional<RegionObserver> getRegionObserver() { 041 return Optional.of(this); 042 } 043 044 public SleepAtFirstRpcCall() { 045 } 046 047 @Override 048 public void postOpen(ObserverContext<? extends RegionCoprocessorEnvironment> c) { 049 RegionCoprocessorEnvironment env = c.getEnvironment(); 050 Configuration conf = env.getConfiguration(); 051 sleepTime.set(conf.getLong(SLEEP_TIME_CONF_KEY, DEFAULT_SLEEP_TIME)); 052 } 053 054 @Override 055 public Result postIncrement(final ObserverContext<? extends RegionCoprocessorEnvironment> e, 056 final Increment increment, final Result result) throws IOException { 057 if (ct.incrementAndGet() == 1) { 058 Threads.sleep(sleepTime.get()); 059 } 060 return result; 061 } 062 063 @Override 064 public Result postAppend(final ObserverContext<? extends RegionCoprocessorEnvironment> e, 065 final Append append, final Result result) throws IOException { 066 if (ct.incrementAndGet() == 1) { 067 Threads.sleep(sleepTime.get()); 068 } 069 return result; 070 } 071}