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.junit.jupiter.api.Assertions.assertEquals; 021import static org.junit.jupiter.api.Assertions.assertTrue; 022import static org.junit.jupiter.api.Assertions.fail; 023import static org.mockito.Mockito.any; 024import static org.mockito.Mockito.doAnswer; 025import static org.mockito.Mockito.mock; 026import static org.mockito.Mockito.when; 027 028import java.io.ByteArrayOutputStream; 029import java.io.PrintStream; 030import org.apache.hadoop.conf.Configuration; 031import org.apache.hadoop.fs.Path; 032import org.apache.hadoop.hbase.HBaseTestingUtil; 033import org.apache.hadoop.hbase.client.Put; 034import org.apache.hadoop.hbase.client.Result; 035import org.apache.hadoop.hbase.io.ImmutableBytesWritable; 036import org.apache.hadoop.hbase.mapreduce.IndexBuilder.Map; 037import org.apache.hadoop.hbase.mapreduce.SampleUploader.Uploader; 038import org.apache.hadoop.hbase.testclassification.LargeTests; 039import org.apache.hadoop.hbase.testclassification.MapReduceTests; 040import org.apache.hadoop.hbase.util.Bytes; 041import org.apache.hadoop.hbase.util.LauncherSecurityManager; 042import org.apache.hadoop.io.LongWritable; 043import org.apache.hadoop.io.Text; 044import org.apache.hadoop.mapreduce.Job; 045import org.apache.hadoop.mapreduce.Mapper; 046import org.apache.hadoop.mapreduce.Mapper.Context; 047import org.apache.hadoop.mapreduce.lib.input.SequenceFileInputFormat; 048import org.junit.jupiter.api.Tag; 049import org.junit.jupiter.api.Test; 050import org.mockito.invocation.InvocationOnMock; 051import org.mockito.stubbing.Answer; 052 053@Tag(MapReduceTests.TAG) 054@Tag(LargeTests.TAG) 055public class TestMapReduceExamples { 056 057 private static HBaseTestingUtil util = new HBaseTestingUtil(); 058 059 /** 060 * Test SampleUploader from examples 061 */ 062 @SuppressWarnings("unchecked") 063 @Test 064 public void testSampleUploader() throws Exception { 065 Configuration configuration = new Configuration(); 066 Uploader uploader = new Uploader(); 067 Mapper<LongWritable, Text, ImmutableBytesWritable, Put>.Context ctx = mock(Context.class); 068 doAnswer(new Answer<Void>() { 069 070 @Override 071 public Void answer(InvocationOnMock invocation) throws Throwable { 072 ImmutableBytesWritable writer = (ImmutableBytesWritable) invocation.getArgument(0); 073 Put put = (Put) invocation.getArgument(1); 074 assertEquals("row", Bytes.toString(writer.get())); 075 assertEquals("row", Bytes.toString(put.getRow())); 076 return null; 077 } 078 }).when(ctx).write(any(), any()); 079 080 uploader.map(null, new Text("row,family,qualifier,value"), ctx); 081 082 Path dir = util.getDataTestDirOnTestFS("testSampleUploader"); 083 084 String[] args = { dir.toString(), "simpleTable" }; 085 Job job = SampleUploader.configureJob(configuration, args); 086 assertEquals(SequenceFileInputFormat.class, job.getInputFormatClass()); 087 } 088 089 /** 090 * Test main method of SampleUploader. 091 */ 092 @Test 093 public void testMainSampleUploader() throws Exception { 094 PrintStream oldPrintStream = System.err; 095 SecurityManager SECURITY_MANAGER = System.getSecurityManager(); 096 LauncherSecurityManager newSecurityManager = new LauncherSecurityManager(); 097 System.setSecurityManager(newSecurityManager); 098 ByteArrayOutputStream data = new ByteArrayOutputStream(); 099 String[] args = {}; 100 System.setErr(new PrintStream(data)); 101 try { 102 System.setErr(new PrintStream(data)); 103 104 try { 105 SampleUploader.main(args); 106 fail("should be SecurityException"); 107 } catch (SecurityException e) { 108 assertEquals(-1, newSecurityManager.getExitCode()); 109 assertTrue(data.toString().contains("Wrong number of arguments:")); 110 assertTrue(data.toString().contains("Usage: SampleUploader <input> <tablename>")); 111 } 112 113 } finally { 114 System.setErr(oldPrintStream); 115 System.setSecurityManager(SECURITY_MANAGER); 116 } 117 118 } 119 120 /** 121 * Test IndexBuilder from examples 122 */ 123 @SuppressWarnings("unchecked") 124 @Test 125 public void testIndexBuilder() throws Exception { 126 Configuration configuration = new Configuration(); 127 String[] args = { "tableName", "columnFamily", "column1", "column2" }; 128 IndexBuilder.configureJob(configuration, args); 129 assertEquals("tableName", configuration.get("index.tablename")); 130 assertEquals("tableName", configuration.get(TableInputFormat.INPUT_TABLE)); 131 assertEquals("column1,column2", configuration.get("index.fields")); 132 133 Map map = new Map(); 134 ImmutableBytesWritable rowKey = new ImmutableBytesWritable(Bytes.toBytes("test")); 135 Mapper<ImmutableBytesWritable, Result, ImmutableBytesWritable, Put>.Context ctx = 136 mock(Context.class); 137 when(ctx.getConfiguration()).thenReturn(configuration); 138 doAnswer(new Answer<Void>() { 139 140 @Override 141 public Void answer(InvocationOnMock invocation) throws Throwable { 142 ImmutableBytesWritable writer = (ImmutableBytesWritable) invocation.getArgument(0); 143 Put put = (Put) invocation.getArgument(1); 144 assertEquals("tableName-column1", Bytes.toString(writer.get())); 145 assertEquals("test", Bytes.toString(put.getRow())); 146 return null; 147 } 148 }).when(ctx).write(any(), any()); 149 Result result = mock(Result.class); 150 when(result.getValue(Bytes.toBytes("columnFamily"), Bytes.toBytes("column1"))) 151 .thenReturn(Bytes.toBytes("test")); 152 map.setup(ctx); 153 map.map(rowKey, result, ctx); 154 } 155 156 /** 157 * Test main method of IndexBuilder 158 */ 159 @Test 160 public void testMainIndexBuilder() throws Exception { 161 PrintStream oldPrintStream = System.err; 162 LauncherSecurityManager newSecurityManager = new LauncherSecurityManager(); 163 System.setSecurityManager(newSecurityManager); 164 ByteArrayOutputStream data = new ByteArrayOutputStream(); 165 String[] args = {}; 166 System.setErr(new PrintStream(data)); 167 try { 168 System.setErr(new PrintStream(data)); 169 try { 170 IndexBuilder.main(args); 171 fail("should be SecurityException"); 172 } catch (SecurityException e) { 173 assertEquals(-1, newSecurityManager.getExitCode()); 174 assertTrue(data.toString().contains("arguments supplied, required: 3")); 175 assertTrue(data.toString() 176 .contains("Usage: IndexBuilder <TABLE_NAME> <COLUMN_FAMILY> <ATTR> [<ATTR> ...]")); 177 } 178 } finally { 179 System.setErr(oldPrintStream); 180 } 181 } 182}