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.mapreduce;
019
020import static org.apache.hadoop.hbase.client.Scan.SCAN_ATTRIBUTES_TABLE_NAME;
021
022import java.io.IOException;
023import java.util.ArrayList;
024import java.util.List;
025import java.util.Map;
026import java.util.TreeMap;
027import java.util.concurrent.ExecutorService;
028import java.util.concurrent.atomic.AtomicInteger;
029import org.apache.hadoop.conf.Configuration;
030import org.apache.hadoop.hbase.HBaseClassTestRule;
031import org.apache.hadoop.hbase.HBaseConfiguration;
032import org.apache.hadoop.hbase.HConstants;
033import org.apache.hadoop.hbase.HRegionLocation;
034import org.apache.hadoop.hbase.ServerName;
035import org.apache.hadoop.hbase.TableName;
036import org.apache.hadoop.hbase.client.Admin;
037import org.apache.hadoop.hbase.client.AsyncConnection;
038import org.apache.hadoop.hbase.client.BufferedMutator;
039import org.apache.hadoop.hbase.client.BufferedMutatorParams;
040import org.apache.hadoop.hbase.client.Connection;
041import org.apache.hadoop.hbase.client.ConnectionUtils;
042import org.apache.hadoop.hbase.client.RegionInfoBuilder;
043import org.apache.hadoop.hbase.client.RegionLocator;
044import org.apache.hadoop.hbase.client.Result;
045import org.apache.hadoop.hbase.client.Scan;
046import org.apache.hadoop.hbase.client.Table;
047import org.apache.hadoop.hbase.client.TableBuilder;
048import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
049import org.apache.hadoop.hbase.security.User;
050import org.apache.hadoop.hbase.testclassification.SmallTests;
051import org.apache.hadoop.hbase.util.Bytes;
052import org.apache.hadoop.hbase.util.Pair;
053import org.apache.hadoop.mapreduce.InputSplit;
054import org.apache.hadoop.mapreduce.JobContext;
055import org.apache.hadoop.mapreduce.RecordReader;
056import org.apache.hadoop.mapreduce.TaskAttemptContext;
057import org.junit.Assert;
058import org.junit.ClassRule;
059import org.junit.Rule;
060import org.junit.Test;
061import org.junit.experimental.categories.Category;
062import org.junit.rules.TestName;
063import org.mockito.Mockito;
064import org.mockito.invocation.InvocationOnMock;
065import org.mockito.stubbing.Answer;
066
067/**
068 * Tests of MultiTableInputFormatBase.
069 */
070@Category({ SmallTests.class })
071public class TestMultiTableInputFormatBase {
072
073  @ClassRule
074  public static final HBaseClassTestRule CLASS_RULE =
075    HBaseClassTestRule.forClass(TestMultiTableInputFormatBase.class);
076
077  @Rule
078  public final TestName name = new TestName();
079
080  /**
081   * Test getSplits only puts up one Connection. In past it has put up many Connections. Each
082   * Connection setup comes with a fresh new cache so we have to do fresh hit on hbase:meta. Should
083   * only do one Connection when doing getSplits even if a MultiTableInputFormat.
084   */
085  @Test
086  public void testMRSplitsConnectionCount() throws IOException {
087    // Make instance of MTIFB.
088    MultiTableInputFormatBase mtif = new MultiTableInputFormatBase() {
089      @Override
090      public RecordReader<ImmutableBytesWritable, Result> createRecordReader(InputSplit split,
091        TaskAttemptContext context) throws IOException, InterruptedException {
092        return super.createRecordReader(split, context);
093      }
094    };
095    // Pass it a mocked JobContext. Make the JC return our Configuration.
096    // Load the Configuration so it returns our special Connection so we can interpolate
097    // canned responses.
098    JobContext mockedJobContext = Mockito.mock(JobContext.class);
099    Configuration c = HBaseConfiguration.create();
100    c.set(ConnectionUtils.HBASE_CLIENT_CONNECTION_IMPL, MRSplitsConnection.class.getName());
101    Mockito.when(mockedJobContext.getConfiguration()).thenReturn(c);
102    // Invent a bunch of scans. Have each Scan go against a different table so a good spread.
103    List<Scan> scans = new ArrayList<>();
104    for (int i = 0; i < 10; i++) {
105      Scan scan = new Scan();
106      String tableName = this.name.getMethodName() + i;
107      scan.setAttribute(SCAN_ATTRIBUTES_TABLE_NAME, Bytes.toBytes(tableName));
108      scans.add(scan);
109    }
110    mtif.setScans(scans);
111    // Get splits. Assert that that more than one.
112    List<InputSplit> splits = mtif.getSplits(mockedJobContext);
113    Assert.assertTrue(splits.size() > 0);
114    // Assert only one Connection was made (see the static counter we have in the mocked
115    // Connection MRSplitsConnection Constructor.
116    Assert.assertEquals(1, MRSplitsConnection.creations.get());
117  }
118
119  /**
120   * Connection to use above in Test.
121   */
122  public static class MRSplitsConnection implements Connection {
123    private final Configuration configuration;
124    static final AtomicInteger creations = new AtomicInteger(0);
125
126    MRSplitsConnection(Configuration conf, ExecutorService pool, User user,
127      Map<String, byte[]> connectionAttributes) throws IOException {
128      this.configuration = conf;
129      creations.incrementAndGet();
130    }
131
132    @Override
133    public void abort(String why, Throwable e) {
134
135    }
136
137    @Override
138    public boolean isAborted() {
139      return false;
140    }
141
142    @Override
143    public Configuration getConfiguration() {
144      return this.configuration;
145    }
146
147    @Override
148    public BufferedMutator getBufferedMutator(TableName tableName) throws IOException {
149      return null;
150    }
151
152    @Override
153    public BufferedMutator getBufferedMutator(BufferedMutatorParams params) throws IOException {
154      return null;
155    }
156
157    @Override
158    public RegionLocator getRegionLocator(final TableName tableName) throws IOException {
159      // Make up array of start keys. We start off w/ empty byte array.
160      final byte[][] startKeys = new byte[][] { HConstants.EMPTY_BYTE_ARRAY, Bytes.toBytes("aaaa"),
161        Bytes.toBytes("bbb"), Bytes.toBytes("ccc"), Bytes.toBytes("ddd"), Bytes.toBytes("eee"),
162        Bytes.toBytes("fff"), Bytes.toBytes("ggg"), Bytes.toBytes("hhh"), Bytes.toBytes("iii"),
163        Bytes.toBytes("lll"), Bytes.toBytes("mmm"), Bytes.toBytes("nnn"), Bytes.toBytes("ooo"),
164        Bytes.toBytes("ppp"), Bytes.toBytes("qqq"), Bytes.toBytes("rrr"), Bytes.toBytes("sss"),
165        Bytes.toBytes("ttt"), Bytes.toBytes("uuu"), Bytes.toBytes("vvv"), Bytes.toBytes("zzz") };
166      // Make an array of end keys. We end with the empty byte array.
167      final byte[][] endKeys =
168        new byte[][] { Bytes.toBytes("aaaa"), Bytes.toBytes("bbb"), Bytes.toBytes("ccc"),
169          Bytes.toBytes("ddd"), Bytes.toBytes("eee"), Bytes.toBytes("fff"), Bytes.toBytes("ggg"),
170          Bytes.toBytes("hhh"), Bytes.toBytes("iii"), Bytes.toBytes("lll"), Bytes.toBytes("mmm"),
171          Bytes.toBytes("nnn"), Bytes.toBytes("ooo"), Bytes.toBytes("ppp"), Bytes.toBytes("qqq"),
172          Bytes.toBytes("rrr"), Bytes.toBytes("sss"), Bytes.toBytes("ttt"), Bytes.toBytes("uuu"),
173          Bytes.toBytes("vvv"), Bytes.toBytes("zzz"), HConstants.EMPTY_BYTE_ARRAY };
174      // Now make a map of start keys to HRegionLocations. Let the server namber derive from
175      // the start key.
176      final Map<byte[], HRegionLocation> map =
177        new TreeMap<byte[], HRegionLocation>(Bytes.BYTES_COMPARATOR);
178      for (byte[] startKey : startKeys) {
179        HRegionLocation hrl =
180          new HRegionLocation(RegionInfoBuilder.newBuilder(tableName).setStartKey(startKey).build(),
181            ServerName.valueOf(Bytes.toString(startKey), 0, 0));
182        map.put(startKey, hrl);
183      }
184      // Get a list of the locations.
185      final List<HRegionLocation> locations = new ArrayList<HRegionLocation>(map.values());
186      // Now make a RegionLocator mock backed by the abpve map and list of locations.
187      RegionLocator mockedRegionLocator = Mockito.mock(RegionLocator.class);
188      Mockito
189        .when(
190          mockedRegionLocator.getRegionLocation(Mockito.any(byte[].class), Mockito.anyBoolean()))
191        .thenAnswer(new Answer<HRegionLocation>() {
192          @Override
193          public HRegionLocation answer(InvocationOnMock invocationOnMock) throws Throwable {
194            Object[] args = invocationOnMock.getArguments();
195            byte[] key = (byte[]) args[0];
196            return map.get(key);
197          }
198        });
199      Mockito.when(mockedRegionLocator.getAllRegionLocations()).thenReturn(locations);
200      Mockito.when(mockedRegionLocator.getStartEndKeys())
201        .thenReturn(new Pair<byte[][], byte[][]>(startKeys, endKeys));
202      Mockito.when(mockedRegionLocator.getName()).thenReturn(tableName);
203      return mockedRegionLocator;
204    }
205
206    @Override
207    public Admin getAdmin() throws IOException {
208      Admin admin = Mockito.mock(Admin.class);
209      Mockito.when(admin.getConfiguration()).thenReturn(getConfiguration());
210      return admin;
211    }
212
213    @Override
214    public Table getTable(TableName tableName) throws IOException {
215      Table table = Mockito.mock(Table.class);
216      Mockito.when(table.getName()).thenReturn(tableName);
217      return table;
218    }
219
220    @Override
221    public void close() throws IOException {
222
223    }
224
225    @Override
226    public boolean isClosed() {
227      return false;
228    }
229
230    @Override
231    public TableBuilder getTableBuilder(TableName tableName, ExecutorService pool) {
232      return Mockito.mock(TableBuilder.class);
233    }
234
235    @Override
236    public void clearRegionLocationCache() {
237    }
238
239    @Override
240    public AsyncConnection toAsyncConnection() {
241      return null;
242    }
243
244    @Override
245    public String getClusterId() {
246      return null;
247    }
248
249  }
250}