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.wal;
019
020import java.io.FileNotFoundException;
021import java.io.IOException;
022import java.io.PrintStream;
023import java.util.ArrayList;
024import java.util.Date;
025import java.util.HashMap;
026import java.util.Iterator;
027import java.util.List;
028import java.util.Map;
029
030import org.apache.hadoop.conf.Configuration;
031import org.apache.hadoop.fs.FileSystem;
032import org.apache.hadoop.fs.Path;
033import org.apache.hadoop.hbase.Cell;
034import org.apache.hadoop.hbase.CellUtil;
035import org.apache.hadoop.hbase.HBaseConfiguration;
036import org.apache.hadoop.hbase.HBaseInterfaceAudience;
037import org.apache.hadoop.hbase.PrivateCellUtil;
038import org.apache.hadoop.hbase.TableName;
039import org.apache.hadoop.hbase.Tag;
040import org.apache.hadoop.hbase.regionserver.wal.ProtobufLogReader;
041import org.apache.hadoop.hbase.util.Bytes;
042import org.apache.hadoop.hbase.util.FSUtils;
043import org.apache.yetus.audience.InterfaceAudience;
044import org.apache.yetus.audience.InterfaceStability;
045import org.slf4j.Logger;
046import org.slf4j.LoggerFactory;
047import org.apache.hbase.thirdparty.org.apache.commons.cli.CommandLine;
048import org.apache.hbase.thirdparty.org.apache.commons.cli.CommandLineParser;
049import org.apache.hbase.thirdparty.org.apache.commons.cli.HelpFormatter;
050import org.apache.hbase.thirdparty.org.apache.commons.cli.Options;
051import org.apache.hbase.thirdparty.org.apache.commons.cli.ParseException;
052import org.apache.hbase.thirdparty.org.apache.commons.cli.PosixParser;
053
054import com.fasterxml.jackson.databind.ObjectMapper;
055
056/**
057 * WALPrettyPrinter prints the contents of a given WAL with a variety of
058 * options affecting formatting and extent of content.
059 *
060 * It targets two usage cases: pretty printing for ease of debugging directly by
061 * humans, and JSON output for consumption by monitoring and/or maintenance
062 * scripts.
063 *
064 * It can filter by row, region, or sequence id.
065 *
066 * It can also toggle output of values.
067 *
068 */
069@InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.TOOLS)
070@InterfaceStability.Evolving
071public class WALPrettyPrinter {
072  private static final Logger LOG = LoggerFactory.getLogger(WALPrettyPrinter.class);
073
074  // Output template for pretty printing.
075  private static final String outputTmpl =
076      "Sequence=%s, table=%s, region=%s, at write timestamp=%s";
077
078  private boolean outputValues;
079  private boolean outputJSON;
080  // The following enable filtering by sequence, region, and row, respectively
081  private long sequence;
082  private String table;
083  private String region;
084  private String row;
085  // enable in order to output a single list of transactions from several files
086  private boolean persistentOutput;
087  private boolean firstTxn;
088  // useful for programmatic capture of JSON output
089  private PrintStream out;
090  // for JSON encoding
091  private static final ObjectMapper MAPPER = new ObjectMapper();
092  //allows for jumping straight to a given portion of the file
093  private long position;
094
095  /**
096   * Basic constructor that simply initializes values to reasonable defaults.
097   */
098  public WALPrettyPrinter() {
099    outputValues = false;
100    outputJSON = false;
101    sequence = -1;
102    table = null;
103    region = null;
104    row = null;
105    persistentOutput = false;
106    firstTxn = true;
107    out = System.out;
108  }
109
110  /**
111   * Fully specified constructor.
112   *
113   * @param outputValues
114   *          when true, enables output of values along with other log
115   *          information
116   * @param outputJSON
117   *          when true, enables output in JSON format rather than a
118   *          "pretty string"
119   * @param sequence
120   *          when nonnegative, serves as a filter; only log entries with this
121   *          sequence id will be printed
122   * @param table
123   *          when non null, serves as a filter. only entries corresponding to this
124   *          table will be printed.
125   * @param region
126   *          when not null, serves as a filter; only log entries from this
127   *          region will be printed
128   * @param row
129   *          when not null, serves as a filter; only log entries from this row
130   *          will be printed
131   * @param persistentOutput
132   *          keeps a single list running for multiple files. if enabled, the
133   *          endPersistentOutput() method must be used!
134   * @param out
135   *          Specifies an alternative to stdout for the destination of this
136   *          PrettyPrinter's output.
137   */
138  public WALPrettyPrinter(boolean outputValues, boolean outputJSON,
139      long sequence, String table, String region, String row, boolean persistentOutput,
140      PrintStream out) {
141    this.outputValues = outputValues;
142    this.outputJSON = outputJSON;
143    this.sequence = sequence;
144    this.table = table;
145    this.region = region;
146    this.row = row;
147    this.persistentOutput = persistentOutput;
148    if (persistentOutput) {
149      beginPersistentOutput();
150    }
151    this.out = out;
152    this.firstTxn = true;
153  }
154
155  /**
156   * turns value output on
157   */
158  public void enableValues() {
159    outputValues = true;
160  }
161
162  /**
163   * turns value output off
164   */
165  public void disableValues() {
166    outputValues = false;
167  }
168
169  /**
170   * turns JSON output on
171   */
172  public void enableJSON() {
173    outputJSON = true;
174  }
175
176  /**
177   * turns JSON output off, and turns on "pretty strings" for human consumption
178   */
179  public void disableJSON() {
180    outputJSON = false;
181  }
182
183  /**
184   * sets the region by which output will be filtered
185   *
186   * @param sequence
187   *          when nonnegative, serves as a filter; only log entries with this
188   *          sequence id will be printed
189   */
190  public void setSequenceFilter(long sequence) {
191    this.sequence = sequence;
192  }
193
194  /**
195   * Sets the table filter. Only log entries for this table are printed.
196   * @param table table name to set.
197   */
198  public void setTableFilter(String table) {
199    this.table = table;
200  }
201  /**
202   * sets the region by which output will be filtered
203   *
204   * @param region
205   *          when not null, serves as a filter; only log entries from this
206   *          region will be printed
207   */
208  public void setRegionFilter(String region) {
209    this.region = region;
210  }
211
212  /**
213   * sets the region by which output will be filtered
214   *
215   * @param row
216   *          when not null, serves as a filter; only log entries from this row
217   *          will be printed
218   */
219  public void setRowFilter(String row) {
220    this.row = row;
221  }
222
223  /**
224   * sets the position to start seeking the WAL file
225   * @param position
226   *          initial position to start seeking the given WAL file
227   */
228  public void setPosition(long position) {
229    this.position = position;
230  }
231
232  /**
233   * enables output as a single, persistent list. at present, only relevant in
234   * the case of JSON output.
235   */
236  public void beginPersistentOutput() {
237    if (persistentOutput) {
238      return;
239    }
240    persistentOutput = true;
241    firstTxn = true;
242    if (outputJSON) {
243      out.print("[");
244    }
245  }
246
247  /**
248   * ends output of a single, persistent list. at present, only relevant in the
249   * case of JSON output.
250   */
251  public void endPersistentOutput() {
252    if (!persistentOutput) {
253      return;
254    }
255    persistentOutput = false;
256    if (outputJSON) {
257      out.print("]");
258    }
259  }
260
261  /**
262   * reads a log file and outputs its contents, one transaction at a time, as
263   * specified by the currently configured options
264   *
265   * @param conf
266   *          the HBase configuration relevant to this log file
267   * @param p
268   *          the path of the log file to be read
269   * @throws IOException
270   *           may be unable to access the configured filesystem or requested
271   *           file.
272   */
273  public void processFile(final Configuration conf, final Path p)
274      throws IOException {
275    FileSystem fs = p.getFileSystem(conf);
276    if (!fs.exists(p)) {
277      throw new FileNotFoundException(p.toString());
278    }
279    if (!fs.isFile(p)) {
280      throw new IOException(p + " is not a file");
281    }
282
283    WAL.Reader log = WALFactory.createReader(fs, p, conf);
284
285    if (log instanceof ProtobufLogReader) {
286      List<String> writerClsNames = ((ProtobufLogReader) log).getWriterClsNames();
287      if (writerClsNames != null && writerClsNames.size() > 0) {
288        out.print("Writer Classes: ");
289        for (int i = 0; i < writerClsNames.size(); i++) {
290          out.print(writerClsNames.get(i));
291          if (i != writerClsNames.size() - 1) {
292            out.print(" ");
293          }
294        }
295        out.println();
296      }
297
298      String cellCodecClsName = ((ProtobufLogReader) log).getCodecClsName();
299      if (cellCodecClsName != null) {
300        out.println("Cell Codec Class: " + cellCodecClsName);
301      }
302    }
303
304    if (outputJSON && !persistentOutput) {
305      out.print("[");
306      firstTxn = true;
307    }
308
309    if (position > 0) {
310      log.seek(position);
311    }
312
313    try {
314      WAL.Entry entry;
315      while ((entry = log.next()) != null) {
316        WALKey key = entry.getKey();
317        WALEdit edit = entry.getEdit();
318        // begin building a transaction structure
319        Map<String, Object> txn = key.toStringMap();
320        long writeTime = key.getWriteTime();
321        // check output filters
322        if (table != null && !((TableName) txn.get("table")).toString().equals(table)) {
323          continue;
324        }
325        if (sequence >= 0 && ((Long) txn.get("sequence")) != sequence) {
326          continue;
327        }
328        if (region != null && !txn.get("region").equals(region)) {
329          continue;
330        }
331        // initialize list into which we will store atomic actions
332        List<Map> actions = new ArrayList<>();
333        for (Cell cell : edit.getCells()) {
334          // add atomic operation to txn
335          Map<String, Object> op = new HashMap<>(toStringMap(cell));
336          if (outputValues) {
337            op.put("value", Bytes.toStringBinary(CellUtil.cloneValue(cell)));
338          }
339          // check row output filter
340          if (row == null || ((String) op.get("row")).equals(row)) {
341            actions.add(op);
342          }
343          op.put("total_size_sum", PrivateCellUtil.estimatedSizeOfCell(cell));
344        }
345        if (actions.isEmpty()) {
346          continue;
347        }
348        txn.put("actions", actions);
349        if (outputJSON) {
350          // JSON output is a straightforward "toString" on the txn object
351          if (firstTxn) {
352            firstTxn = false;
353          } else {
354            out.print(",");
355          }
356          // encode and print JSON
357          out.print(MAPPER.writeValueAsString(txn));
358        } else {
359          // Pretty output, complete with indentation by atomic action
360          out.println(String.format(outputTmpl,
361              txn.get("sequence"), txn.get("table"), txn.get("region"), new Date(writeTime)));
362          for (int i = 0; i < actions.size(); i++) {
363            Map op = actions.get(i);
364            out.println("row=" + op.get("row") +
365                ", column=" + op.get("family") + ":" + op.get("qualifier"));
366            if (op.get("tag") != null) {
367              out.println("    tag: " + op.get("tag"));
368            }
369            if (outputValues) {
370              out.println("    value: " + op.get("value"));
371            }
372            out.println("cell total size sum: " + op.get("total_size_sum"));
373          }
374        }
375        out.println("edit heap size: " + entry.getEdit().heapSize());
376        out.println("position: " + log.getPosition());
377      }
378    } finally {
379      log.close();
380    }
381    if (outputJSON && !persistentOutput) {
382      out.print("]");
383    }
384  }
385
386  private static Map<String, Object> toStringMap(Cell cell) {
387    Map<String, Object> stringMap = new HashMap<>();
388    stringMap.put("row",
389        Bytes.toStringBinary(cell.getRowArray(), cell.getRowOffset(), cell.getRowLength()));
390    stringMap.put("family", Bytes.toStringBinary(cell.getFamilyArray(), cell.getFamilyOffset(),
391                cell.getFamilyLength()));
392    stringMap.put("qualifier",
393        Bytes.toStringBinary(cell.getQualifierArray(), cell.getQualifierOffset(),
394            cell.getQualifierLength()));
395    stringMap.put("timestamp", cell.getTimestamp());
396    stringMap.put("vlen", cell.getValueLength());
397    if (cell.getTagsLength() > 0) {
398      List<String> tagsString = new ArrayList<>();
399      Iterator<Tag> tagsIterator = PrivateCellUtil.tagsIterator(cell);
400      while (tagsIterator.hasNext()) {
401        Tag tag = tagsIterator.next();
402        tagsString
403            .add((tag.getType()) + ":" + Bytes.toStringBinary(Tag.cloneValue(tag)));
404      }
405      stringMap.put("tag", tagsString);
406    }
407    return stringMap;
408  }
409
410  public static void main(String[] args) throws IOException {
411    run(args);
412  }
413
414  /**
415   * Pass one or more log file names and formatting options and it will dump out
416   * a text version of the contents on <code>stdout</code>.
417   *
418   * @param args
419   *          Command line arguments
420   * @throws IOException
421   *           Thrown upon file system errors etc.
422   */
423  public static void run(String[] args) throws IOException {
424    // create options
425    Options options = new Options();
426    options.addOption("h", "help", false, "Output help message");
427    options.addOption("j", "json", false, "Output JSON");
428    options.addOption("p", "printvals", false, "Print values");
429    options.addOption("t", "table", true, "Table name to filter by.");
430    options.addOption("r", "region", true,
431        "Region to filter by. Pass encoded region name; e.g. '9192caead6a5a20acb4454ffbc79fa14'");
432    options.addOption("s", "sequence", true,
433        "Sequence to filter by. Pass sequence number.");
434    options.addOption("w", "row", true, "Row to filter by. Pass row name.");
435    options.addOption("g", "goto", true, "Position to seek to in the file");
436
437    WALPrettyPrinter printer = new WALPrettyPrinter();
438    CommandLineParser parser = new PosixParser();
439    List<?> files = null;
440    try {
441      CommandLine cmd = parser.parse(options, args);
442      files = cmd.getArgList();
443      if (files.isEmpty() || cmd.hasOption("h")) {
444        HelpFormatter formatter = new HelpFormatter();
445        formatter.printHelp("WAL <filename...>", options, true);
446        System.exit(-1);
447      }
448      // configure the pretty printer using command line options
449      if (cmd.hasOption("p")) {
450        printer.enableValues();
451      }
452      if (cmd.hasOption("j")) {
453        printer.enableJSON();
454      }
455      if (cmd.hasOption("t")) {
456        printer.setTableFilter(cmd.getOptionValue("t"));
457      }
458      if (cmd.hasOption("r")) {
459        printer.setRegionFilter(cmd.getOptionValue("r"));
460      }
461      if (cmd.hasOption("s")) {
462        printer.setSequenceFilter(Long.parseLong(cmd.getOptionValue("s")));
463      }
464      if (cmd.hasOption("w")) {
465        printer.setRowFilter(cmd.getOptionValue("w"));
466      }
467      if (cmd.hasOption("g")) {
468        printer.setPosition(Long.parseLong(cmd.getOptionValue("g")));
469      }
470    } catch (ParseException e) {
471      LOG.error("Failed to parse commandLine arguments", e);
472      HelpFormatter formatter = new HelpFormatter();
473      formatter.printHelp("HFile filename(s) ", options, true);
474      System.exit(-1);
475    }
476    // get configuration, file system, and process the given files
477    Configuration conf = HBaseConfiguration.create();
478    FSUtils.setFsDefault(conf, FSUtils.getRootDir(conf));
479
480    // begin output
481    printer.beginPersistentOutput();
482    for (Object f : files) {
483      Path file = new Path((String) f);
484      FileSystem fs = file.getFileSystem(conf);
485      if (!fs.exists(file)) {
486        System.err.println("ERROR, file doesnt exist: " + file);
487        return;
488      }
489      printer.processFile(conf, file);
490    }
491    printer.endPersistentOutput();
492  }
493}