compaction_job.cc 64.4 KB
Newer Older
1
//  Copyright (c) 2011-present, Facebook, Inc.  All rights reserved.
S
Siying Dong 已提交
2 3 4
//  This source code is licensed under both the GPLv2 (found in the
//  COPYING file in the root directory) and Apache 2.0 License
//  (found in the LICENSE.Apache file in the root directory).
I
Igor Canadi 已提交
5 6 7 8 9 10 11 12 13 14 15 16
//
// Copyright (c) 2011 The LevelDB Authors. All rights reserved.
// Use of this source code is governed by a BSD-style license that can be
// found in the LICENSE file. See the AUTHORS file for names of contributors.


#ifndef __STDC_FORMAT_MACROS
#define __STDC_FORMAT_MACROS
#endif

#include <inttypes.h>
#include <algorithm>
17
#include <functional>
I
Igor Canadi 已提交
18
#include <list>
19 20
#include <memory>
#include <random>
21 22 23
#include <set>
#include <thread>
#include <utility>
24
#include <vector>
I
Igor Canadi 已提交
25 26

#include "db/builder.h"
V
Vijay Nadimpalli 已提交
27
#include "db/compaction/compaction_job.h"
28
#include "db/db_impl/db_impl.h"
I
Igor Canadi 已提交
29 30
#include "db/db_iter.h"
#include "db/dbformat.h"
31
#include "db/error_handler.h"
32
#include "db/event_helpers.h"
I
Igor Canadi 已提交
33 34 35 36 37
#include "db/log_reader.h"
#include "db/log_writer.h"
#include "db/memtable.h"
#include "db/memtable_list.h"
#include "db/merge_context.h"
38
#include "db/merge_helper.h"
39
#include "db/range_del_aggregator.h"
I
Igor Canadi 已提交
40
#include "db/version_set.h"
41 42
#include "file/filename.h"
#include "file/sst_file_manager_impl.h"
43 44 45
#include "monitoring/iostats_context_imp.h"
#include "monitoring/perf_context_imp.h"
#include "monitoring/thread_status_util.h"
46
#include "port/port.h"
I
Igor Canadi 已提交
47 48 49 50 51
#include "rocksdb/db.h"
#include "rocksdb/env.h"
#include "rocksdb/statistics.h"
#include "rocksdb/status.h"
#include "rocksdb/table.h"
52 53
#include "table/block_based/block.h"
#include "table/block_based/block_based_table_factory.h"
54
#include "table/merging_iterator.h"
I
Igor Canadi 已提交
55
#include "table/table_builder.h"
56
#include "test_util/sync_point.h"
I
Igor Canadi 已提交
57
#include "util/coding.h"
58
#include "util/file_reader_writer.h"
I
Igor Canadi 已提交
59
#include "util/log_buffer.h"
60
#include "util/logging.h"
I
Igor Canadi 已提交
61
#include "util/mutexlock.h"
62
#include "util/random.h"
I
Igor Canadi 已提交
63
#include "util/stop_watch.h"
64
#include "util/string_util.h"
I
Igor Canadi 已提交
65 66 67

namespace rocksdb {

68 69 70 71 72 73 74 75 76 77 78 79 80 81 82 83 84 85 86 87 88 89 90 91 92 93 94 95 96 97 98 99
const char* GetCompactionReasonString(CompactionReason compaction_reason) {
  switch (compaction_reason) {
    case CompactionReason::kUnknown:
      return "Unknown";
    case CompactionReason::kLevelL0FilesNum:
      return "LevelL0FilesNum";
    case CompactionReason::kLevelMaxLevelSize:
      return "LevelMaxLevelSize";
    case CompactionReason::kUniversalSizeAmplification:
      return "UniversalSizeAmplification";
    case CompactionReason::kUniversalSizeRatio:
      return "UniversalSizeRatio";
    case CompactionReason::kUniversalSortedRunNum:
      return "UniversalSortedRunNum";
    case CompactionReason::kFIFOMaxSize:
      return "FIFOMaxSize";
    case CompactionReason::kFIFOReduceNumFiles:
      return "FIFOReduceNumFiles";
    case CompactionReason::kFIFOTtl:
      return "FIFOTtl";
    case CompactionReason::kManualCompaction:
      return "ManualCompaction";
    case CompactionReason::kFilesMarkedForCompaction:
      return "FilesMarkedForCompaction";
    case CompactionReason::kBottommostFiles:
      return "BottommostFiles";
    case CompactionReason::kTtl:
      return "Ttl";
    case CompactionReason::kFlush:
      return "Flush";
    case CompactionReason::kExternalSstIngestion:
      return "ExternalSstIngestion";
S
Sagar Vemuri 已提交
100 101
    case CompactionReason::kPeriodicCompaction:
      return "PeriodicCompaction";
102 103 104 105 106 107 108 109
    case CompactionReason::kNumOfReasons:
      // fall through
    default:
      assert(false);
      return "Invalid";
  }
}

110
// Maintains state for each sub-compaction
111
struct CompactionJob::SubcompactionState {
112
  const Compaction* compaction;
113
  std::unique_ptr<CompactionIterator> c_iter;
I
Igor Canadi 已提交
114

115 116 117 118 119
  // The boundaries of the key-range this compaction is interested in. No two
  // subcompactions may have overlapping key-ranges.
  // 'start' is inclusive, 'end' is exclusive, and nullptr means unbounded
  Slice *start, *end;

120
  // The return status of this subcompaction
121 122
  Status status;

123
  // Files produced by this subcompaction
I
Igor Canadi 已提交
124
  struct Output {
125 126
    FileMetaData meta;
    bool finished;
127
    std::shared_ptr<const TableProperties> table_properties;
I
Igor Canadi 已提交
128 129 130
  };

  // State kept for output being generated
131
  std::vector<Output> outputs;
132
  std::unique_ptr<WritableFileWriter> outfile;
I
Igor Canadi 已提交
133
  std::unique_ptr<TableBuilder> builder;
134
  Output* current_output() {
135
    if (outputs.empty()) {
136
      // This subcompaction's outptut could be empty if compaction was aborted
137 138
      // before this subcompaction had a chance to generate any output files.
      // When subcompactions are executed sequentially this is more likely and
139
      // will be particulalry likely for the later subcompactions to be empty.
140 141 142 143 144
      // Once they are run in parallel however it should be much rarer.
      return nullptr;
    } else {
      return &outputs.back();
    }
145
  }
I
Igor Canadi 已提交
146

147 148
  uint64_t current_output_file_size;

149
  // State during the subcompaction
I
Igor Canadi 已提交
150
  uint64_t total_bytes;
151 152 153
  uint64_t num_input_records;
  uint64_t num_output_records;
  CompactionJobStats compaction_job_stats;
154
  uint64_t approx_size;
155
  // An index that used to speed up ShouldStopBefore().
156 157
  size_t grandparent_index = 0;
  // The number of bytes overlapping between the current output and
158
  // grandparent files used in ShouldStopBefore().
159
  uint64_t overlapped_bytes = 0;
160
  // A flag determine whether the key has been seen in ShouldStopBefore()
161
  bool seen_key = false;
162

163 164
  SubcompactionState(Compaction* c, Slice* _start, Slice* _end,
                     uint64_t size = 0)
165 166 167 168 169
      : compaction(c),
        start(_start),
        end(_end),
        outfile(nullptr),
        builder(nullptr),
170
        current_output_file_size(0),
171 172
        total_bytes(0),
        num_input_records(0),
173
        num_output_records(0),
174 175 176
        approx_size(size),
        grandparent_index(0),
        overlapped_bytes(0),
177
        seen_key(false) {
D
Dmitri Smirnov 已提交
178 179
    assert(compaction != nullptr);
  }
D
Dmitri Smirnov 已提交
180

181
  SubcompactionState(SubcompactionState&& o) { *this = std::move(o); }
D
Dmitri Smirnov 已提交
182

183
  SubcompactionState& operator=(SubcompactionState&& o) {
D
Dmitri Smirnov 已提交
184 185 186 187 188 189 190
    compaction = std::move(o.compaction);
    start = std::move(o.start);
    end = std::move(o.end);
    status = std::move(o.status);
    outputs = std::move(o.outputs);
    outfile = std::move(o.outfile);
    builder = std::move(o.builder);
191
    current_output_file_size = std::move(o.current_output_file_size);
D
Dmitri Smirnov 已提交
192 193 194
    total_bytes = std::move(o.total_bytes);
    num_input_records = std::move(o.num_input_records);
    num_output_records = std::move(o.num_output_records);
195 196
    compaction_job_stats = std::move(o.compaction_job_stats);
    approx_size = std::move(o.approx_size);
197 198 199
    grandparent_index = std::move(o.grandparent_index);
    overlapped_bytes = std::move(o.overlapped_bytes);
    seen_key = std::move(o.seen_key);
D
Dmitri Smirnov 已提交
200 201 202
    return *this;
  }

203
  // Because member std::unique_ptrs do not have these.
204
  SubcompactionState(const SubcompactionState&) = delete;
D
Dmitri Smirnov 已提交
205

206
  SubcompactionState& operator=(const SubcompactionState&) = delete;
207 208 209

  // Returns true iff we should stop building the current output
  // before processing "internal_key".
210
  bool ShouldStopBefore(const Slice& internal_key, uint64_t curr_file_size) {
211 212 213 214 215 216 217 218 219 220 221 222 223 224 225
    const InternalKeyComparator* icmp =
        &compaction->column_family_data()->internal_comparator();
    const std::vector<FileMetaData*>& grandparents = compaction->grandparents();

    // Scan to find earliest grandparent file that contains key.
    while (grandparent_index < grandparents.size() &&
           icmp->Compare(internal_key,
                         grandparents[grandparent_index]->largest.Encode()) >
               0) {
      if (seen_key) {
        overlapped_bytes += grandparents[grandparent_index]->fd.GetFileSize();
      }
      assert(grandparent_index + 1 >= grandparents.size() ||
             icmp->Compare(
                 grandparents[grandparent_index]->largest.Encode(),
226
                 grandparents[grandparent_index + 1]->smallest.Encode()) <= 0);
227 228 229 230
      grandparent_index++;
    }
    seen_key = true;

231 232
    if (overlapped_bytes + curr_file_size >
        compaction->max_compaction_bytes()) {
233 234 235 236 237 238 239
      // Too much overlap for current output; start new output
      overlapped_bytes = 0;
      return true;
    }

    return false;
  }
240 241 242 243 244
};

// Maintains state for the entire compaction
struct CompactionJob::CompactionState {
  Compaction* const compaction;
I
Igor Canadi 已提交
245

246 247
  // REQUIRED: subcompaction states are stored in order of increasing
  // key-range
248
  std::vector<CompactionJob::SubcompactionState> sub_compact_states;
249 250 251 252 253
  Status status;

  uint64_t total_bytes;
  uint64_t num_input_records;
  uint64_t num_output_records;
I
Igor Canadi 已提交
254

S
sdong 已提交
255 256 257 258 259
  explicit CompactionState(Compaction* c)
      : compaction(c),
        total_bytes(0),
        num_input_records(0),
        num_output_records(0) {}
I
Igor Canadi 已提交
260

261 262 263 264 265 266 267 268 269
  size_t NumOutputFiles() {
    size_t total = 0;
    for (auto& s : sub_compact_states) {
      total += s.outputs.size();
    }
    return total;
  }

  Slice SmallestUserKey() {
270 271 272 273
    for (const auto& sub_compact_state : sub_compact_states) {
      if (!sub_compact_state.outputs.empty() &&
          sub_compact_state.outputs[0].finished) {
        return sub_compact_state.outputs[0].meta.smallest.user_key();
274 275
      }
    }
276
    // If there is no finished output, return an empty slice.
277
    return Slice(nullptr, 0);
278 279 280
  }

  Slice LargestUserKey() {
281 282 283 284 285
    for (auto it = sub_compact_states.rbegin(); it < sub_compact_states.rend();
         ++it) {
      if (!it->outputs.empty() && it->current_output()->finished) {
        assert(it->current_output() != nullptr);
        return it->current_output()->meta.largest.user_key();
286 287
      }
    }
288
    // If there is no finished output, return an empty slice.
289
    return Slice(nullptr, 0);
290
  }
I
Igor Canadi 已提交
291 292
};

293
void CompactionJob::AggregateStatistics() {
294
  for (SubcompactionState& sc : compact_->sub_compact_states) {
295 296 297
    compact_->total_bytes += sc.total_bytes;
    compact_->num_input_records += sc.num_input_records;
    compact_->num_output_records += sc.num_output_records;
298 299 300
  }
  if (compaction_job_stats_) {
    for (SubcompactionState& sc : compact_->sub_compact_states) {
301 302 303 304 305
      compaction_job_stats_->Add(sc.compaction_job_stats);
    }
  }
}

I
Igor Canadi 已提交
306
CompactionJob::CompactionJob(
307
    int job_id, Compaction* compaction, const ImmutableDBOptions& db_options,
308
    const EnvOptions env_options, VersionSet* versions,
309
    const std::atomic<bool>* shutting_down,
310
    const SequenceNumber preserve_deletes_seqnum, LogBuffer* log_buffer,
I
Igor Canadi 已提交
311
    Directory* db_directory, Directory* output_directory, Statistics* stats,
312
    InstrumentedMutex* db_mutex, ErrorHandler* db_error_handler,
I
Igor Canadi 已提交
313
    std::vector<SequenceNumber> existing_snapshots,
314
    SequenceNumber earliest_write_conflict_snapshot,
Y
Yi Wu 已提交
315 316
    const SnapshotChecker* snapshot_checker, std::shared_ptr<Cache> table_cache,
    EventLogger* event_logger, bool paranoid_file_checks, bool measure_io_stats,
317
    const std::string& dbname, CompactionJobStats* compaction_job_stats,
318
    Env::Priority thread_pri, SnapshotListFetchCallback* snap_list_callback)
319 320
    : job_id_(job_id),
      compact_(new CompactionState(compaction)),
321
      compaction_job_stats_(compaction_job_stats),
322
      compaction_stats_(compaction->compaction_reason(), 1),
323
      dbname_(dbname),
I
Igor Canadi 已提交
324 325 326
      db_options_(db_options),
      env_options_(env_options),
      env_(db_options.env),
327
      env_options_for_read_(
328
          env_->OptimizeForCompactionTableRead(env_options, db_options_)),
I
Igor Canadi 已提交
329 330
      versions_(versions),
      shutting_down_(shutting_down),
331
      preserve_deletes_seqnum_(preserve_deletes_seqnum),
I
Igor Canadi 已提交
332 333
      log_buffer_(log_buffer),
      db_directory_(db_directory),
334
      output_directory_(output_directory),
I
Igor Canadi 已提交
335
      stats_(stats),
336
      db_mutex_(db_mutex),
337
      db_error_handler_(db_error_handler),
I
Igor Canadi 已提交
338
      existing_snapshots_(std::move(existing_snapshots)),
339
      snap_list_callback_(snap_list_callback),
340
      earliest_write_conflict_snapshot_(earliest_write_conflict_snapshot),
Y
Yi Wu 已提交
341
      snapshot_checker_(snapshot_checker),
I
Igor Canadi 已提交
342
      table_cache_(std::move(table_cache)),
I
Igor Canadi 已提交
343
      event_logger_(event_logger),
344
      bottommost_level_(false),
345
      paranoid_file_checks_(paranoid_file_checks),
S
Stream  
Shaohua Li 已提交
346
      measure_io_stats_(measure_io_stats),
347 348
      write_hint_(Env::WLTH_NOT_SET),
      thread_pri_(thread_pri) {
349
  assert(log_buffer_ != nullptr);
350 351
  const auto* cfd = compact_->compaction->column_family_data();
  ThreadStatusUtil::SetColumnFamily(cfd, cfd->ioptions()->env,
Y
Yi Wu 已提交
352
                                    db_options_.enable_thread_tracking);
353
  ThreadStatusUtil::SetThreadOperation(ThreadStatus::OP_COMPACTION);
354
  ReportStartedCompaction(compaction);
355 356 357 358 359 360
}

CompactionJob::~CompactionJob() {
  assert(compact_ == nullptr);
  ThreadStatusUtil::ResetThreadStatus();
}
I
Igor Canadi 已提交
361

362
void CompactionJob::ReportStartedCompaction(Compaction* compaction) {
363 364
  const auto* cfd = compact_->compaction->column_family_data();
  ThreadStatusUtil::SetColumnFamily(cfd, cfd->ioptions()->env,
Y
Yi Wu 已提交
365
                                    db_options_.enable_thread_tracking);
366

367 368
  ThreadStatusUtil::SetThreadOperationProperty(ThreadStatus::COMPACTION_JOB_ID,
                                               job_id_);
369 370 371 372 373 374

  ThreadStatusUtil::SetThreadOperationProperty(
      ThreadStatus::COMPACTION_INPUT_OUTPUT_LEVEL,
      (static_cast<uint64_t>(compact_->compaction->start_level()) << 32) +
          compact_->compaction->output_level());

375 376 377
  // In the current design, a CompactionJob is always created
  // for non-trivial compaction.
  assert(compaction->IsTrivialMove() == false ||
378
         compaction->is_manual_compaction() == true);
379

380 381
  ThreadStatusUtil::SetThreadOperationProperty(
      ThreadStatus::COMPACTION_PROP_FLAGS,
S
sdong 已提交
382
      compaction->is_manual_compaction() +
383
          (compaction->deletion_compaction() << 1));
384 385 386 387 388 389 390 391 392 393 394 395 396 397

  ThreadStatusUtil::SetThreadOperationProperty(
      ThreadStatus::COMPACTION_TOTAL_INPUT_BYTES,
      compaction->CalculateTotalInputSize());

  IOSTATS_RESET(bytes_written);
  IOSTATS_RESET(bytes_read);
  ThreadStatusUtil::SetThreadOperationProperty(
      ThreadStatus::COMPACTION_BYTES_WRITTEN, 0);
  ThreadStatusUtil::SetThreadOperationProperty(
      ThreadStatus::COMPACTION_BYTES_READ, 0);

  // Set the thread operation after operation properties
  // to ensure GetThreadList() can always show them all together.
398
  ThreadStatusUtil::SetThreadOperation(ThreadStatus::OP_COMPACTION);
399 400 401

  if (compaction_job_stats_) {
    compaction_job_stats_->is_manual_compaction =
S
sdong 已提交
402
        compaction->is_manual_compaction();
403
  }
404 405
}

I
Igor Canadi 已提交
406
void CompactionJob::Prepare() {
407 408
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_PREPARE);
I
Igor Canadi 已提交
409 410

  // Generate file_levels_ for compaction berfore making Iterator
411 412
  auto* c = compact_->compaction;
  assert(c->column_family_data() != nullptr);
413 414
  assert(c->column_family_data()->current()->storage_info()->NumLevelFiles(
             compact_->compaction->level()) > 0);
I
Igor Canadi 已提交
415

416 417
  write_hint_ =
      c->column_family_data()->CalculateSSTWriteHint(c->output_level());
418
  bottommost_level_ = c->bottommost_level();
419

420
  if (c->ShouldFormSubcompactions()) {
S
Siying Dong 已提交
421 422 423 424
    {
      StopWatch sw(env_, stats_, SUBCOMPACTION_SETUP_TIME);
      GenSubcompactionBoundaries();
    }
425 426 427 428 429 430 431
    assert(sizes_.size() == boundaries_.size() + 1);

    for (size_t i = 0; i <= boundaries_.size(); i++) {
      Slice* start = i == 0 ? nullptr : &boundaries_[i - 1];
      Slice* end = i == boundaries_.size() ? nullptr : &boundaries_[i];
      compact_->sub_compact_states.emplace_back(c, start, end, sizes_[i]);
    }
S
Siying Dong 已提交
432 433
    RecordInHistogram(stats_, NUM_SUBCOMPACTIONS_SCHEDULED,
                      compact_->sub_compact_states.size());
434 435 436
  } else {
    compact_->sub_compact_states.emplace_back(c, nullptr, nullptr);
  }
437 438
}

439 440 441
struct RangeWithSize {
  Range range;
  uint64_t size;
442

443 444 445 446 447 448 449
  RangeWithSize(const Slice& a, const Slice& b, uint64_t s = 0)
      : range(a, b), size(s) {}
};

void CompactionJob::GenSubcompactionBoundaries() {
  auto* c = compact_->compaction;
  auto* cfd = c->column_family_data();
450 451
  const Comparator* cfd_comparator = cfd->user_comparator();
  std::vector<Slice> bounds;
452 453 454 455
  int start_lvl = c->start_level();
  int out_lvl = c->output_level();

  // Add the starting and/or ending key of certain input files as a potential
456
  // boundary
457 458 459 460 461 462 463
  for (size_t lvl_idx = 0; lvl_idx < c->num_input_levels(); lvl_idx++) {
    int lvl = c->level(lvl_idx);
    if (lvl >= start_lvl && lvl <= out_lvl) {
      const LevelFilesBrief* flevel = c->input_levels(lvl_idx);
      size_t num_files = flevel->num_files;

      if (num_files == 0) {
464
        continue;
465
      }
A
Ari Ekmekji 已提交
466

467 468 469 470
      if (lvl == 0) {
        // For level 0 add the starting and ending key of each file since the
        // files may have greatly differing key ranges (not range-partitioned)
        for (size_t i = 0; i < num_files; i++) {
471 472
          bounds.emplace_back(flevel->files[i].smallest_key);
          bounds.emplace_back(flevel->files[i].largest_key);
473 474 475 476
        }
      } else {
        // For all other levels add the smallest/largest key in the level to
        // encompass the range covered by that level
477 478
        bounds.emplace_back(flevel->files[0].smallest_key);
        bounds.emplace_back(flevel->files[num_files - 1].largest_key);
479 480 481 482 483 484
        if (lvl == out_lvl) {
          // For the last level include the starting keys of all files since
          // the last level is the largest and probably has the widest key
          // range. Since it's range partitioned, the ending key of one file
          // and the starting key of the next are very close (or identical).
          for (size_t i = 1; i < num_files; i++) {
485
            bounds.emplace_back(flevel->files[i].smallest_key);
A
Ari Ekmekji 已提交
486
          }
487 488 489 490
        }
      }
    }
  }
491

492
  std::sort(bounds.begin(), bounds.end(),
493 494 495 496
            [cfd_comparator](const Slice& a, const Slice& b) -> bool {
              return cfd_comparator->Compare(ExtractUserKey(a),
                                             ExtractUserKey(b)) < 0;
            });
497
  // Remove duplicated entries from bounds
498 499 500 501 502 503 504
  bounds.erase(
      std::unique(bounds.begin(), bounds.end(),
                  [cfd_comparator](const Slice& a, const Slice& b) -> bool {
                    return cfd_comparator->Compare(ExtractUserKey(a),
                                                   ExtractUserKey(b)) == 0;
                  }),
      bounds.end());
505

506 507 508 509
  // Combine consecutive pairs of boundaries into ranges with an approximate
  // size of data covered by keys in that range
  uint64_t sum = 0;
  std::vector<RangeWithSize> ranges;
510 511 512 513
  // Get input version from CompactionState since it's already referenced
  // earlier in SetInputVersioCompaction::SetInputVersion and will not change
  // when db_mutex_ is released below
  auto* v = compact_->compaction->input_version();
514 515
  for (auto it = bounds.begin();;) {
    const Slice a = *it;
516
    ++it;
517 518 519 520 521 522

    if (it == bounds.end()) {
      break;
    }

    const Slice b = *it;
523 524 525 526 527

    // ApproximateSize could potentially create table reader iterator to seek
    // to the index block and may incur I/O cost in the process. Unlock db
    // mutex to reduce contention
    db_mutex_->Unlock();
528
    uint64_t size = versions_->ApproximateSize(v, a, b, start_lvl, out_lvl + 1);
529
    db_mutex_->Lock();
530 531 532 533 534 535
    ranges.emplace_back(a, b, size);
    sum += size;
  }

  // Group the ranges into subcompactions
  const double min_file_fill_percent = 4.0 / 5;
536 537 538 539 540 541
  int base_level = v->storage_info()->base_level();
  uint64_t max_output_files = static_cast<uint64_t>(std::ceil(
      sum / min_file_fill_percent /
      MaxFileSizeForLevel(*(c->mutable_cf_options()), out_lvl,
          c->immutable_cf_options()->compaction_style, base_level,
          c->immutable_cf_options()->level_compaction_dynamic_level_bytes)));
542 543
  uint64_t subcompactions =
      std::min({static_cast<uint64_t>(ranges.size()),
544
                static_cast<uint64_t>(c->max_subcompactions()),
545 546 547
                max_output_files});

  if (subcompactions > 1) {
548
    double mean = sum * 1.0 / subcompactions;
549 550 551 552
    // Greedily add ranges to the subcompaction until the sum of the ranges'
    // sizes becomes >= the expected mean size of a subcompaction
    sum = 0;
    for (size_t i = 0; i < ranges.size() - 1; i++) {
553
      sum += ranges[i].size;
554 555 556
      if (subcompactions == 1) {
        // If there's only one left to schedule then it goes to the end so no
        // need to put an end boundary
557
        continue;
558 559 560 561 562 563 564 565 566 567 568 569
      }
      if (sum >= mean) {
        boundaries_.emplace_back(ExtractUserKey(ranges[i].range.limit));
        sizes_.emplace_back(sum);
        subcompactions--;
        sum = 0;
      }
    }
    sizes_.emplace_back(sum + ranges.back().size);
  } else {
    // Only one range so its size is the total sum of sizes computed above
    sizes_.emplace_back(sum);
570
  }
I
Igor Canadi 已提交
571 572 573
}

Status CompactionJob::Run() {
574 575
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_RUN);
576
  TEST_SYNC_POINT("CompactionJob::Run():Start");
I
Igor Canadi 已提交
577
  log_buffer_->FlushBufferToLog();
578
  LogCompaction();
579

580 581
  const size_t num_threads = compact_->sub_compact_states.size();
  assert(num_threads > 0);
582
  const uint64_t start_micros = env_->NowMicros();
583 584

  // Launch a thread for each of subcompactions 1...num_threads-1
D
Dmitri Smirnov 已提交
585
  std::vector<port::Thread> thread_pool;
586 587 588 589
  thread_pool.reserve(num_threads - 1);
  for (size_t i = 1; i < compact_->sub_compact_states.size(); i++) {
    thread_pool.emplace_back(&CompactionJob::ProcessKeyValueCompaction, this,
                             &compact_->sub_compact_states[i]);
590
  }
591 592 593 594 595 596 597 598 599 600

  // Always schedule the first subcompaction (whether or not there are also
  // others) in the current thread to be efficient with resources
  ProcessKeyValueCompaction(&compact_->sub_compact_states[0]);

  // Wait for all other threads (if there are any) to finish execution
  for (auto& thread : thread_pool) {
    thread.join();
  }

601
  compaction_stats_.micros = env_->NowMicros() - start_micros;
602 603 604 605 606 607
  compaction_stats_.cpu_micros = 0;
  for (size_t i = 0; i < compact_->sub_compact_states.size(); i++) {
    compaction_stats_.cpu_micros +=
        compact_->sub_compact_states[i].compaction_job_stats.cpu_micros;
  }

S
Siying Dong 已提交
608 609 610
  RecordTimeToHistogram(stats_, COMPACTION_TIME, compaction_stats_.micros);
  RecordTimeToHistogram(stats_, COMPACTION_CPU_TIME,
                        compaction_stats_.cpu_micros);
611

612 613
  TEST_SYNC_POINT("CompactionJob::Run:BeforeVerify");

614
  // Check if any thread encountered an error during execution
615 616 617 618
  Status status;
  for (const auto& state : compact_->sub_compact_states) {
    if (!state.status.ok()) {
      status = state.status;
619 620 621 622
      break;
    }
  }

623 624 625 626
  if (status.ok() && output_directory_) {
    status = output_directory_->Fsync();
  }

627 628 629 630 631 632 633 634 635 636 637 638 639 640 641 642 643 644 645 646 647 648 649 650 651 652
  if (status.ok()) {
    thread_pool.clear();
    std::vector<const FileMetaData*> files_meta;
    for (const auto& state : compact_->sub_compact_states) {
      for (const auto& output : state.outputs) {
        files_meta.emplace_back(&output.meta);
      }
    }
    ColumnFamilyData* cfd = compact_->compaction->column_family_data();
    auto prefix_extractor =
        compact_->compaction->mutable_cf_options()->prefix_extractor.get();
    std::atomic<size_t> next_file_meta_idx(0);
    auto verify_table = [&](Status& output_status) {
      while (true) {
        size_t file_idx = next_file_meta_idx.fetch_add(1);
        if (file_idx >= files_meta.size()) {
          break;
        }
        // Verify that the table is usable
        // We set for_compaction to false and don't OptimizeForCompactionTableRead
        // here because this is a special case after we finish the table building
        // No matter whether use_direct_io_for_flush_and_compaction is true,
        // we will regard this verification as user reads since the goal is
        // to cache it here for further user reads
        InternalIterator* iter = cfd->table_cache()->NewIterator(
            ReadOptions(), env_options_, cfd->internal_comparator(),
653
            *files_meta[file_idx], nullptr /* range_del_agg */,
654 655 656 657 658 659 660 661 662 663 664 665 666 667 668 669 670 671 672 673 674 675 676 677 678 679 680 681 682 683 684 685 686 687 688 689
            prefix_extractor, nullptr,
            cfd->internal_stats()->GetFileReadHist(
                compact_->compaction->output_level()),
            false, nullptr /* arena */, false /* skip_filters */,
            compact_->compaction->output_level());
        auto s = iter->status();

        if (s.ok() && paranoid_file_checks_) {
          for (iter->SeekToFirst(); iter->Valid(); iter->Next()) {}
          s = iter->status();
        }

        delete iter;

        if (!s.ok()) {
          output_status = s;
          break;
        }
      }
    };
    for (size_t i = 1; i < compact_->sub_compact_states.size(); i++) {
      thread_pool.emplace_back(verify_table,
                               std::ref(compact_->sub_compact_states[i].status));
    }
    verify_table(compact_->sub_compact_states[0].status);
    for (auto& thread : thread_pool) {
      thread.join();
    }
    for (const auto& state : compact_->sub_compact_states) {
      if (!state.status.ok()) {
        status = state.status;
        break;
      }
    }
  }

690 691 692
  TablePropertiesCollection tp;
  for (const auto& state : compact_->sub_compact_states) {
    for (const auto& output : state.outputs) {
693 694 695
      auto fn =
          TableFileName(state.compaction->immutable_cf_options()->cf_paths,
                        output.meta.fd.GetNumber(), output.meta.fd.GetPathId());
696 697 698 699 700
      tp[fn] = output.table_properties;
    }
  }
  compact_->compaction->SetOutputTableProperties(std::move(tp));

701 702
  // Finish up all book-keeping to unify the subcompaction results
  AggregateStatistics();
703 704 705 706 707
  UpdateCompactionStats();
  RecordCompactionIOStats();
  LogFlush(db_options_.info_log);
  TEST_SYNC_POINT("CompactionJob::Run():End");

708
  compact_->status = status;
709 710 711
  return status;
}

712
Status CompactionJob::Install(const MutableCFOptions& mutable_cf_options) {
713 714
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_INSTALL);
715
  db_mutex_->AssertHeld();
716
  Status status = compact_->status;
I
Igor Canadi 已提交
717 718
  ColumnFamilyData* cfd = compact_->compaction->column_family_data();
  cfd->internal_stats()->AddCompactionStats(
719
      compact_->compaction->output_level(), thread_pri_, compaction_stats_);
I
Igor Canadi 已提交
720

721
  if (status.ok()) {
722
    status = InstallCompactionResults(mutable_cf_options);
I
Igor Canadi 已提交
723 724
  }
  VersionStorageInfo::LevelSummaryStorage tmp;
725
  auto vstorage = cfd->current()->storage_info();
I
Igor Canadi 已提交
726
  const auto& stats = compaction_stats_;
727 728 729

  double read_write_amp = 0.0;
  double write_amp = 0.0;
730 731 732
  double bytes_read_per_sec = 0;
  double bytes_written_per_sec = 0;

733 734 735 736 737 738 739
  if (stats.bytes_read_non_output_levels > 0) {
    read_write_amp = (stats.bytes_written + stats.bytes_read_output_level +
                      stats.bytes_read_non_output_levels) /
                     static_cast<double>(stats.bytes_read_non_output_levels);
    write_amp = stats.bytes_written /
                static_cast<double>(stats.bytes_read_non_output_levels);
  }
740 741 742 743 744 745 746 747
  if (stats.micros > 0) {
    bytes_read_per_sec =
        (stats.bytes_read_non_output_levels + stats.bytes_read_output_level) /
        static_cast<double>(stats.micros);
    bytes_written_per_sec =
        stats.bytes_written / static_cast<double>(stats.micros);
  }

748
  ROCKS_LOG_BUFFER(
749 750 751 752
      log_buffer_,
      "[%s] compacted to: %s, MB/sec: %.1f rd, %.1f wr, level %d, "
      "files in(%d, %d) out(%d) "
      "MB in(%.1f, %.1f) out(%.1f), read-write-amplify(%.1f) "
S
Siying Dong 已提交
753 754
      "write-amplify(%.1f) %s, records in: %" PRIu64
      ", records dropped: %" PRIu64 " output_compression: %s\n",
755 756
      cfd->GetName().c_str(), vstorage->LevelSummary(&tmp), bytes_read_per_sec,
      bytes_written_per_sec, compact_->compaction->output_level(),
757
      stats.num_input_files_in_non_output_levels,
758
      stats.num_input_files_in_output_level, stats.num_output_files,
759 760
      stats.bytes_read_non_output_levels / 1048576.0,
      stats.bytes_read_output_level / 1048576.0,
761
      stats.bytes_written / 1048576.0, read_write_amp, write_amp,
762
      status.ToString().c_str(), stats.num_input_records,
763 764 765
      stats.num_dropped_records,
      CompressionTypeToString(compact_->compaction->output_compression())
          .c_str());
I
Igor Canadi 已提交
766

767 768
  UpdateCompactionJobStats(stats);

769
  auto stream = event_logger_->LogToBuffer(log_buffer_);
770 771
  stream << "job" << job_id_ << "event"
         << "compaction_finished"
A
Ari Ekmekji 已提交
772
         << "compaction_time_micros" << compaction_stats_.micros
773
         << "compaction_time_cpu_micros" << compaction_stats_.cpu_micros
774
         << "output_level" << compact_->compaction->output_level()
775
         << "num_output_files" << compact_->NumOutputFiles()
776 777 778 779 780
         << "total_output_size" << compact_->total_bytes << "num_input_records"
         << compact_->num_input_records << "num_output_records"
         << compact_->num_output_records << "num_subcompactions"
         << compact_->sub_compact_states.size() << "output_compression"
         << CompressionTypeToString(compact_->compaction->output_compression());
781

782 783 784 785 786 787 788
  if (compaction_job_stats_ != nullptr) {
    stream << "num_single_delete_mismatches"
           << compaction_job_stats_->num_single_del_mismatch;
    stream << "num_single_delete_fallthrough"
           << compaction_job_stats_->num_single_del_fallthru;
  }

789 790 791 792 793 794 795 796 797
  if (measure_io_stats_ && compaction_job_stats_ != nullptr) {
    stream << "file_write_nanos" << compaction_job_stats_->file_write_nanos;
    stream << "file_range_sync_nanos"
           << compaction_job_stats_->file_range_sync_nanos;
    stream << "file_fsync_nanos" << compaction_job_stats_->file_fsync_nanos;
    stream << "file_prepare_write_nanos"
           << compaction_job_stats_->file_prepare_write_nanos;
  }

798 799 800 801 802 803 804
  stream << "lsm_state";
  stream.StartArray();
  for (int level = 0; level < vstorage->num_levels(); ++level) {
    stream << vstorage->NumLevelFiles(level);
  }
  stream.EndArray();

805 806
  CleanupCompaction();
  return status;
I
Igor Canadi 已提交
807 808
}

809
void CompactionJob::ProcessKeyValueCompaction(SubcompactionState* sub_compact) {
810
  assert(sub_compact != nullptr);
811 812 813

  uint64_t prev_cpu_micros = env_->NowCPUNanos() / 1000;

814
  ColumnFamilyData* cfd = sub_compact->compaction->column_family_data();
815 816 817 818 819 820 821 822 823 824 825 826 827 828 829 830 831 832

  // Create compaction filter and fail the compaction if
  // IgnoreSnapshots() = false because it is not supported anymore
  const CompactionFilter* compaction_filter =
      cfd->ioptions()->compaction_filter;
  std::unique_ptr<CompactionFilter> compaction_filter_from_factory = nullptr;
  if (compaction_filter == nullptr) {
    compaction_filter_from_factory =
        sub_compact->compaction->CreateCompactionFilter();
    compaction_filter = compaction_filter_from_factory.get();
  }
  if (compaction_filter != nullptr && !compaction_filter->IgnoreSnapshots()) {
    sub_compact->status = Status::NotSupported(
        "CompactionFilter::IgnoreSnapshots() = false is not supported "
        "anymore.");
    return;
  }

833 834
  CompactionRangeDelAggregator range_del_agg(&cfd->internal_comparator(),
                                             existing_snapshots_);
835 836 837

  // Although the v2 aggregator is what the level iterator(s) know about,
  // the AddTombstones calls will be propagated down to the v1 aggregator.
838
  std::unique_ptr<InternalIterator> input(versions_->MakeInputIterator(
839
      sub_compact->compaction, &range_del_agg, env_options_for_read_));
840

841 842
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_PROCESS_KV);
843 844 845

  // I/O measurement variables
  PerfLevel prev_perf_level = PerfLevel::kEnableTime;
846
  const uint64_t kRecordStatsEvery = 1000;
847 848 849 850
  uint64_t prev_write_nanos = 0;
  uint64_t prev_fsync_nanos = 0;
  uint64_t prev_range_sync_nanos = 0;
  uint64_t prev_prepare_write_nanos = 0;
851 852
  uint64_t prev_cpu_write_nanos = 0;
  uint64_t prev_cpu_read_nanos = 0;
853 854
  if (measure_io_stats_) {
    prev_perf_level = GetPerfLevel();
855
    SetPerfLevel(PerfLevel::kEnableTimeAndCPUTimeExceptForMutex);
I
Igor Canadi 已提交
856 857 858 859
    prev_write_nanos = IOSTATS(write_nanos);
    prev_fsync_nanos = IOSTATS(fsync_nanos);
    prev_range_sync_nanos = IOSTATS(range_sync_nanos);
    prev_prepare_write_nanos = IOSTATS(prepare_write_nanos);
860 861
    prev_cpu_write_nanos = IOSTATS(cpu_write_nanos);
    prev_cpu_read_nanos = IOSTATS(cpu_read_nanos);
862 863
  }

I
Igor Canadi 已提交
864 865 866 867 868
  MergeHelper merge(
      env_, cfd->user_comparator(), cfd->ioptions()->merge_operator,
      compaction_filter, db_options_.info_log.get(),
      false /* internal key corruption is expected */,
      existing_snapshots_.empty() ? 0 : existing_snapshots_.back(),
869 870
      snapshot_checker_, compact_->compaction->level(),
      db_options_.statistics.get(), shutting_down_);
I
Igor Canadi 已提交
871

872
  TEST_SYNC_POINT("CompactionJob::Run():Inprogress");
873

874 875
  Slice* start = sub_compact->start;
  Slice* end = sub_compact->end;
876 877 878
  if (start != nullptr) {
    IterKey start_iter;
    start_iter.SetInternalKey(*start, kMaxSequenceNumber, kValueTypeForSeek);
879
    input->Seek(start_iter.GetInternalKey());
880 881 882 883
  } else {
    input->SeekToFirst();
  }

884 885 886
  Status status;
  sub_compact->c_iter.reset(new CompactionIterator(
      input.get(), cfd->user_comparator(), &merge, versions_->LastSequence(),
Y
Yi Wu 已提交
887
      &existing_snapshots_, earliest_write_conflict_snapshot_,
888
      snapshot_checker_, env_, ShouldReportDetailedTime(env_, stats_), false,
889
      &range_del_agg, sub_compact->compaction, compaction_filter,
890 891 892
      shutting_down_, preserve_deletes_seqnum_,
      // Currently range_del_agg is incompatible with snapshot refresh feature.
      range_del_agg.IsEmpty() ? snap_list_callback_ : nullptr));
893 894
  auto c_iter = sub_compact->c_iter.get();
  c_iter->SeekToFirst();
895
  if (c_iter->Valid() && sub_compact->compaction->output_level() != 0) {
896 897 898
    // ShouldStopBefore() maintains state based on keys processed so far. The
    // compaction loop always calls it on the "next" key, thus won't tell it the
    // first key. So we do that here.
899 900
    sub_compact->ShouldStopBefore(c_iter->key(),
                                  sub_compact->current_output_file_size);
901
  }
902
  const auto& c_iter_stats = c_iter->iter_stats();
903

904
  while (status.ok() && !cfd->IsDropped() && c_iter->Valid()) {
905 906 907 908
    // Invariant: c_iter.status() is guaranteed to be OK if c_iter->Valid()
    // returns true.
    const Slice& key = c_iter->key();
    const Slice& value = c_iter->value();
909

910 911
    // If an end key (exclusive) is specified, check if the current key is
    // >= than it and exit if it is because the iterator is out of its range
912
    if (end != nullptr &&
913
        cfd->user_comparator()->Compare(c_iter->user_key(), *end) >= 0) {
914
      break;
I
Igor Canadi 已提交
915
    }
916 917 918 919 920
    if (c_iter_stats.num_input_records % kRecordStatsEvery ==
        kRecordStatsEvery - 1) {
      RecordDroppedKeys(c_iter_stats, &sub_compact->compaction_job_stats);
      c_iter->ResetRecordCounts();
      RecordCompactionIOStats();
I
Igor Canadi 已提交
921 922
    }

923 924 925 926
    // Open output file if necessary
    if (sub_compact->builder == nullptr) {
      status = OpenCompactionOutputFile(sub_compact);
      if (!status.ok()) {
927 928
        break;
      }
929 930 931 932
    }
    assert(sub_compact->builder != nullptr);
    assert(sub_compact->current_output() != nullptr);
    sub_compact->builder->Add(key, value);
933
    sub_compact->current_output_file_size = sub_compact->builder->FileSize();
934 935 936 937
    sub_compact->current_output()->meta.UpdateBoundaries(
        key, c_iter->ikey().sequence);
    sub_compact->num_output_records++;

938 939 940 941
    // Close output file if it is big enough. Two possibilities determine it's
    // time to close it: (1) the current key should be this file's last key, (2)
    // the next key should not be in this file.
    //
942 943 944 945
    // TODO(aekmekji): determine if file should be closed earlier than this
    // during subcompactions (i.e. if output size, estimated by input size, is
    // going to be 1.2MB and max_output_file_size = 1MB, prefer to have 0.6MB
    // and 0.6MB instead of 1MB and 0.2MB)
946 947
    bool output_file_ended = false;
    Status input_status;
948 949 950
    if (sub_compact->compaction->output_level() != 0 &&
        sub_compact->current_output_file_size >=
            sub_compact->compaction->max_output_file_size()) {
951 952 953 954 955 956 957 958
      // (1) this key terminates the file. For historical reasons, the iterator
      // status before advancing will be given to FinishCompactionOutputFile().
      input_status = input->status();
      output_file_ended = true;
    }
    c_iter->Next();
    if (!output_file_ended && c_iter->Valid() &&
        sub_compact->compaction->output_level() != 0 &&
959 960
        sub_compact->ShouldStopBefore(c_iter->key(),
                                      sub_compact->current_output_file_size) &&
961 962 963 964 965 966 967 968
        sub_compact->builder != nullptr) {
      // (2) this key belongs to the next file. For historical reasons, the
      // iterator status after advancing will be given to
      // FinishCompactionOutputFile().
      input_status = input->status();
      output_file_ended = true;
    }
    if (output_file_ended) {
969 970 971 972
      const Slice* next_key = nullptr;
      if (c_iter->Valid()) {
        next_key = &c_iter->key();
      }
A
Andrew Kryczka 已提交
973
      CompactionIterationStats range_del_out_stats;
974
      status =
975
          FinishCompactionOutputFile(input_status, sub_compact, &range_del_agg,
976
                                     &range_del_out_stats, next_key);
A
Andrew Kryczka 已提交
977 978
      RecordDroppedKeys(range_del_out_stats,
                        &sub_compact->compaction_job_stats);
I
Igor Canadi 已提交
979 980
    }
  }
981

982 983 984 985 986
  sub_compact->num_input_records = c_iter_stats.num_input_records;
  sub_compact->compaction_job_stats.num_input_deletion_records =
      c_iter_stats.num_input_deletion_records;
  sub_compact->compaction_job_stats.num_corrupt_keys =
      c_iter_stats.num_input_corrupt_records;
987 988 989 990
  sub_compact->compaction_job_stats.num_single_del_fallthru =
      c_iter_stats.num_single_del_fallthru;
  sub_compact->compaction_job_stats.num_single_del_mismatch =
      c_iter_stats.num_single_del_mismatch;
991 992 993 994 995 996 997 998
  sub_compact->compaction_job_stats.total_input_raw_key_bytes +=
      c_iter_stats.total_input_raw_key_bytes;
  sub_compact->compaction_job_stats.total_input_raw_value_bytes +=
      c_iter_stats.total_input_raw_value_bytes;

  RecordTick(stats_, FILTER_OPERATION_TOTAL_TIME,
             c_iter_stats.total_filter_time);
  RecordDroppedKeys(c_iter_stats, &sub_compact->compaction_job_stats);
I
Igor Canadi 已提交
999 1000
  RecordCompactionIOStats();

1001 1002 1003 1004 1005 1006 1007
  if (status.ok() && cfd->IsDropped()) {
    status =
        Status::ColumnFamilyDropped("Column family dropped during compaction");
  }
  if ((status.ok() || status.IsColumnFamilyDropped()) &&
      shutting_down_->load(std::memory_order_relaxed)) {
    status = Status::ShutdownInProgress("Database shutdown");
1008
  }
1009 1010 1011 1012 1013 1014 1015
  if (status.ok()) {
    status = input->status();
  }
  if (status.ok()) {
    status = c_iter->status();
  }

1016
  if (status.ok() && sub_compact->builder == nullptr &&
1017
      sub_compact->outputs.size() == 0 && !range_del_agg.IsEmpty()) {
1018
    // handle subcompaction containing only range deletions
1019 1020
    status = OpenCompactionOutputFile(sub_compact);
  }
1021 1022 1023 1024

  // Call FinishCompactionOutputFile() even if status is not ok: it needs to
  // close the output file.
  if (sub_compact->builder != nullptr) {
A
Andrew Kryczka 已提交
1025
    CompactionIterationStats range_del_out_stats;
1026
    Status s = FinishCompactionOutputFile(status, sub_compact, &range_del_agg,
1027
                                          &range_del_out_stats);
1028 1029 1030
    if (status.ok()) {
      status = s;
    }
A
Andrew Kryczka 已提交
1031
    RecordDroppedKeys(range_del_out_stats, &sub_compact->compaction_job_stats);
1032 1033
  }

1034 1035 1036
  sub_compact->compaction_job_stats.cpu_micros =
      env_->NowCPUNanos() / 1000 - prev_cpu_micros;

1037 1038
  if (measure_io_stats_) {
    sub_compact->compaction_job_stats.file_write_nanos +=
I
Igor Canadi 已提交
1039
        IOSTATS(write_nanos) - prev_write_nanos;
1040
    sub_compact->compaction_job_stats.file_fsync_nanos +=
I
Igor Canadi 已提交
1041
        IOSTATS(fsync_nanos) - prev_fsync_nanos;
1042
    sub_compact->compaction_job_stats.file_range_sync_nanos +=
I
Igor Canadi 已提交
1043
        IOSTATS(range_sync_nanos) - prev_range_sync_nanos;
1044
    sub_compact->compaction_job_stats.file_prepare_write_nanos +=
I
Igor Canadi 已提交
1045
        IOSTATS(prepare_write_nanos) - prev_prepare_write_nanos;
1046
    sub_compact->compaction_job_stats.cpu_micros -=
1047 1048 1049
        (IOSTATS(cpu_write_nanos) - prev_cpu_write_nanos +
         IOSTATS(cpu_read_nanos) - prev_cpu_read_nanos) /
        1000;
1050
    if (prev_perf_level != PerfLevel::kEnableTimeAndCPUTimeExceptForMutex) {
1051 1052 1053 1054
      SetPerfLevel(prev_perf_level);
    }
  }

1055 1056
  sub_compact->c_iter.reset();
  input.reset();
1057
  sub_compact->status = status;
I
Igor Canadi 已提交
1058 1059
}

1060
void CompactionJob::RecordDroppedKeys(
A
Andrew Kryczka 已提交
1061
    const CompactionIterationStats& c_iter_stats,
1062
    CompactionJobStats* compaction_job_stats) {
1063 1064 1065
  if (c_iter_stats.num_record_drop_user > 0) {
    RecordTick(stats_, COMPACTION_KEY_DROP_USER,
               c_iter_stats.num_record_drop_user);
1066
  }
1067 1068 1069
  if (c_iter_stats.num_record_drop_hidden > 0) {
    RecordTick(stats_, COMPACTION_KEY_DROP_NEWER_ENTRY,
               c_iter_stats.num_record_drop_hidden);
1070
    if (compaction_job_stats) {
1071 1072
      compaction_job_stats->num_records_replaced +=
          c_iter_stats.num_record_drop_hidden;
1073 1074
    }
  }
1075 1076 1077
  if (c_iter_stats.num_record_drop_obsolete > 0) {
    RecordTick(stats_, COMPACTION_KEY_DROP_OBSOLETE,
               c_iter_stats.num_record_drop_obsolete);
1078
    if (compaction_job_stats) {
1079 1080
      compaction_job_stats->num_expired_deletion_records +=
          c_iter_stats.num_record_drop_obsolete;
1081
    }
1082
  }
A
Andrew Kryczka 已提交
1083 1084 1085 1086 1087 1088 1089 1090
  if (c_iter_stats.num_record_drop_range_del > 0) {
    RecordTick(stats_, COMPACTION_KEY_DROP_RANGE_DEL,
               c_iter_stats.num_record_drop_range_del);
  }
  if (c_iter_stats.num_range_del_drop_obsolete > 0) {
    RecordTick(stats_, COMPACTION_RANGE_DEL_DROP_OBSOLETE,
               c_iter_stats.num_range_del_drop_obsolete);
  }
1091 1092 1093 1094
  if (c_iter_stats.num_optimized_del_drop_obsolete > 0) {
    RecordTick(stats_, COMPACTION_OPTIMIZED_DEL_DROP_OBSOLETE,
               c_iter_stats.num_optimized_del_drop_obsolete);
  }
1095 1096
}

1097
Status CompactionJob::FinishCompactionOutputFile(
1098
    const Status& input_status, SubcompactionState* sub_compact,
1099
    CompactionRangeDelAggregator* range_del_agg,
A
Andrew Kryczka 已提交
1100
    CompactionIterationStats* range_del_out_stats,
1101
    const Slice* next_table_min_key /* = nullptr */) {
1102 1103
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_SYNC_FILE);
1104 1105 1106
  assert(sub_compact != nullptr);
  assert(sub_compact->outfile);
  assert(sub_compact->builder != nullptr);
1107
  assert(sub_compact->current_output() != nullptr);
I
Igor Canadi 已提交
1108

1109
  uint64_t output_number = sub_compact->current_output()->meta.fd.GetNumber();
I
Igor Canadi 已提交
1110 1111
  assert(output_number != 0);

1112 1113 1114
  ColumnFamilyData* cfd = sub_compact->compaction->column_family_data();
  const Comparator* ucmp = cfd->user_comparator();

I
Igor Canadi 已提交
1115
  // Check for iterator errors
1116
  Status s = input_status;
1117
  auto meta = &sub_compact->current_output()->meta;
1118
  assert(meta != nullptr);
1119
  if (s.ok()) {
1120
    Slice lower_bound_guard, upper_bound_guard;
Z
zhangjinpeng1987 已提交
1121
    std::string smallest_user_key;
1122
    const Slice *lower_bound, *upper_bound;
1123
    bool lower_bound_from_sub_compact = false;
1124 1125 1126 1127
    if (sub_compact->outputs.size() == 1) {
      // For the first output table, include range tombstones before the min key
      // but after the subcompaction boundary.
      lower_bound = sub_compact->start;
1128
      lower_bound_from_sub_compact = true;
1129 1130 1131 1132
    } else if (meta->smallest.size() > 0) {
      // For subsequent output tables, only include range tombstones from min
      // key onwards since the previous file was extended to contain range
      // tombstones falling before min key.
Z
zhangjinpeng1987 已提交
1133 1134
      smallest_user_key = meta->smallest.user_key().ToString(false /*hex*/);
      lower_bound_guard = Slice(smallest_user_key);
1135 1136 1137 1138 1139
      lower_bound = &lower_bound_guard;
    } else {
      lower_bound = nullptr;
    }
    if (next_table_min_key != nullptr) {
1140 1141 1142 1143 1144 1145
      // This may be the last file in the subcompaction in some cases, so we
      // need to compare the end key of subcompaction with the next file start
      // key. When the end key is chosen by the subcompaction, we know that
      // it must be the biggest key in output file. Therefore, it is safe to
      // use the smaller key as the upper bound of the output file, to ensure
      // that there is no overlapping between different output files.
1146
      upper_bound_guard = ExtractUserKey(*next_table_min_key);
1147 1148 1149 1150 1151 1152
      if (sub_compact->end != nullptr &&
          ucmp->Compare(upper_bound_guard, *sub_compact->end) >= 0) {
        upper_bound = sub_compact->end;
      } else {
        upper_bound = &upper_bound_guard;
      }
1153 1154 1155 1156 1157
    } else {
      // This is the last file in the subcompaction, so extend until the
      // subcompaction ends.
      upper_bound = sub_compact->end;
    }
1158 1159 1160 1161
    auto earliest_snapshot = kMaxSequenceNumber;
    if (existing_snapshots_.size() > 0) {
      earliest_snapshot = existing_snapshots_[0];
    }
1162 1163 1164 1165 1166 1167 1168
    bool has_overlapping_endpoints;
    if (upper_bound != nullptr && meta->largest.size() > 0) {
      has_overlapping_endpoints =
          ucmp->Compare(meta->largest.user_key(), *upper_bound) == 0;
    } else {
      has_overlapping_endpoints = false;
    }
1169

1170 1171 1172 1173 1174 1175 1176
    // The end key of the subcompaction must be bigger or equal to the upper
    // bound. If the end of subcompaction is null or the upper bound is null,
    // it means that this file is the last file in the compaction. So there
    // will be no overlapping between this file and others.
    assert(sub_compact->end == nullptr ||
           upper_bound == nullptr ||
           ucmp->Compare(*upper_bound , *sub_compact->end) <= 0);
1177 1178 1179 1180 1181 1182 1183 1184 1185 1186
    auto it = range_del_agg->NewIterator(lower_bound, upper_bound,
                                         has_overlapping_endpoints);
    // Position the range tombstone output iterator. There may be tombstone
    // fragments that are entirely out of range, so make sure that we do not
    // include those.
    if (lower_bound != nullptr) {
      it->Seek(*lower_bound);
    } else {
      it->SeekToFirst();
    }
1187 1188
    for (; it->Valid(); it->Next()) {
      auto tombstone = it->Tombstone();
1189 1190 1191 1192 1193 1194 1195 1196 1197 1198 1199 1200
      if (upper_bound != nullptr) {
        int cmp = ucmp->Compare(*upper_bound, tombstone.start_key_);
        if ((has_overlapping_endpoints && cmp < 0) ||
            (!has_overlapping_endpoints && cmp <= 0)) {
          // Tombstones starting after upper_bound only need to be included in
          // the next table. If the current SST ends before upper_bound, i.e.,
          // `has_overlapping_endpoints == false`, we can also skip over range
          // tombstones that start exactly at upper_bound. Such range tombstones
          // will be included in the next file and are not relevant to the point
          // keys or endpoints of the current file.
          break;
        }
1201 1202 1203 1204 1205 1206 1207 1208 1209 1210 1211
      }

      if (bottommost_level_ && tombstone.seq_ <= earliest_snapshot) {
        // TODO(andrewkr): tombstones that span multiple output files are
        // counted for each compaction output file, so lots of double counting.
        range_del_out_stats->num_range_del_drop_obsolete++;
        range_del_out_stats->num_record_drop_obsolete++;
        continue;
      }

      auto kv = tombstone.Serialize();
1212 1213
      assert(lower_bound == nullptr ||
             ucmp->Compare(*lower_bound, kv.second) < 0);
1214 1215 1216 1217 1218 1219 1220 1221
      sub_compact->builder->Add(kv.first.Encode(), kv.second);
      InternalKey smallest_candidate = std::move(kv.first);
      if (lower_bound != nullptr &&
          ucmp->Compare(smallest_candidate.user_key(), *lower_bound) <= 0) {
        // Pretend the smallest key has the same user key as lower_bound
        // (the max key in the previous table or subcompaction) in order for
        // files to appear key-space partitioned.
        //
1222 1223 1224 1225 1226 1227 1228 1229 1230 1231 1232 1233 1234 1235 1236 1237 1238 1239
        // When lower_bound is chosen by a subcompaction, we know that
        // subcompactions over smaller keys cannot contain any keys at
        // lower_bound. We also know that smaller subcompactions exist, because
        // otherwise the subcompaction woud be unbounded on the left. As a
        // result, we know that no other files on the output level will contain
        // actual keys at lower_bound (an output file may have a largest key of
        // lower_bound@kMaxSequenceNumber, but this only indicates a large range
        // tombstone was truncated). Therefore, it is safe to use the
        // tombstone's sequence number, to ensure that keys at lower_bound at
        // lower levels are covered by truncated tombstones.
        //
        // If lower_bound was chosen by the smallest data key in the file,
        // choose lowest seqnum so this file's smallest internal key comes after
        // the previous file's largest. The fake seqnum is OK because the read
        // path's file-picking code only considers user key.
        smallest_candidate = InternalKey(
            *lower_bound, lower_bound_from_sub_compact ? tombstone.seq_ : 0,
            kTypeRangeDeletion);
1240 1241 1242 1243 1244 1245 1246 1247 1248 1249 1250 1251 1252 1253 1254 1255 1256 1257 1258 1259 1260
      }
      InternalKey largest_candidate = tombstone.SerializeEndKey();
      if (upper_bound != nullptr &&
          ucmp->Compare(*upper_bound, largest_candidate.user_key()) <= 0) {
        // Pretend the largest key has the same user key as upper_bound (the
        // min key in the following table or subcompaction) in order for files
        // to appear key-space partitioned.
        //
        // Choose highest seqnum so this file's largest internal key comes
        // before the next file's/subcompaction's smallest. The fake seqnum is
        // OK because the read path's file-picking code only considers the user
        // key portion.
        //
        // Note Seek() also creates InternalKey with (user_key,
        // kMaxSequenceNumber), but with kTypeDeletion (0x7) instead of
        // kTypeRangeDeletion (0xF), so the range tombstone comes before the
        // Seek() key in InternalKey's ordering. So Seek() will look in the
        // next file for the user key.
        largest_candidate =
            InternalKey(*upper_bound, kMaxSequenceNumber, kTypeRangeDeletion);
      }
1261 1262 1263 1264 1265 1266
#ifndef NDEBUG
      SequenceNumber smallest_ikey_seqnum = kMaxSequenceNumber;
      if (meta->smallest.size() > 0) {
        smallest_ikey_seqnum = GetInternalKeySeqno(meta->smallest.Encode());
      }
#endif
1267 1268 1269
      meta->UpdateBoundariesForRange(smallest_candidate, largest_candidate,
                                     tombstone.seq_,
                                     cfd->internal_comparator());
1270 1271 1272 1273 1274 1275 1276 1277

      // The smallest key in a file is used for range tombstone truncation, so
      // it cannot have a seqnum of 0 (unless the smallest data key in a file
      // has a seqnum of 0). Otherwise, the truncated tombstone may expose
      // deleted keys at lower levels.
      assert(smallest_ikey_seqnum == 0 ||
             ExtractInternalKeyFooter(meta->smallest.Encode()) !=
                 PackSequenceAndType(0, kTypeRangeDeletion));
1278
    }
S
Siying Dong 已提交
1279
    meta->marked_for_compaction = sub_compact->builder->NeedCompact();
1280
  }
1281
  const uint64_t current_entries = sub_compact->builder->NumEntries();
I
Igor Canadi 已提交
1282
  if (s.ok()) {
1283
    s = sub_compact->builder->Finish();
I
Igor Canadi 已提交
1284
  } else {
1285
    sub_compact->builder->Abandon();
I
Igor Canadi 已提交
1286
  }
1287
  const uint64_t current_bytes = sub_compact->builder->FileSize();
S
Siying Dong 已提交
1288 1289 1290
  if (s.ok()) {
    meta->fd.file_size = current_bytes;
  }
1291
  sub_compact->current_output()->finished = true;
1292
  sub_compact->total_bytes += current_bytes;
I
Igor Canadi 已提交
1293 1294

  // Finish and check for file errors
S
Sagar Vemuri 已提交
1295
  if (s.ok()) {
1296
    StopWatch sw(env_, stats_, COMPACTION_OUTFILE_SYNC_MICROS);
1297
    s = sub_compact->outfile->Sync(db_options_.use_fsync);
I
Igor Canadi 已提交
1298 1299
  }
  if (s.ok()) {
1300
    s = sub_compact->outfile->Close();
I
Igor Canadi 已提交
1301
  }
1302
  sub_compact->outfile.reset();
I
Igor Canadi 已提交
1303

1304 1305 1306 1307 1308 1309
  TableProperties tp;
  if (s.ok()) {
    tp = sub_compact->builder->GetTableProperties();
  }

  if (s.ok() && current_entries == 0 && tp.num_range_deletions == 0) {
Z
zhangjinpeng1987 已提交
1310 1311 1312
    // If there is nothing to output, no necessary to generate a sst file.
    // This happens when the output level is bottom level, at the same time
    // the sub_compact output nothing.
1313 1314 1315
    std::string fname =
        TableFileName(sub_compact->compaction->immutable_cf_options()->cf_paths,
                      meta->fd.GetNumber(), meta->fd.GetPathId());
Z
zhangjinpeng1987 已提交
1316 1317 1318 1319 1320 1321
    env_->DeleteFile(fname);

    // Also need to remove the file from outputs, or it will be added to the
    // VersionEdit.
    assert(!sub_compact->outputs.empty());
    sub_compact->outputs.pop_back();
1322
    meta = nullptr;
Z
zhangjinpeng1987 已提交
1323 1324
  }

1325
  if (s.ok() && (current_entries > 0 || tp.num_range_deletions > 0)) {
1326
    // Output to event logger and fire events.
1327 1328 1329 1330 1331 1332 1333 1334
    sub_compact->current_output()->table_properties =
        std::make_shared<TableProperties>(tp);
    ROCKS_LOG_INFO(db_options_.info_log,
                   "[%s] [JOB %d] Generated table #%" PRIu64 ": %" PRIu64
                   " keys, %" PRIu64 " bytes%s",
                   cfd->GetName().c_str(), job_id_, output_number,
                   current_entries, current_bytes,
                   meta->marked_for_compaction ? " (need compaction)" : "");
I
Igor Canadi 已提交
1335
  }
S
Siying Dong 已提交
1336 1337 1338
  std::string fname;
  FileDescriptor output_fd;
  if (meta != nullptr) {
1339 1340 1341
    fname =
        TableFileName(sub_compact->compaction->immutable_cf_options()->cf_paths,
                      meta->fd.GetNumber(), meta->fd.GetPathId());
S
Siying Dong 已提交
1342 1343 1344 1345
    output_fd = meta->fd;
  } else {
    fname = "(nil)";
  }
1346 1347
  EventHelpers::LogAndNotifyTableFileCreationFinished(
      event_logger_, cfd->ioptions()->listeners, dbname_, cfd->GetName(), fname,
S
Siying Dong 已提交
1348
      job_id_, output_fd, tp, TableFileCreationReason::kCompaction, s);
1349

1350
#ifndef ROCKSDB_LITE
1351 1352 1353
  // Report new file to SstFileManagerImpl
  auto sfm =
      static_cast<SstFileManagerImpl*>(db_options_.sst_file_manager.get());
S
Siying Dong 已提交
1354
  if (sfm && meta != nullptr && meta->fd.GetPathId() == 0) {
1355
    sfm->OnAddFile(fname);
1356
    if (sfm->IsMaxAllowedSpaceReached()) {
1357 1358
      // TODO(ajkr): should we return OK() if max space was reached by the final
      // compaction output file (similarly to how flush works when full)?
1359
      s = Status::SpaceLimit("Max allowed space was reached");
1360 1361 1362
      TEST_SYNC_POINT(
          "CompactionJob::FinishCompactionOutputFile:"
          "MaxAllowedSpaceReached");
1363
      InstrumentedMutexLock l(db_mutex_);
1364
      db_error_handler_->SetBGError(s, BackgroundErrorReason::kCompaction);
1365 1366
    }
  }
1367
#endif
1368

1369
  sub_compact->builder.reset();
1370
  sub_compact->current_output_file_size = 0;
I
Igor Canadi 已提交
1371 1372 1373
  return s;
}

I
Igor Canadi 已提交
1374
Status CompactionJob::InstallCompactionResults(
1375 1376
    const MutableCFOptions& mutable_cf_options) {
  db_mutex_->AssertHeld();
I
Igor Canadi 已提交
1377

1378
  auto* compaction = compact_->compaction;
I
Igor Canadi 已提交
1379 1380 1381 1382
  // paranoia: verify that the files that we started with
  // still exist in the current version and in the same original level.
  // This ensures that a concurrent compaction did not erroneously
  // pick the same files to compact_.
1383 1384 1385
  if (!versions_->VerifyCompactionFileConsistency(compaction)) {
    Compaction::InputLevelSummaryBuffer inputs_summary;

1386 1387 1388
    ROCKS_LOG_ERROR(db_options_.info_log, "[%s] [JOB %d] Compaction %s aborted",
                    compaction->column_family_data()->GetName().c_str(),
                    job_id_, compaction->InputLevelSummary(&inputs_summary));
I
Igor Canadi 已提交
1389 1390 1391
    return Status::Corruption("Compaction input files inconsistent");
  }

1392 1393
  {
    Compaction::InputLevelSummaryBuffer inputs_summary;
1394 1395
    ROCKS_LOG_INFO(
        db_options_.info_log, "[%s] [JOB %d] Compacted %s => %" PRIu64 " bytes",
1396 1397 1398
        compaction->column_family_data()->GetName().c_str(), job_id_,
        compaction->InputLevelSummary(&inputs_summary), compact_->total_bytes);
  }
I
Igor Canadi 已提交
1399

G
Gihwan Oh 已提交
1400
  // Add compaction inputs
1401
  compaction->AddInputDeletions(compact_->compaction->edit());
1402

1403 1404 1405
  for (const auto& sub_compact : compact_->sub_compact_states) {
    for (const auto& out : sub_compact.outputs) {
      compaction->edit()->AddFile(compaction->output_level(), out.meta);
1406
    }
1407 1408
  }
  return versions_->LogAndApply(compaction->column_family_data(),
I
Igor Canadi 已提交
1409
                                mutable_cf_options, compaction->edit(),
1410
                                db_mutex_, db_directory_);
I
Igor Canadi 已提交
1411 1412 1413 1414
}

void CompactionJob::RecordCompactionIOStats() {
  RecordTick(stats_, COMPACT_READ_BYTES, IOSTATS(bytes_read));
1415 1416
  ThreadStatusUtil::IncreaseThreadOperationProperty(
      ThreadStatus::COMPACTION_BYTES_READ, IOSTATS(bytes_read));
I
Igor Canadi 已提交
1417 1418
  IOSTATS_RESET(bytes_read);
  RecordTick(stats_, COMPACT_WRITE_BYTES, IOSTATS(bytes_written));
1419 1420
  ThreadStatusUtil::IncreaseThreadOperationProperty(
      ThreadStatus::COMPACTION_BYTES_WRITTEN, IOSTATS(bytes_written));
I
Igor Canadi 已提交
1421 1422 1423
  IOSTATS_RESET(bytes_written);
}

1424 1425
Status CompactionJob::OpenCompactionOutputFile(
    SubcompactionState* sub_compact) {
1426 1427
  assert(sub_compact != nullptr);
  assert(sub_compact->builder == nullptr);
1428 1429
  // no need to lock because VersionSet::next_file_number_ is atomic
  uint64_t file_number = versions_->NewFileNumber();
1430 1431 1432
  std::string fname =
      TableFileName(sub_compact->compaction->immutable_cf_options()->cf_paths,
                    file_number, sub_compact->compaction->output_path_id());
1433 1434 1435 1436 1437 1438 1439 1440
  // Fire events.
  ColumnFamilyData* cfd = sub_compact->compaction->column_family_data();
#ifndef ROCKSDB_LITE
  EventHelpers::NotifyTableFileCreationStarted(
      cfd->ioptions()->listeners, dbname_, cfd->GetName(), fname, job_id_,
      TableFileCreationReason::kCompaction);
#endif  // !ROCKSDB_LITE
  // Make the output file
1441
  std::unique_ptr<WritableFile> writable_file;
A
Aliaksei Sandryhaila 已提交
1442
#ifndef NDEBUG
1443
  bool syncpoint_arg = env_options_.use_direct_writes;
1444
  TEST_SYNC_POINT_CALLBACK("CompactionJob::OpenCompactionOutputFile",
1445
                           &syncpoint_arg);
A
Aliaksei Sandryhaila 已提交
1446
#endif
1447
  Status s = NewWritableFile(env_, fname, &writable_file, env_options_);
I
Igor Canadi 已提交
1448
  if (!s.ok()) {
1449 1450
    ROCKS_LOG_ERROR(
        db_options_.info_log,
1451
        "[%s] [JOB %d] OpenCompactionOutputFiles for table #%" PRIu64
1452
        " fails at NewWritableFile with status %s",
1453 1454
        sub_compact->compaction->column_family_data()->GetName().c_str(),
        job_id_, file_number, s.ToString().c_str());
I
Igor Canadi 已提交
1455
    LogFlush(db_options_.info_log);
1456 1457 1458 1459
    EventHelpers::LogAndNotifyTableFileCreationFinished(
        event_logger_, cfd->ioptions()->listeners, dbname_, cfd->GetName(),
        fname, job_id_, FileDescriptor(), TableProperties(),
        TableFileCreationReason::kCompaction, s);
I
Igor Canadi 已提交
1460 1461
    return s;
  }
1462

1463
  SubcompactionState::Output out;
1464 1465 1466
  out.meta.fd =
      FileDescriptor(file_number, sub_compact->compaction->output_path_id(), 0);
  out.finished = false;
I
Igor Canadi 已提交
1467

1468
  sub_compact->outputs.push_back(out);
1469
  writable_file->SetIOPriority(Env::IO_LOW);
S
Stream  
Shaohua Li 已提交
1470
  writable_file->SetWriteLifeTimeHint(write_hint_);
1471 1472
  writable_file->SetPreallocationBlockSize(static_cast<size_t>(
      sub_compact->compaction->OutputFilePreallocationSize()));
1473 1474
  const auto& listeners =
      sub_compact->compaction->immutable_cf_options()->listeners;
1475 1476
  sub_compact->outfile.reset(
      new WritableFileWriter(std::move(writable_file), fname, env_options_,
1477
                             env_, db_options_.statistics.get(), listeners));
I
Igor Canadi 已提交
1478

1479 1480 1481
  // If the Column family flag is to only optimize filters for hits,
  // we can skip creating filters if this is the bottommost_level where
  // data is going to be found
1482 1483
  bool skip_filters =
      cfd->ioptions()->optimize_filters_for_hits && bottommost_level_;
S
Sagar Vemuri 已提交
1484

S
Sagar Vemuri 已提交
1485 1486 1487 1488 1489 1490 1491 1492 1493 1494 1495
  int64_t temp_current_time = 0;
  auto get_time_status = env_->GetCurrentTime(&temp_current_time);
  // Safe to proceed even if GetCurrentTime fails. So, log and proceed.
  if (!get_time_status.ok()) {
    ROCKS_LOG_WARN(db_options_.info_log,
                   "Failed to get current time. Status: %s",
                   get_time_status.ToString().c_str());
  }
  uint64_t current_time = static_cast<uint64_t>(temp_current_time);

  uint64_t latest_key_time =
S
Sagar Vemuri 已提交
1496
      sub_compact->compaction->MaxInputFileCreationTime();
S
Sagar Vemuri 已提交
1497 1498
  if (latest_key_time == 0) {
    latest_key_time = current_time;
S
Sagar Vemuri 已提交
1499 1500
  }

1501
  sub_compact->builder.reset(NewTableBuilder(
1502 1503 1504 1505
      *cfd->ioptions(), *(sub_compact->compaction->mutable_cf_options()),
      cfd->internal_comparator(), cfd->int_tbl_prop_collector_factories(),
      cfd->GetID(), cfd->GetName(), sub_compact->outfile.get(),
      sub_compact->compaction->output_compression(),
1506
      0 /*sample_for_compression */,
1507
      sub_compact->compaction->output_compression_opts(),
S
Sagar Vemuri 已提交
1508 1509 1510
      sub_compact->compaction->output_level(), skip_filters, latest_key_time,
      0 /* oldest_key_time */, sub_compact->compaction->max_output_file_size(),
      current_time));
I
Igor Canadi 已提交
1511 1512 1513 1514
  LogFlush(db_options_.info_log);
  return s;
}

1515
void CompactionJob::CleanupCompaction() {
1516
  for (SubcompactionState& sub_compact : compact_->sub_compact_states) {
1517
    const auto& sub_status = sub_compact.status;
I
Igor Canadi 已提交
1518

1519 1520 1521 1522 1523 1524 1525
    if (sub_compact.builder != nullptr) {
      // May happen if we get a shutdown call in the middle of compaction
      sub_compact.builder->Abandon();
      sub_compact.builder.reset();
    } else {
      assert(!sub_status.ok() || sub_compact.outfile == nullptr);
    }
1526
    for (const auto& out : sub_compact.outputs) {
1527 1528 1529
      // If this file was inserted into the table cache then remove
      // them here because this compaction was not committed.
      if (!sub_status.ok()) {
1530
        TableCache::Evict(table_cache_.get(), out.meta.fd.GetNumber());
1531
      }
I
Igor Canadi 已提交
1532 1533 1534 1535 1536 1537
    }
  }
  delete compact_;
  compact_ = nullptr;
}

1538 1539
#ifndef ROCKSDB_LITE
namespace {
1540
void CopyPrefix(const Slice& src, size_t prefix_length, std::string* dst) {
1541 1542 1543
  assert(prefix_length > 0);
  size_t length = src.size() > prefix_length ? prefix_length : src.size();
  dst->assign(src.data(), length);
1544 1545 1546 1547 1548
}
}  // namespace

#endif  // !ROCKSDB_LITE

A
Andres Notzli 已提交
1549
void CompactionJob::UpdateCompactionStats() {
1550 1551 1552 1553 1554 1555
  Compaction* compaction = compact_->compaction;
  compaction_stats_.num_input_files_in_non_output_levels = 0;
  compaction_stats_.num_input_files_in_output_level = 0;
  for (int input_level = 0;
       input_level < static_cast<int>(compaction->num_input_levels());
       ++input_level) {
1556
    if (compaction->level(input_level) != compaction->output_level()) {
1557 1558
      UpdateCompactionInputStatsHelper(
          &compaction_stats_.num_input_files_in_non_output_levels,
1559
          &compaction_stats_.bytes_read_non_output_levels, input_level);
1560 1561 1562
    } else {
      UpdateCompactionInputStatsHelper(
          &compaction_stats_.num_input_files_in_output_level,
1563
          &compaction_stats_.bytes_read_output_level, input_level);
1564 1565
    }
  }
A
Andres Notzli 已提交
1566

1567 1568 1569 1570 1571 1572 1573 1574 1575
  for (const auto& sub_compact : compact_->sub_compact_states) {
    size_t num_output_files = sub_compact.outputs.size();
    if (sub_compact.builder != nullptr) {
      // An error occurred so ignore the last output.
      assert(num_output_files > 0);
      --num_output_files;
    }
    compaction_stats_.num_output_files += static_cast<int>(num_output_files);

1576 1577
    for (const auto& out : sub_compact.outputs) {
      compaction_stats_.bytes_written += out.meta.fd.file_size;
1578 1579 1580 1581 1582
    }
    if (sub_compact.num_input_records > sub_compact.num_output_records) {
      compaction_stats_.num_dropped_records +=
          sub_compact.num_input_records - sub_compact.num_output_records;
    }
A
Andres Notzli 已提交
1583
  }
1584 1585
}

1586 1587 1588
void CompactionJob::UpdateCompactionInputStatsHelper(int* num_files,
                                                     uint64_t* bytes_read,
                                                     int input_level) {
1589 1590 1591 1592 1593 1594 1595 1596 1597 1598 1599 1600
  const Compaction* compaction = compact_->compaction;
  auto num_input_files = compaction->num_input_files(input_level);
  *num_files += static_cast<int>(num_input_files);

  for (size_t i = 0; i < num_input_files; ++i) {
    const auto* file_meta = compaction->input(input_level, i);
    *bytes_read += file_meta->fd.GetFileSize();
    compaction_stats_.num_input_records +=
        static_cast<uint64_t>(file_meta->num_entries);
  }
}

1601 1602 1603 1604 1605 1606 1607 1608
void CompactionJob::UpdateCompactionJobStats(
    const InternalStats::CompactionStats& stats) const {
#ifndef ROCKSDB_LITE
  if (compaction_job_stats_) {
    compaction_job_stats_->elapsed_micros = stats.micros;

    // input information
    compaction_job_stats_->total_input_bytes =
1609 1610
        stats.bytes_read_non_output_levels + stats.bytes_read_output_level;
    compaction_job_stats_->num_input_records = compact_->num_input_records;
1611
    compaction_job_stats_->num_input_files =
1612 1613
        stats.num_input_files_in_non_output_levels +
        stats.num_input_files_in_output_level;
1614
    compaction_job_stats_->num_input_files_at_output_level =
1615
        stats.num_input_files_in_output_level;
1616 1617 1618

    // output information
    compaction_job_stats_->total_output_bytes = stats.bytes_written;
1619
    compaction_job_stats_->num_output_records = compact_->num_output_records;
1620
    compaction_job_stats_->num_output_files = stats.num_output_files;
1621

1622
    if (compact_->NumOutputFiles() > 0U) {
1623 1624 1625 1626 1627 1628
      CopyPrefix(compact_->SmallestUserKey(),
                 CompactionJobStats::kMaxPrefixLength,
                 &compaction_job_stats_->smallest_output_key_prefix);
      CopyPrefix(compact_->LargestUserKey(),
                 CompactionJobStats::kMaxPrefixLength,
                 &compaction_job_stats_->largest_output_key_prefix);
1629 1630
    }
  }
1631 1632
#else
  (void)stats;
1633 1634 1635
#endif  // !ROCKSDB_LITE
}

1636 1637 1638 1639
void CompactionJob::LogCompaction() {
  Compaction* compaction = compact_->compaction;
  ColumnFamilyData* cfd = compaction->column_family_data();

A
Andres Notzli 已提交
1640 1641 1642 1643
  // Let's check if anything will get logged. Don't prepare all the info if
  // we're not logging
  if (db_options_.info_log_level <= InfoLogLevel::INFO_LEVEL) {
    Compaction::InputLevelSummaryBuffer inputs_summary;
1644 1645 1646 1647
    ROCKS_LOG_INFO(
        db_options_.info_log, "[%s] [JOB %d] Compacting %s, score %.2f",
        cfd->GetName().c_str(), job_id_,
        compaction->InputLevelSummary(&inputs_summary), compaction->score());
A
Andres Notzli 已提交
1648 1649
    char scratch[2345];
    compaction->Summary(scratch, sizeof(scratch));
1650 1651
    ROCKS_LOG_INFO(db_options_.info_log, "[%s] Compaction start summary: %s\n",
                   cfd->GetName().c_str(), scratch);
A
Andres Notzli 已提交
1652 1653
    // build event logger report
    auto stream = event_logger_->Log();
1654 1655 1656 1657
    stream << "job" << job_id_ << "event"
           << "compaction_started"
           << "compaction_reason"
           << GetCompactionReasonString(compaction->compaction_reason());
A
Andres Notzli 已提交
1658 1659 1660 1661 1662 1663 1664 1665 1666 1667 1668 1669 1670
    for (size_t i = 0; i < compaction->num_input_levels(); ++i) {
      stream << ("files_L" + ToString(compaction->level(i)));
      stream.StartArray();
      for (auto f : *compaction->inputs(i)) {
        stream << f->fd.GetNumber();
      }
      stream.EndArray();
    }
    stream << "score" << compaction->score() << "input_data_size"
           << compaction->CalculateTotalInputSize();
  }
}

I
Igor Canadi 已提交
1671
}  // namespace rocksdb