2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org
.apache
.hadoop
.hbase
.client
;
20 import java
.io
.IOException
;
21 import java
.util
.List
;
22 import java
.util
.Optional
;
23 import java
.util
.concurrent
.atomic
.AtomicLong
;
24 import org
.apache
.hadoop
.conf
.Configuration
;
25 import org
.apache
.hadoop
.hbase
.Cell
;
26 import org
.apache
.hadoop
.hbase
.HBaseTestingUtility
;
27 import org
.apache
.hadoop
.hbase
.HConstants
;
28 import org
.apache
.hadoop
.hbase
.coprocessor
.ObserverContext
;
29 import org
.apache
.hadoop
.hbase
.coprocessor
.RegionCoprocessor
;
30 import org
.apache
.hadoop
.hbase
.coprocessor
.RegionCoprocessorEnvironment
;
31 import org
.apache
.hadoop
.hbase
.coprocessor
.RegionObserver
;
32 import org
.apache
.hadoop
.hbase
.util
.Bytes
;
33 import org
.apache
.hadoop
.hbase
.util
.Threads
;
34 import org
.apache
.hadoop
.hbase
.wal
.WALEdit
;
35 import org
.junit
.AfterClass
;
36 import org
.junit
.BeforeClass
;
37 import org
.junit
.Rule
;
38 import org
.junit
.rules
.TestName
;
41 * Based class for testing timeout logic for {@link ConnectionImplementation}.
43 public abstract class AbstractTestCITimeout
{
45 protected static final HBaseTestingUtility TEST_UTIL
= new HBaseTestingUtility();
47 protected static final byte[] FAM_NAM
= Bytes
.toBytes("f");
50 public final TestName name
= new TestName();
53 * This copro sleeps 20 second. The first call it fails. The second time, it works.
55 public static class SleepAndFailFirstTime
implements RegionCoprocessor
, RegionObserver
{
56 static final AtomicLong ct
= new AtomicLong(0);
57 static final String SLEEP_TIME_CONF_KEY
= "hbase.coprocessor.SleepAndFailFirstTime.sleepTime";
58 static final long DEFAULT_SLEEP_TIME
= 20000;
59 static final AtomicLong sleepTime
= new AtomicLong(DEFAULT_SLEEP_TIME
);
61 public SleepAndFailFirstTime() {
65 public Optional
<RegionObserver
> getRegionObserver() {
66 return Optional
.of(this);
70 public void postOpen(ObserverContext
<RegionCoprocessorEnvironment
> c
) {
71 RegionCoprocessorEnvironment env
= c
.getEnvironment();
72 Configuration conf
= env
.getConfiguration();
73 sleepTime
.set(conf
.getLong(SLEEP_TIME_CONF_KEY
, DEFAULT_SLEEP_TIME
));
77 public void preGetOp(final ObserverContext
<RegionCoprocessorEnvironment
> e
, final Get get
,
78 final List
<Cell
> results
) throws IOException
{
79 Threads
.sleep(sleepTime
.get());
80 if (ct
.incrementAndGet() == 1) {
81 throw new IOException("first call I fail");
86 public void prePut(final ObserverContext
<RegionCoprocessorEnvironment
> e
, final Put put
,
87 final WALEdit edit
, final Durability durability
) throws IOException
{
88 Threads
.sleep(sleepTime
.get());
89 if (ct
.incrementAndGet() == 1) {
90 throw new IOException("first call I fail");
95 public void preDelete(final ObserverContext
<RegionCoprocessorEnvironment
> e
,
96 final Delete delete
, final WALEdit edit
, final Durability durability
) throws IOException
{
97 Threads
.sleep(sleepTime
.get());
98 if (ct
.incrementAndGet() == 1) {
99 throw new IOException("first call I fail");
104 public Result
preIncrement(final ObserverContext
<RegionCoprocessorEnvironment
> e
,
105 final Increment increment
) throws IOException
{
106 Threads
.sleep(sleepTime
.get());
107 if (ct
.incrementAndGet() == 1) {
108 throw new IOException("first call I fail");
115 public static class SleepCoprocessor
implements RegionCoprocessor
, RegionObserver
{
116 public static final int SLEEP_TIME
= 5000;
119 public Optional
<RegionObserver
> getRegionObserver() {
120 return Optional
.of(this);
124 public void preGetOp(final ObserverContext
<RegionCoprocessorEnvironment
> e
, final Get get
,
125 final List
<Cell
> results
) throws IOException
{
126 Threads
.sleep(SLEEP_TIME
);
130 public void prePut(final ObserverContext
<RegionCoprocessorEnvironment
> e
, final Put put
,
131 final WALEdit edit
, final Durability durability
) throws IOException
{
132 Threads
.sleep(SLEEP_TIME
);
136 public Result
preIncrement(final ObserverContext
<RegionCoprocessorEnvironment
> e
,
137 final Increment increment
) throws IOException
{
138 Threads
.sleep(SLEEP_TIME
);
143 public void preDelete(final ObserverContext
<RegionCoprocessorEnvironment
> e
,
144 final Delete delete
, final WALEdit edit
, final Durability durability
) throws IOException
{
145 Threads
.sleep(SLEEP_TIME
);
150 public static void setUpBeforeClass() throws Exception
{
151 TEST_UTIL
.getConfiguration().setBoolean(HConstants
.STATUS_PUBLISHED
, true);
152 // Up the handlers; this test needs more than usual.
153 TEST_UTIL
.getConfiguration().setInt(HConstants
.REGION_SERVER_HIGH_PRIORITY_HANDLER_COUNT
, 10);
154 TEST_UTIL
.getConfiguration().setInt(HConstants
.HBASE_CLIENT_RETRIES_NUMBER
, 5);
155 TEST_UTIL
.getConfiguration().setInt(HConstants
.REGION_SERVER_HANDLER_COUNT
, 3);
156 TEST_UTIL
.startMiniCluster(2);
161 public static void tearDownAfterClass() throws Exception
{
162 TEST_UTIL
.shutdownMiniCluster();