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.test;
019
020import static org.junit.Assert.assertEquals;
021import static org.junit.Assert.assertTrue;
022
023import java.io.IOException;
024import java.security.PrivilegedExceptionAction;
025import java.util.List;
026import java.util.Random;
027import java.util.concurrent.ThreadLocalRandom;
028import org.apache.hadoop.conf.Configuration;
029import org.apache.hadoop.fs.Path;
030import org.apache.hadoop.hbase.HBaseConfiguration;
031import org.apache.hadoop.hbase.HColumnDescriptor;
032import org.apache.hadoop.hbase.HConstants;
033import org.apache.hadoop.hbase.HTableDescriptor;
034import org.apache.hadoop.hbase.IntegrationTestingUtility;
035import org.apache.hadoop.hbase.client.Admin;
036import org.apache.hadoop.hbase.client.Connection;
037import org.apache.hadoop.hbase.client.ConnectionFactory;
038import org.apache.hadoop.hbase.client.Put;
039import org.apache.hadoop.hbase.client.Result;
040import org.apache.hadoop.hbase.client.Scan;
041import org.apache.hadoop.hbase.client.ScannerCallable;
042import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
043import org.apache.hadoop.hbase.mapreduce.TableMapReduceUtil;
044import org.apache.hadoop.hbase.mapreduce.TableMapper;
045import org.apache.hadoop.hbase.mapreduce.TableRecordReaderImpl;
046import org.apache.hadoop.hbase.security.User;
047import org.apache.hadoop.hbase.security.visibility.Authorizations;
048import org.apache.hadoop.hbase.security.visibility.CellVisibility;
049import org.apache.hadoop.hbase.security.visibility.VisibilityClient;
050import org.apache.hadoop.hbase.security.visibility.VisibilityTestUtil;
051import org.apache.hadoop.hbase.testclassification.IntegrationTests;
052import org.apache.hadoop.hbase.util.AbstractHBaseTool;
053import org.apache.hadoop.hbase.util.Bytes;
054import org.apache.hadoop.io.BytesWritable;
055import org.apache.hadoop.io.NullWritable;
056import org.apache.hadoop.mapreduce.Counter;
057import org.apache.hadoop.mapreduce.Job;
058import org.apache.hadoop.mapreduce.lib.output.FileOutputFormat;
059import org.apache.hadoop.util.ToolRunner;
060import org.junit.experimental.categories.Category;
061
062import org.apache.hbase.thirdparty.com.google.common.base.Splitter;
063import org.apache.hbase.thirdparty.com.google.common.collect.Iterables;
064import org.apache.hbase.thirdparty.org.apache.commons.cli.CommandLine;
065
066/**
067 * A large test which loads a lot of data with cell visibility, and verifies the data. Test adds 2
068 * users with different sets of visibility labels authenticated for them. Every row (so cells in
069 * that) added with visibility expressions. In load step, 200 map tasks are launched, which in turn
070 * write loadmapper.num_to_write (default 100K) rows to an hbase table. Rows are written in blocks,
071 * for a total of 100 blocks. Verify step scans the table as both users with Authorizations. This
072 * step asserts that user can see only those rows (and so cells) with visibility for which they have
073 * label auth. This class can be run as a unit test, as an integration test, or from the command
074 * line. Originally taken from Apache Bigtop. Issue user names as comma seperated list. ./hbase
075 * IntegrationTestWithCellVisibilityLoadAndVerify -u usera,userb
076 */
077@Category(IntegrationTests.class)
078public class IntegrationTestWithCellVisibilityLoadAndVerify extends IntegrationTestLoadAndVerify {
079  private static final String ERROR_STR =
080    "Two user names are to be specified seperated by a ',' like 'usera,userb'";
081  private static final char NOT = '!';
082  private static final char OR = '|';
083  private static final char AND = '&';
084  private static final String TEST_NAME = "IntegrationTestCellVisibilityLoadAndVerify";
085  private static final String CONFIDENTIAL = "confidential";
086  private static final String TOPSECRET = "topsecret";
087  private static final String SECRET = "secret";
088  private static final String PUBLIC = "public";
089  private static final String PRIVATE = "private";
090  private static final String[] LABELS = { CONFIDENTIAL, TOPSECRET, SECRET, PRIVATE, PUBLIC };
091  private static final String[] VISIBILITY_EXPS =
092    { CONFIDENTIAL + AND + TOPSECRET + AND + PRIVATE, CONFIDENTIAL + OR + TOPSECRET, PUBLIC,
093      '(' + SECRET + OR + PRIVATE + ')' + AND + NOT + CONFIDENTIAL };
094  private static final int VISIBILITY_EXPS_COUNT = VISIBILITY_EXPS.length;
095  private static final byte[] TEST_FAMILY = Bytes.toBytes("f1");
096  private static final byte[] TEST_QUALIFIER = Bytes.toBytes("q1");
097  private static final String NUM_TO_WRITE_KEY = "loadmapper.num_to_write";
098  private static final long NUM_TO_WRITE_DEFAULT = 100 * 1000;
099  private static final int SCANNER_CACHING = 500;
100  private static String USER_OPT = "users";
101  private static String userNames = "user1,user2";
102
103  private long numRowsLoadedWithExp1, numRowsLoadedWithExp2, numRowsLoadWithExp3,
104      numRowsLoadWithExp4;
105  private long numRowsReadWithExp1, numRowsReadWithExp2, numRowsReadWithExp3, numRowsReadWithExp4;
106
107  private static User USER1, USER2;
108
109  private enum Counters {
110    ROWS_VIS_EXP_1,
111    ROWS_VIS_EXP_2,
112    ROWS_VIS_EXP_3,
113    ROWS_VIS_EXP_4;
114  }
115
116  @Override
117  public void setUpCluster() throws Exception {
118    util = getTestingUtil(null);
119    Configuration conf = util.getConfiguration();
120    VisibilityTestUtil.enableVisiblityLabels(conf);
121    conf.set("hbase.superuser", User.getCurrent().getName());
122    conf.setBoolean("dfs.permissions", false);
123    super.setUpCluster();
124    List<String> users = Splitter.on(',').splitToList(userNames);
125    if (users.size() != 2) {
126      System.err.println(ERROR_STR);
127      throw new IOException(ERROR_STR);
128    }
129    String user1 = Iterables.get(users, 0);
130    String user2 = Iterables.get(users, 1);
131    System.out.println(userNames + " " + user1 + " " + user2);
132    USER1 = User.createUserForTesting(conf, user1, new String[] {});
133    USER2 = User.createUserForTesting(conf, user2, new String[] {});
134    addLabelsAndAuths();
135  }
136
137  @Override
138  protected void addOptions() {
139    super.addOptions();
140    addOptWithArg("u", USER_OPT, "User names to be passed");
141  }
142
143  private void addLabelsAndAuths() throws Exception {
144    try {
145      VisibilityClient.addLabels(util.getConnection(), LABELS);
146      VisibilityClient.setAuths(util.getConnection(),
147        new String[] { CONFIDENTIAL, TOPSECRET, SECRET, PRIVATE }, USER1.getName());
148      VisibilityClient.setAuths(util.getConnection(), new String[] { PUBLIC }, USER2.getName());
149    } catch (Throwable t) {
150      throw new IOException(t);
151    }
152  }
153
154  public static class LoadWithCellVisibilityMapper extends LoadMapper {
155    private Counter rowsExp1, rowsExp2, rowsExp3, rowsexp4;
156
157    @Override
158    public void setup(Context context) throws IOException {
159      super.setup(context);
160      rowsExp1 = context.getCounter(Counters.ROWS_VIS_EXP_1);
161      rowsExp2 = context.getCounter(Counters.ROWS_VIS_EXP_2);
162      rowsExp3 = context.getCounter(Counters.ROWS_VIS_EXP_3);
163      rowsexp4 = context.getCounter(Counters.ROWS_VIS_EXP_4);
164    }
165
166    @Override
167    protected void map(NullWritable key, NullWritable value, Context context)
168      throws IOException, InterruptedException {
169      String suffix = "/" + shortTaskId;
170      int BLOCK_SIZE = (int) (recordsToWrite / 100);
171      Random rand = ThreadLocalRandom.current();
172      for (long i = 0; i < recordsToWrite;) {
173        for (long idx = 0; idx < BLOCK_SIZE && i < recordsToWrite; idx++, i++) {
174          int expIdx = rand.nextInt(VISIBILITY_EXPS_COUNT);
175          String exp = VISIBILITY_EXPS[expIdx];
176          byte[] row = Bytes.add(Bytes.toBytes(i), Bytes.toBytes(suffix), Bytes.toBytes(exp));
177          Put p = new Put(row);
178          p.addColumn(TEST_FAMILY, TEST_QUALIFIER, HConstants.EMPTY_BYTE_ARRAY);
179          p.setCellVisibility(new CellVisibility(exp));
180          getCounter(expIdx).increment(1);
181          mutator.mutate(p);
182
183          if (i % 100 == 0) {
184            context.setStatus("Written " + i + "/" + recordsToWrite + " records");
185            context.progress();
186          }
187        }
188        // End of block, flush all of them before we start writing anything
189        // pointing to these!
190        mutator.flush();
191      }
192    }
193
194    private Counter getCounter(int idx) {
195      switch (idx) {
196        case 0:
197          return rowsExp1;
198        case 1:
199          return rowsExp2;
200        case 2:
201          return rowsExp3;
202        case 3:
203          return rowsexp4;
204        default:
205          return null;
206      }
207    }
208  }
209
210  public static class VerifyMapper extends TableMapper<BytesWritable, BytesWritable> {
211    private Counter rowsExp1, rowsExp2, rowsExp3, rowsExp4;
212
213    @Override
214    public void setup(Context context) throws IOException {
215      rowsExp1 = context.getCounter(Counters.ROWS_VIS_EXP_1);
216      rowsExp2 = context.getCounter(Counters.ROWS_VIS_EXP_2);
217      rowsExp3 = context.getCounter(Counters.ROWS_VIS_EXP_3);
218      rowsExp4 = context.getCounter(Counters.ROWS_VIS_EXP_4);
219    }
220
221    @Override
222    protected void map(ImmutableBytesWritable key, Result value, Context context)
223      throws IOException, InterruptedException {
224      byte[] row = value.getRow();
225      Counter c = getCounter(row);
226      c.increment(1);
227    }
228
229    private Counter getCounter(byte[] row) {
230      Counter c = null;
231      if (Bytes.indexOf(row, Bytes.toBytes(VISIBILITY_EXPS[0])) != -1) {
232        c = rowsExp1;
233      } else if (Bytes.indexOf(row, Bytes.toBytes(VISIBILITY_EXPS[1])) != -1) {
234        c = rowsExp2;
235      } else if (Bytes.indexOf(row, Bytes.toBytes(VISIBILITY_EXPS[2])) != -1) {
236        c = rowsExp3;
237      } else if (Bytes.indexOf(row, Bytes.toBytes(VISIBILITY_EXPS[3])) != -1) {
238        c = rowsExp4;
239      }
240      return c;
241    }
242  }
243
244  @Override
245  protected Job doLoad(Configuration conf, HTableDescriptor htd) throws Exception {
246    Job job = super.doLoad(conf, htd);
247    this.numRowsLoadedWithExp1 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_1).getValue();
248    this.numRowsLoadedWithExp2 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_2).getValue();
249    this.numRowsLoadWithExp3 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_3).getValue();
250    this.numRowsLoadWithExp4 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_4).getValue();
251    System.out.println("Rows loaded with cell visibility " + VISIBILITY_EXPS[0] + " : "
252      + this.numRowsLoadedWithExp1);
253    System.out.println("Rows loaded with cell visibility " + VISIBILITY_EXPS[1] + " : "
254      + this.numRowsLoadedWithExp2);
255    System.out.println(
256      "Rows loaded with cell visibility " + VISIBILITY_EXPS[2] + " : " + this.numRowsLoadWithExp3);
257    System.out.println(
258      "Rows loaded with cell visibility " + VISIBILITY_EXPS[3] + " : " + this.numRowsLoadWithExp4);
259    return job;
260  }
261
262  @Override
263  protected void setMapperClass(Job job) {
264    job.setMapperClass(LoadWithCellVisibilityMapper.class);
265  }
266
267  @Override
268  protected void doVerify(final Configuration conf, final HTableDescriptor htd) throws Exception {
269    System.out.println(String.format("Verifying for auths %s, %s, %s, %s", CONFIDENTIAL, TOPSECRET,
270      SECRET, PRIVATE));
271    PrivilegedExceptionAction<Job> scanAction = new PrivilegedExceptionAction<Job>() {
272      @Override
273      public Job run() throws Exception {
274        return doVerify(conf, htd, CONFIDENTIAL, TOPSECRET, SECRET, PRIVATE);
275      }
276    };
277    Job job = USER1.runAs(scanAction);
278    this.numRowsReadWithExp1 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_1).getValue();
279    this.numRowsReadWithExp2 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_2).getValue();
280    this.numRowsReadWithExp3 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_3).getValue();
281    this.numRowsReadWithExp4 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_4).getValue();
282    assertEquals(this.numRowsLoadedWithExp1, this.numRowsReadWithExp1);
283    assertEquals(this.numRowsLoadedWithExp2, this.numRowsReadWithExp2);
284    assertEquals(0, this.numRowsReadWithExp3);
285    assertEquals(0, this.numRowsReadWithExp4);
286
287    // PUBLIC label auth is not provided for user1 user.
288    System.out.println(String.format("Verifying for auths %s, %s", PRIVATE, PUBLIC));
289    scanAction = new PrivilegedExceptionAction<Job>() {
290      @Override
291      public Job run() throws Exception {
292        return doVerify(conf, htd, PRIVATE, PUBLIC);
293      }
294    };
295    job = USER1.runAs(scanAction);
296    this.numRowsReadWithExp1 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_1).getValue();
297    this.numRowsReadWithExp2 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_2).getValue();
298    this.numRowsReadWithExp3 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_3).getValue();
299    this.numRowsReadWithExp4 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_4).getValue();
300    assertEquals(0, this.numRowsReadWithExp1);
301    assertEquals(0, this.numRowsReadWithExp2);
302    assertEquals(0, this.numRowsReadWithExp3);
303    assertEquals(this.numRowsLoadWithExp4, this.numRowsReadWithExp4);
304
305    // Normal user only having PUBLIC label auth and can view only those cells.
306    System.out.println(String.format("Verifying for auths %s, %s", PRIVATE, PUBLIC));
307    scanAction = new PrivilegedExceptionAction<Job>() {
308      @Override
309      public Job run() throws Exception {
310        return doVerify(conf, htd, PRIVATE, PUBLIC);
311      }
312    };
313    job = USER2.runAs(scanAction);
314    this.numRowsReadWithExp1 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_1).getValue();
315    this.numRowsReadWithExp2 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_2).getValue();
316    this.numRowsReadWithExp3 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_3).getValue();
317    this.numRowsReadWithExp4 = job.getCounters().findCounter(Counters.ROWS_VIS_EXP_4).getValue();
318    assertEquals(0, this.numRowsReadWithExp1);
319    assertEquals(0, this.numRowsReadWithExp2);
320    assertEquals(this.numRowsLoadWithExp3, this.numRowsReadWithExp3);
321    assertEquals(0, this.numRowsReadWithExp4);
322  }
323
324  private Job doVerify(Configuration conf, HTableDescriptor htd, String... auths)
325    throws IOException, InterruptedException, ClassNotFoundException {
326    Path outputDir = getTestDir(TEST_NAME, "verify-output");
327    Job job = new Job(conf);
328    job.setJarByClass(this.getClass());
329    job.setJobName(TEST_NAME + " Verification for " + htd.getTableName());
330    setJobScannerConf(job);
331    Scan scan = new Scan();
332    scan.setAuthorizations(new Authorizations(auths));
333    TableMapReduceUtil.initTableMapperJob(htd.getTableName().getNameAsString(), scan,
334      VerifyMapper.class, NullWritable.class, NullWritable.class, job);
335    TableMapReduceUtil.addDependencyJarsForClasses(job.getConfiguration(), AbstractHBaseTool.class);
336    int scannerCaching = conf.getInt("verify.scannercaching", SCANNER_CACHING);
337    TableMapReduceUtil.setScannerCaching(job, scannerCaching);
338    job.setNumReduceTasks(0);
339    FileOutputFormat.setOutputPath(job, outputDir);
340    assertTrue(job.waitForCompletion(true));
341    return job;
342  }
343
344  private static void setJobScannerConf(Job job) {
345    job.getConfiguration().setBoolean(ScannerCallable.LOG_SCANNER_ACTIVITY, true);
346    long lpr = job.getConfiguration().getLong(NUM_TO_WRITE_KEY, NUM_TO_WRITE_DEFAULT) / 100;
347    job.getConfiguration().setInt(TableRecordReaderImpl.LOG_PER_ROW_COUNT, (int) lpr);
348  }
349
350  @Override
351  public void printUsage() {
352    System.err.println(this.getClass().getSimpleName() + " -u usera,userb [-Doptions]");
353    System.err.println("  Loads a table with cell visibilities and verifies with Authorizations");
354    System.err.println("Options");
355    System.err
356      .println("  -Dloadmapper.table=<name>        Table to write/verify (default autogen)");
357    System.err.println("  -Dloadmapper.num_to_write=<n>    "
358      + "Number of rows per mapper (default 100,000 per mapper)");
359    System.err.println("  -Dloadmapper.numPresplits=<n>    "
360      + "Number of presplit regions to start with (default 40)");
361    System.err
362      .println("  -Dloadmapper.map.tasks=<n>       Number of map tasks for load (default 200)");
363    System.err.println("  -Dverify.scannercaching=<n>      "
364      + "Number hbase scanner caching rows to read (default 50)");
365  }
366
367  @Override
368  public int runTestFromCommandLine() throws Exception {
369    IntegrationTestingUtility.setUseDistributedCluster(getConf());
370    int numPresplits = getConf().getInt("loadmapper.numPresplits", 5);
371    // create HTableDescriptor for specified table
372    HTableDescriptor htd = new HTableDescriptor(getTablename());
373    htd.addFamily(new HColumnDescriptor(TEST_FAMILY));
374
375    try (Connection conn = ConnectionFactory.createConnection(getConf());
376      Admin admin = conn.getAdmin()) {
377      admin.createTable(htd, Bytes.toBytes(0L), Bytes.toBytes(-1L), numPresplits);
378    }
379    doLoad(getConf(), htd);
380    doVerify(getConf(), htd);
381    getTestingUtil(getConf()).deleteTable(getTablename());
382    return 0;
383  }
384
385  @Override
386  protected void processOptions(CommandLine cmd) {
387    List<String> args = cmd.getArgList();
388    if (args.size() > 0) {
389      printUsage();
390      throw new RuntimeException("No args expected.");
391    }
392    // We always want loadAndVerify action
393    args.add("loadAndVerify");
394    if (cmd.hasOption(USER_OPT)) {
395      userNames = cmd.getOptionValue(USER_OPT);
396    }
397    super.processOptions(cmd);
398  }
399
400  public static void main(String argv[]) throws Exception {
401    Configuration conf = HBaseConfiguration.create();
402    IntegrationTestingUtility.setUseDistributedCluster(conf);
403    int ret = ToolRunner.run(conf, new IntegrationTestWithCellVisibilityLoadAndVerify(), argv);
404    System.exit(ret);
405  }
406}