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.List;
022import java.util.Optional;
023import java.util.concurrent.atomic.AtomicLong;
024import org.apache.hadoop.conf.Configuration;
025import org.apache.hadoop.hbase.Cell;
026import org.apache.hadoop.hbase.HBaseTestingUtil;
027import org.apache.hadoop.hbase.HConstants;
028import org.apache.hadoop.hbase.coprocessor.ObserverContext;
029import org.apache.hadoop.hbase.coprocessor.RegionCoprocessor;
030import org.apache.hadoop.hbase.coprocessor.RegionCoprocessorEnvironment;
031import org.apache.hadoop.hbase.coprocessor.RegionObserver;
032import org.apache.hadoop.hbase.util.Bytes;
033import org.apache.hadoop.hbase.util.Threads;
034import org.apache.hadoop.hbase.wal.WALEdit;
035import org.junit.AfterClass;
036import org.junit.BeforeClass;
037import org.junit.Rule;
038import org.junit.rules.TestName;
039
040/**
041 * Based class for testing timeout logic.
042 */
043public abstract class AbstractTestCITimeout {
044
045  protected static final HBaseTestingUtil TEST_UTIL = new HBaseTestingUtil();
046
047  protected static final byte[] FAM_NAM = Bytes.toBytes("f");
048
049  @Rule
050  public final TestName name = new TestName();
051
052  /**
053   * This copro sleeps 20 second. The first call it fails. The second time, it works.
054   */
055  public static class SleepAndFailFirstTime implements RegionCoprocessor, RegionObserver {
056    static final AtomicLong ct = new AtomicLong(0);
057    static final String SLEEP_TIME_CONF_KEY = "hbase.coprocessor.SleepAndFailFirstTime.sleepTime";
058    static final long DEFAULT_SLEEP_TIME = 20000;
059    static final AtomicLong sleepTime = new AtomicLong(DEFAULT_SLEEP_TIME);
060
061    public SleepAndFailFirstTime() {
062    }
063
064    @Override
065    public Optional<RegionObserver> getRegionObserver() {
066      return Optional.of(this);
067    }
068
069    @Override
070    public void postOpen(ObserverContext<? extends RegionCoprocessorEnvironment> c) {
071      RegionCoprocessorEnvironment env = c.getEnvironment();
072      Configuration conf = env.getConfiguration();
073      sleepTime.set(conf.getLong(SLEEP_TIME_CONF_KEY, DEFAULT_SLEEP_TIME));
074    }
075
076    @Override
077    public void preGetOp(final ObserverContext<? extends RegionCoprocessorEnvironment> e,
078      final Get get, final List<Cell> results) throws IOException {
079      Threads.sleep(sleepTime.get());
080      if (ct.incrementAndGet() == 1) {
081        throw new IOException("first call I fail");
082      }
083    }
084
085    @Override
086    public void prePut(final ObserverContext<? extends RegionCoprocessorEnvironment> e,
087      final Put put, final WALEdit edit, final Durability durability) throws IOException {
088      Threads.sleep(sleepTime.get());
089      if (ct.incrementAndGet() == 1) {
090        throw new IOException("first call I fail");
091      }
092    }
093
094    @Override
095    public void preDelete(final ObserverContext<? extends RegionCoprocessorEnvironment> e,
096      final Delete delete, final WALEdit edit, final Durability durability) throws IOException {
097      Threads.sleep(sleepTime.get());
098      if (ct.incrementAndGet() == 1) {
099        throw new IOException("first call I fail");
100      }
101    }
102
103    @Override
104    public Result preIncrement(final ObserverContext<? extends 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");
109      }
110      return null;
111    }
112
113  }
114
115  public static class SleepCoprocessor implements RegionCoprocessor, RegionObserver {
116    public static final int SLEEP_TIME = 5000;
117
118    @Override
119    public Optional<RegionObserver> getRegionObserver() {
120      return Optional.of(this);
121    }
122
123    @Override
124    public void preGetOp(final ObserverContext<? extends RegionCoprocessorEnvironment> e,
125      final Get get, final List<Cell> results) throws IOException {
126      Threads.sleep(SLEEP_TIME);
127    }
128
129    @Override
130    public void prePut(final ObserverContext<? extends RegionCoprocessorEnvironment> e,
131      final Put put, final WALEdit edit, final Durability durability) throws IOException {
132      Threads.sleep(SLEEP_TIME);
133    }
134
135    @Override
136    public Result preIncrement(final ObserverContext<? extends RegionCoprocessorEnvironment> e,
137      final Increment increment) throws IOException {
138      Threads.sleep(SLEEP_TIME);
139      return null;
140    }
141
142    @Override
143    public void preDelete(final ObserverContext<? extends RegionCoprocessorEnvironment> e,
144      final Delete delete, final WALEdit edit, final Durability durability) throws IOException {
145      Threads.sleep(SLEEP_TIME);
146    }
147  }
148
149  @BeforeClass
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);
157
158  }
159
160  @AfterClass
161  public static void tearDownAfterClass() throws Exception {
162    TEST_UTIL.shutdownMiniCluster();
163  }
164}