compaction_job.cc 66.6 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
//
// 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.

#include <algorithm>
11
#include <cinttypes>
12
#include <functional>
I
Igor Canadi 已提交
13
#include <list>
14 15
#include <memory>
#include <random>
16 17 18
#include <set>
#include <thread>
#include <utility>
19
#include <vector>
I
Igor Canadi 已提交
20 21

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

62
namespace ROCKSDB_NAMESPACE {
I
Igor Canadi 已提交
63

64 65 66 67 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
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 已提交
96 97
    case CompactionReason::kPeriodicCompaction:
      return "PeriodicCompaction";
98 99 100 101 102 103 104 105
    case CompactionReason::kNumOfReasons:
      // fall through
    default:
      assert(false);
      return "Invalid";
  }
}

106
// Maintains state for each sub-compaction
107
struct CompactionJob::SubcompactionState {
108
  const Compaction* compaction;
109
  std::unique_ptr<CompactionIterator> c_iter;
I
Igor Canadi 已提交
110

111 112 113 114 115
  // 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;

116
  // The return status of this subcompaction
117 118
  Status status;

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

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

143 144
  uint64_t current_output_file_size;

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

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

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

177
  SubcompactionState& operator=(SubcompactionState&& o) {
D
Dmitri Smirnov 已提交
178 179 180 181 182 183 184
    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);
185
    current_output_file_size = std::move(o.current_output_file_size);
D
Dmitri Smirnov 已提交
186 187
    total_bytes = std::move(o.total_bytes);
    num_output_records = std::move(o.num_output_records);
188 189
    compaction_job_stats = std::move(o.compaction_job_stats);
    approx_size = std::move(o.approx_size);
190 191 192
    grandparent_index = std::move(o.grandparent_index);
    overlapped_bytes = std::move(o.overlapped_bytes);
    seen_key = std::move(o.seen_key);
D
Dmitri Smirnov 已提交
193 194 195
    return *this;
  }

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

199
  SubcompactionState& operator=(const SubcompactionState&) = delete;
200 201 202

  // Returns true iff we should stop building the current output
  // before processing "internal_key".
203
  bool ShouldStopBefore(const Slice& internal_key, uint64_t curr_file_size) {
204 205 206 207 208 209 210 211 212 213 214 215 216 217 218
    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(),
219
                 grandparents[grandparent_index + 1]->smallest.Encode()) <= 0);
220 221 222 223
      grandparent_index++;
    }
    seen_key = true;

224 225
    if (overlapped_bytes + curr_file_size >
        compaction->max_compaction_bytes()) {
226 227 228 229 230 231 232
      // Too much overlap for current output; start new output
      overlapped_bytes = 0;
      return true;
    }

    return false;
  }
233 234 235 236 237
};

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

239 240
  // REQUIRED: subcompaction states are stored in order of increasing
  // key-range
241
  std::vector<CompactionJob::SubcompactionState> sub_compact_states;
242 243 244 245
  Status status;

  uint64_t total_bytes;
  uint64_t num_output_records;
I
Igor Canadi 已提交
246

S
sdong 已提交
247 248 249 250
  explicit CompactionState(Compaction* c)
      : compaction(c),
        total_bytes(0),
        num_output_records(0) {}
I
Igor Canadi 已提交
251

252 253 254 255 256 257 258 259 260
  size_t NumOutputFiles() {
    size_t total = 0;
    for (auto& s : sub_compact_states) {
      total += s.outputs.size();
    }
    return total;
  }

  Slice SmallestUserKey() {
261 262 263 264
    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();
265 266
      }
    }
267
    // If there is no finished output, return an empty slice.
268
    return Slice(nullptr, 0);
269 270 271
  }

  Slice LargestUserKey() {
272 273 274 275 276
    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();
277 278
      }
    }
279
    // If there is no finished output, return an empty slice.
280
    return Slice(nullptr, 0);
281
  }
I
Igor Canadi 已提交
282 283
};

284
void CompactionJob::AggregateStatistics() {
285
  for (SubcompactionState& sc : compact_->sub_compact_states) {
286 287
    compact_->total_bytes += sc.total_bytes;
    compact_->num_output_records += sc.num_output_records;
288 289 290
  }
  if (compaction_job_stats_) {
    for (SubcompactionState& sc : compact_->sub_compact_states) {
291 292 293 294 295
      compaction_job_stats_->Add(sc.compaction_job_stats);
    }
  }
}

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

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

356
void CompactionJob::ReportStartedCompaction(Compaction* compaction) {
357 358
  const auto* cfd = compact_->compaction->column_family_data();
  ThreadStatusUtil::SetColumnFamily(cfd, cfd->ioptions()->env,
Y
Yi Wu 已提交
359
                                    db_options_.enable_thread_tracking);
360

361 362
  ThreadStatusUtil::SetThreadOperationProperty(ThreadStatus::COMPACTION_JOB_ID,
                                               job_id_);
363 364 365 366 367 368

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

369 370 371
  // In the current design, a CompactionJob is always created
  // for non-trivial compaction.
  assert(compaction->IsTrivialMove() == false ||
372
         compaction->is_manual_compaction() == true);
373

374 375
  ThreadStatusUtil::SetThreadOperationProperty(
      ThreadStatus::COMPACTION_PROP_FLAGS,
S
sdong 已提交
376
      compaction->is_manual_compaction() +
377
          (compaction->deletion_compaction() << 1));
378 379 380 381 382 383 384 385 386 387 388 389 390 391

  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.
392
  ThreadStatusUtil::SetThreadOperation(ThreadStatus::OP_COMPACTION);
393 394 395

  if (compaction_job_stats_) {
    compaction_job_stats_->is_manual_compaction =
S
sdong 已提交
396
        compaction->is_manual_compaction();
397
  }
398 399
}

I
Igor Canadi 已提交
400
void CompactionJob::Prepare() {
401 402
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_PREPARE);
I
Igor Canadi 已提交
403 404

  // Generate file_levels_ for compaction berfore making Iterator
405 406
  auto* c = compact_->compaction;
  assert(c->column_family_data() != nullptr);
407 408
  assert(c->column_family_data()->current()->storage_info()->NumLevelFiles(
             compact_->compaction->level()) > 0);
I
Igor Canadi 已提交
409

410 411
  write_hint_ =
      c->column_family_data()->CalculateSSTWriteHint(c->output_level());
412
  bottommost_level_ = c->bottommost_level();
413

414
  if (c->ShouldFormSubcompactions()) {
S
Siying Dong 已提交
415 416 417 418
    {
      StopWatch sw(env_, stats_, SUBCOMPACTION_SETUP_TIME);
      GenSubcompactionBoundaries();
    }
419 420 421 422 423 424 425
    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 已提交
426 427
    RecordInHistogram(stats_, NUM_SUBCOMPACTIONS_SCHEDULED,
                      compact_->sub_compact_states.size());
428 429 430
  } else {
    compact_->sub_compact_states.emplace_back(c, nullptr, nullptr);
  }
431 432
}

433 434 435
struct RangeWithSize {
  Range range;
  uint64_t size;
436

437 438 439 440 441 442 443
  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();
444 445
  const Comparator* cfd_comparator = cfd->user_comparator();
  std::vector<Slice> bounds;
446 447 448 449
  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
450
  // boundary
451 452 453 454 455 456 457
  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) {
458
        continue;
459
      }
A
Ari Ekmekji 已提交
460

461 462 463 464
      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++) {
465 466
          bounds.emplace_back(flevel->files[i].smallest_key);
          bounds.emplace_back(flevel->files[i].largest_key);
467 468 469 470
        }
      } else {
        // For all other levels add the smallest/largest key in the level to
        // encompass the range covered by that level
471 472
        bounds.emplace_back(flevel->files[0].smallest_key);
        bounds.emplace_back(flevel->files[num_files - 1].largest_key);
473 474 475 476 477 478
        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++) {
479
            bounds.emplace_back(flevel->files[i].smallest_key);
A
Ari Ekmekji 已提交
480
          }
481 482 483 484
        }
      }
    }
  }
485

486
  std::sort(bounds.begin(), bounds.end(),
487 488 489 490
            [cfd_comparator](const Slice& a, const Slice& b) -> bool {
              return cfd_comparator->Compare(ExtractUserKey(a),
                                             ExtractUserKey(b)) < 0;
            });
491
  // Remove duplicated entries from bounds
492 493 494 495 496 497 498
  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());
499

500 501 502 503
  // 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;
504 505 506 507
  // 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();
508 509
  for (auto it = bounds.begin();;) {
    const Slice a = *it;
510
    ++it;
511 512 513 514 515 516

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

    const Slice b = *it;
517 518 519 520 521

    // 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();
522 523
    uint64_t size = versions_->ApproximateSize(SizeApproximationOptions(), v, a,
                                               b, start_lvl, out_lvl + 1,
524
                                               TableReaderCaller::kCompaction);
525
    db_mutex_->Lock();
526 527 528 529 530 531
    ranges.emplace_back(a, b, size);
    sum += size;
  }

  // Group the ranges into subcompactions
  const double min_file_fill_percent = 4.0 / 5;
532 533 534 535 536 537
  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)));
538 539
  uint64_t subcompactions =
      std::min({static_cast<uint64_t>(ranges.size()),
540
                static_cast<uint64_t>(c->max_subcompactions()),
541 542 543
                max_output_files});

  if (subcompactions > 1) {
544
    double mean = sum * 1.0 / subcompactions;
545 546 547
    // Greedily add ranges to the subcompaction until the sum of the ranges'
    // sizes becomes >= the expected mean size of a subcompaction
    sum = 0;
548
    for (size_t i = 0; i + 1 < ranges.size(); i++) {
549
      sum += ranges[i].size;
550 551 552
      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
553
        continue;
554 555 556 557 558 559 560 561 562 563 564 565
      }
      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);
566
  }
I
Igor Canadi 已提交
567 568 569
}

Status CompactionJob::Run() {
570 571
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_RUN);
572
  TEST_SYNC_POINT("CompactionJob::Run():Start");
I
Igor Canadi 已提交
573
  log_buffer_->FlushBufferToLog();
574
  LogCompaction();
575

576 577
  const size_t num_threads = compact_->sub_compact_states.size();
  assert(num_threads > 0);
578
  const uint64_t start_micros = env_->NowMicros();
579 580

  // Launch a thread for each of subcompactions 1...num_threads-1
D
Dmitri Smirnov 已提交
581
  std::vector<port::Thread> thread_pool;
582 583 584 585
  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]);
586
  }
587 588 589 590 591 592 593 594 595 596

  // 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();
  }

597
  compaction_stats_.micros = env_->NowMicros() - start_micros;
598 599 600 601 602 603
  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 已提交
604 605 606
  RecordTimeToHistogram(stats_, COMPACTION_TIME, compaction_stats_.micros);
  RecordTimeToHistogram(stats_, COMPACTION_CPU_TIME,
                        compaction_stats_.cpu_micros);
607

608 609
  TEST_SYNC_POINT("CompactionJob::Run:BeforeVerify");

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

619
  IOStatus io_s;
620
  if (status.ok() && output_directory_) {
621 622 623 624 625
    io_s = output_directory_->Fsync(IOOptions(), nullptr);
  }
  if (!io_s.ok()) {
    io_status_ = io_s;
    status = io_s;
626 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(
653
            ReadOptions(), file_options_, cfd->internal_comparator(),
654 655
            *files_meta[file_idx], /*range_del_agg=*/nullptr, prefix_extractor,
            /*table_reader_ptr=*/nullptr,
656 657
            cfd->internal_stats()->GetFileReadHist(
                compact_->compaction->output_level()),
658 659
            TableReaderCaller::kCompactionRefill, /*arena=*/nullptr,
            /*skip_filters=*/false, compact_->compaction->output_level(),
660 661
            MaxFileSizeForL0MetaPin(
                *compact_->compaction->mutable_cf_options()),
662
            /*smallest_compaction_key=*/nullptr,
663 664
            /*largest_compaction_key=*/nullptr,
            /*allow_unprepared_value=*/false);
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 690 691 692 693 694 695
        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;
      }
    }
  }

696 697 698
  TablePropertiesCollection tp;
  for (const auto& state : compact_->sub_compact_states) {
    for (const auto& output : state.outputs) {
699 700 701
      auto fn =
          TableFileName(state.compaction->immutable_cf_options()->cf_paths,
                        output.meta.fd.GetNumber(), output.meta.fd.GetPathId());
702 703 704 705 706
      tp[fn] = output.table_properties;
    }
  }
  compact_->compaction->SetOutputTableProperties(std::move(tp));

707 708
  // Finish up all book-keeping to unify the subcompaction results
  AggregateStatistics();
709 710 711 712 713
  UpdateCompactionStats();
  RecordCompactionIOStats();
  LogFlush(db_options_.info_log);
  TEST_SYNC_POINT("CompactionJob::Run():End");

714
  compact_->status = status;
715 716 717
  return status;
}

718
Status CompactionJob::Install(const MutableCFOptions& mutable_cf_options) {
719 720
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_INSTALL);
721
  db_mutex_->AssertHeld();
722
  Status status = compact_->status;
I
Igor Canadi 已提交
723 724
  ColumnFamilyData* cfd = compact_->compaction->column_family_data();
  cfd->internal_stats()->AddCompactionStats(
725
      compact_->compaction->output_level(), thread_pri_, compaction_stats_);
I
Igor Canadi 已提交
726

727
  if (status.ok()) {
728
    status = InstallCompactionResults(mutable_cf_options);
I
Igor Canadi 已提交
729
  }
730 731 732
  if (!versions_->io_status().ok()) {
    io_status_ = versions_->io_status();
  }
I
Igor Canadi 已提交
733
  VersionStorageInfo::LevelSummaryStorage tmp;
734
  auto vstorage = cfd->current()->storage_info();
I
Igor Canadi 已提交
735
  const auto& stats = compaction_stats_;
736 737 738

  double read_write_amp = 0.0;
  double write_amp = 0.0;
739 740 741
  double bytes_read_per_sec = 0;
  double bytes_written_per_sec = 0;

742 743 744 745 746 747 748
  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);
  }
749 750 751 752 753 754 755 756
  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);
  }

757
  ROCKS_LOG_BUFFER(
758 759 760 761
      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 已提交
762 763
      "write-amplify(%.1f) %s, records in: %" PRIu64
      ", records dropped: %" PRIu64 " output_compression: %s\n",
764 765
      cfd->GetName().c_str(), vstorage->LevelSummary(&tmp), bytes_read_per_sec,
      bytes_written_per_sec, compact_->compaction->output_level(),
766
      stats.num_input_files_in_non_output_levels,
767
      stats.num_input_files_in_output_level, stats.num_output_files,
768 769
      stats.bytes_read_non_output_levels / 1048576.0,
      stats.bytes_read_output_level / 1048576.0,
770
      stats.bytes_written / 1048576.0, read_write_amp, write_amp,
771
      status.ToString().c_str(), stats.num_input_records,
772 773 774
      stats.num_dropped_records,
      CompressionTypeToString(compact_->compaction->output_compression())
          .c_str());
I
Igor Canadi 已提交
775

776 777
  UpdateCompactionJobStats(stats);

778
  auto stream = event_logger_->LogToBuffer(log_buffer_);
779 780
  stream << "job" << job_id_ << "event"
         << "compaction_finished"
781 782 783 784 785 786
         << "compaction_time_micros" << stats.micros
         << "compaction_time_cpu_micros" << stats.cpu_micros << "output_level"
         << compact_->compaction->output_level() << "num_output_files"
         << compact_->NumOutputFiles() << "total_output_size"
         << compact_->total_bytes << "num_input_records"
         << stats.num_input_records << "num_output_records"
787 788 789
         << compact_->num_output_records << "num_subcompactions"
         << compact_->sub_compact_states.size() << "output_compression"
         << CompressionTypeToString(compact_->compaction->output_compression());
790

791 792 793 794 795 796 797
  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;
  }

798 799 800 801 802 803 804 805 806
  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;
  }

807 808 809 810 811 812 813
  stream << "lsm_state";
  stream.StartArray();
  for (int level = 0; level < vstorage->num_levels(); ++level) {
    stream << vstorage->NumLevelFiles(level);
  }
  stream.EndArray();

814 815
  CleanupCompaction();
  return status;
I
Igor Canadi 已提交
816 817
}

818
void CompactionJob::ProcessKeyValueCompaction(SubcompactionState* sub_compact) {
819
  assert(sub_compact != nullptr);
820 821 822

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

823
  ColumnFamilyData* cfd = sub_compact->compaction->column_family_data();
824 825 826 827 828 829 830 831 832 833 834 835 836 837 838 839 840 841

  // 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;
  }

842 843
  CompactionRangeDelAggregator range_del_agg(&cfd->internal_comparator(),
                                             existing_snapshots_);
844 845 846

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

850 851
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_PROCESS_KV);
852 853 854

  // I/O measurement variables
  PerfLevel prev_perf_level = PerfLevel::kEnableTime;
855
  const uint64_t kRecordStatsEvery = 1000;
856 857 858 859
  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;
860 861
  uint64_t prev_cpu_write_nanos = 0;
  uint64_t prev_cpu_read_nanos = 0;
862 863
  if (measure_io_stats_) {
    prev_perf_level = GetPerfLevel();
864
    SetPerfLevel(PerfLevel::kEnableTimeAndCPUTimeExceptForMutex);
I
Igor Canadi 已提交
865 866 867 868
    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);
869 870
    prev_cpu_write_nanos = IOSTATS(cpu_write_nanos);
    prev_cpu_read_nanos = IOSTATS(cpu_read_nanos);
871 872
  }

I
Igor Canadi 已提交
873 874 875 876 877
  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(),
878
      snapshot_checker_, compact_->compaction->level(),
879
      db_options_.statistics.get());
I
Igor Canadi 已提交
880

881
  TEST_SYNC_POINT("CompactionJob::Run():Inprogress");
882 883 884 885
  TEST_SYNC_POINT_CALLBACK(
      "CompactionJob::Run():PausingManualCompaction:1",
      reinterpret_cast<void*>(
          const_cast<std::atomic<bool>*>(manual_compaction_paused_)));
886

887 888
  Slice* start = sub_compact->start;
  Slice* end = sub_compact->end;
889 890 891
  if (start != nullptr) {
    IterKey start_iter;
    start_iter.SetInternalKey(*start, kMaxSequenceNumber, kValueTypeForSeek);
892
    input->Seek(start_iter.GetInternalKey());
893 894 895 896
  } else {
    input->SeekToFirst();
  }

897 898 899
  Status status;
  sub_compact->c_iter.reset(new CompactionIterator(
      input.get(), cfd->user_comparator(), &merge, versions_->LastSequence(),
Y
Yi Wu 已提交
900
      &existing_snapshots_, earliest_write_conflict_snapshot_,
901 902 903 904
      snapshot_checker_, env_, ShouldReportDetailedTime(env_, stats_),
      /*expect_valid_internal_key=*/true, &range_del_agg,
      sub_compact->compaction, compaction_filter, shutting_down_,
      preserve_deletes_seqnum_, manual_compaction_paused_,
905
      db_options_.info_log));
906 907
  auto c_iter = sub_compact->c_iter.get();
  c_iter->SeekToFirst();
908
  if (c_iter->Valid() && sub_compact->compaction->output_level() != 0) {
909 910 911
    // 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.
912 913
    sub_compact->ShouldStopBefore(c_iter->key(),
                                  sub_compact->current_output_file_size);
914
  }
915
  const auto& c_iter_stats = c_iter->iter_stats();
916

917
  while (status.ok() && !cfd->IsDropped() && c_iter->Valid()) {
918 919 920 921
    // 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();
922

923 924
    // 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
925
    if (end != nullptr &&
926
        cfd->user_comparator()->Compare(c_iter->user_key(), *end) >= 0) {
927
      break;
I
Igor Canadi 已提交
928
    }
929 930 931 932 933
    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 已提交
934 935
    }

936 937 938 939
    // Open output file if necessary
    if (sub_compact->builder == nullptr) {
      status = OpenCompactionOutputFile(sub_compact);
      if (!status.ok()) {
940 941
        break;
      }
942 943 944 945
    }
    assert(sub_compact->builder != nullptr);
    assert(sub_compact->current_output() != nullptr);
    sub_compact->builder->Add(key, value);
946 947
    sub_compact->current_output_file_size =
        sub_compact->builder->EstimatedFileSize();
948
    const ParsedInternalKey& ikey = c_iter->ikey();
949
    sub_compact->current_output()->meta.UpdateBoundaries(
950
        key, value, ikey.sequence, ikey.type);
951 952
    sub_compact->num_output_records++;

953 954 955 956
    // 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.
    //
957 958 959 960
    // 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)
961 962
    bool output_file_ended = false;
    Status input_status;
963 964 965
    if (sub_compact->compaction->output_level() != 0 &&
        sub_compact->current_output_file_size >=
            sub_compact->compaction->max_output_file_size()) {
966 967 968 969 970
      // (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;
    }
971 972 973 974
    TEST_SYNC_POINT_CALLBACK(
        "CompactionJob::Run():PausingManualCompaction:2",
        reinterpret_cast<void*>(
            const_cast<std::atomic<bool>*>(manual_compaction_paused_)));
975
    c_iter->Next();
976 977 978
    if (c_iter->status().IsManualCompactionPaused()) {
      break;
    }
979 980
    if (!output_file_ended && c_iter->Valid() &&
        sub_compact->compaction->output_level() != 0 &&
981 982
        sub_compact->ShouldStopBefore(c_iter->key(),
                                      sub_compact->current_output_file_size) &&
983 984 985 986 987 988 989 990
        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) {
991 992 993 994
      const Slice* next_key = nullptr;
      if (c_iter->Valid()) {
        next_key = &c_iter->key();
      }
A
Andrew Kryczka 已提交
995
      CompactionIterationStats range_del_out_stats;
996
      status =
997
          FinishCompactionOutputFile(input_status, sub_compact, &range_del_agg,
998
                                     &range_del_out_stats, next_key);
A
Andrew Kryczka 已提交
999 1000
      RecordDroppedKeys(range_del_out_stats,
                        &sub_compact->compaction_job_stats);
I
Igor Canadi 已提交
1001 1002
    }
  }
1003

1004 1005 1006 1007
  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;
1008 1009 1010 1011
  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;
1012 1013 1014 1015 1016 1017 1018 1019
  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 已提交
1020 1021
  RecordCompactionIOStats();

1022 1023 1024 1025 1026 1027 1028
  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");
1029
  }
1030 1031 1032 1033 1034
  if ((status.ok() || status.IsColumnFamilyDropped()) &&
      (manual_compaction_paused_ &&
       manual_compaction_paused_->load(std::memory_order_relaxed))) {
    status = Status::Incomplete(Status::SubCode::kManualCompactionPaused);
  }
1035 1036 1037 1038 1039 1040 1041
  if (status.ok()) {
    status = input->status();
  }
  if (status.ok()) {
    status = c_iter->status();
  }

1042
  if (status.ok() && sub_compact->builder == nullptr &&
1043
      sub_compact->outputs.size() == 0 && !range_del_agg.IsEmpty()) {
1044
    // handle subcompaction containing only range deletions
1045 1046
    status = OpenCompactionOutputFile(sub_compact);
  }
1047 1048 1049 1050

  // Call FinishCompactionOutputFile() even if status is not ok: it needs to
  // close the output file.
  if (sub_compact->builder != nullptr) {
A
Andrew Kryczka 已提交
1051
    CompactionIterationStats range_del_out_stats;
1052
    Status s = FinishCompactionOutputFile(status, sub_compact, &range_del_agg,
1053
                                          &range_del_out_stats);
1054 1055 1056
    if (status.ok()) {
      status = s;
    }
A
Andrew Kryczka 已提交
1057
    RecordDroppedKeys(range_del_out_stats, &sub_compact->compaction_job_stats);
1058 1059
  }

1060 1061 1062
  sub_compact->compaction_job_stats.cpu_micros =
      env_->NowCPUNanos() / 1000 - prev_cpu_micros;

1063 1064
  if (measure_io_stats_) {
    sub_compact->compaction_job_stats.file_write_nanos +=
I
Igor Canadi 已提交
1065
        IOSTATS(write_nanos) - prev_write_nanos;
1066
    sub_compact->compaction_job_stats.file_fsync_nanos +=
I
Igor Canadi 已提交
1067
        IOSTATS(fsync_nanos) - prev_fsync_nanos;
1068
    sub_compact->compaction_job_stats.file_range_sync_nanos +=
I
Igor Canadi 已提交
1069
        IOSTATS(range_sync_nanos) - prev_range_sync_nanos;
1070
    sub_compact->compaction_job_stats.file_prepare_write_nanos +=
I
Igor Canadi 已提交
1071
        IOSTATS(prepare_write_nanos) - prev_prepare_write_nanos;
1072
    sub_compact->compaction_job_stats.cpu_micros -=
1073 1074 1075
        (IOSTATS(cpu_write_nanos) - prev_cpu_write_nanos +
         IOSTATS(cpu_read_nanos) - prev_cpu_read_nanos) /
        1000;
1076
    if (prev_perf_level != PerfLevel::kEnableTimeAndCPUTimeExceptForMutex) {
1077 1078 1079 1080
      SetPerfLevel(prev_perf_level);
    }
  }

1081 1082
  sub_compact->c_iter.reset();
  input.reset();
1083
  sub_compact->status = status;
I
Igor Canadi 已提交
1084 1085
}

1086
void CompactionJob::RecordDroppedKeys(
A
Andrew Kryczka 已提交
1087
    const CompactionIterationStats& c_iter_stats,
1088
    CompactionJobStats* compaction_job_stats) {
1089 1090 1091
  if (c_iter_stats.num_record_drop_user > 0) {
    RecordTick(stats_, COMPACTION_KEY_DROP_USER,
               c_iter_stats.num_record_drop_user);
1092
  }
1093 1094 1095
  if (c_iter_stats.num_record_drop_hidden > 0) {
    RecordTick(stats_, COMPACTION_KEY_DROP_NEWER_ENTRY,
               c_iter_stats.num_record_drop_hidden);
1096
    if (compaction_job_stats) {
1097 1098
      compaction_job_stats->num_records_replaced +=
          c_iter_stats.num_record_drop_hidden;
1099 1100
    }
  }
1101 1102 1103
  if (c_iter_stats.num_record_drop_obsolete > 0) {
    RecordTick(stats_, COMPACTION_KEY_DROP_OBSOLETE,
               c_iter_stats.num_record_drop_obsolete);
1104
    if (compaction_job_stats) {
1105 1106
      compaction_job_stats->num_expired_deletion_records +=
          c_iter_stats.num_record_drop_obsolete;
1107
    }
1108
  }
A
Andrew Kryczka 已提交
1109 1110 1111 1112 1113 1114 1115 1116
  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);
  }
1117 1118 1119 1120
  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);
  }
1121 1122
}

1123
Status CompactionJob::FinishCompactionOutputFile(
1124
    const Status& input_status, SubcompactionState* sub_compact,
1125
    CompactionRangeDelAggregator* range_del_agg,
A
Andrew Kryczka 已提交
1126
    CompactionIterationStats* range_del_out_stats,
1127
    const Slice* next_table_min_key /* = nullptr */) {
1128 1129
  AutoThreadOperationStageUpdater stage_updater(
      ThreadStatus::STAGE_COMPACTION_SYNC_FILE);
1130 1131 1132
  assert(sub_compact != nullptr);
  assert(sub_compact->outfile);
  assert(sub_compact->builder != nullptr);
1133
  assert(sub_compact->current_output() != nullptr);
I
Igor Canadi 已提交
1134

1135
  uint64_t output_number = sub_compact->current_output()->meta.fd.GetNumber();
I
Igor Canadi 已提交
1136 1137
  assert(output_number != 0);

1138 1139 1140
  ColumnFamilyData* cfd = sub_compact->compaction->column_family_data();
  const Comparator* ucmp = cfd->user_comparator();

I
Igor Canadi 已提交
1141
  // Check for iterator errors
1142
  Status s = input_status;
1143
  auto meta = &sub_compact->current_output()->meta;
1144
  assert(meta != nullptr);
1145
  if (s.ok()) {
1146
    Slice lower_bound_guard, upper_bound_guard;
Z
zhangjinpeng1987 已提交
1147
    std::string smallest_user_key;
1148
    const Slice *lower_bound, *upper_bound;
1149
    bool lower_bound_from_sub_compact = false;
1150 1151 1152 1153
    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;
1154
      lower_bound_from_sub_compact = true;
1155 1156 1157 1158
    } 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 已提交
1159 1160
      smallest_user_key = meta->smallest.user_key().ToString(false /*hex*/);
      lower_bound_guard = Slice(smallest_user_key);
1161 1162 1163 1164 1165
      lower_bound = &lower_bound_guard;
    } else {
      lower_bound = nullptr;
    }
    if (next_table_min_key != nullptr) {
1166 1167 1168 1169 1170 1171
      // 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.
1172
      upper_bound_guard = ExtractUserKey(*next_table_min_key);
1173 1174 1175 1176 1177 1178
      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;
      }
1179 1180 1181 1182 1183
    } else {
      // This is the last file in the subcompaction, so extend until the
      // subcompaction ends.
      upper_bound = sub_compact->end;
    }
1184 1185 1186 1187
    auto earliest_snapshot = kMaxSequenceNumber;
    if (existing_snapshots_.size() > 0) {
      earliest_snapshot = existing_snapshots_[0];
    }
1188 1189 1190 1191 1192 1193 1194
    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;
    }
1195

1196 1197 1198 1199 1200 1201 1202
    // 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);
1203 1204 1205 1206 1207 1208 1209 1210 1211 1212
    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();
    }
1213 1214
    for (; it->Valid(); it->Next()) {
      auto tombstone = it->Tombstone();
1215 1216 1217 1218 1219 1220 1221 1222 1223 1224 1225 1226
      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;
        }
1227 1228 1229 1230 1231 1232 1233 1234 1235 1236 1237
      }

      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();
1238 1239
      assert(lower_bound == nullptr ||
             ucmp->Compare(*lower_bound, kv.second) < 0);
1240 1241 1242 1243 1244 1245 1246 1247
      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.
        //
1248 1249 1250 1251 1252 1253 1254 1255 1256 1257 1258 1259 1260 1261 1262 1263 1264 1265
        // 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);
1266 1267 1268 1269 1270 1271 1272 1273 1274 1275 1276 1277 1278 1279 1280 1281 1282 1283 1284 1285 1286
      }
      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);
      }
1287 1288 1289 1290 1291 1292
#ifndef NDEBUG
      SequenceNumber smallest_ikey_seqnum = kMaxSequenceNumber;
      if (meta->smallest.size() > 0) {
        smallest_ikey_seqnum = GetInternalKeySeqno(meta->smallest.Encode());
      }
#endif
1293 1294 1295
      meta->UpdateBoundariesForRange(smallest_candidate, largest_candidate,
                                     tombstone.seq_,
                                     cfd->internal_comparator());
1296 1297 1298 1299 1300 1301 1302 1303

      // 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));
1304
    }
S
Siying Dong 已提交
1305
    meta->marked_for_compaction = sub_compact->builder->NeedCompact();
1306
  }
1307
  const uint64_t current_entries = sub_compact->builder->NumEntries();
I
Igor Canadi 已提交
1308
  if (s.ok()) {
1309
    s = sub_compact->builder->Finish();
I
Igor Canadi 已提交
1310
  } else {
1311
    sub_compact->builder->Abandon();
I
Igor Canadi 已提交
1312
  }
1313 1314 1315 1316
  if (!sub_compact->builder->io_status().ok()) {
    io_status_ = sub_compact->builder->io_status();
    s = io_status_;
  }
1317
  const uint64_t current_bytes = sub_compact->builder->FileSize();
S
Siying Dong 已提交
1318 1319 1320
  if (s.ok()) {
    meta->fd.file_size = current_bytes;
  }
1321
  sub_compact->current_output()->finished = true;
1322
  sub_compact->total_bytes += current_bytes;
I
Igor Canadi 已提交
1323 1324

  // Finish and check for file errors
1325
  IOStatus io_s;
S
Sagar Vemuri 已提交
1326
  if (s.ok()) {
1327
    StopWatch sw(env_, stats_, COMPACTION_OUTFILE_SYNC_MICROS);
1328
    io_s = sub_compact->outfile->Sync(db_options_.use_fsync);
I
Igor Canadi 已提交
1329
  }
1330 1331 1332
  if (io_s.ok()) {
    io_s = sub_compact->outfile->Close();
  }
1333 1334 1335 1336 1337 1338
  if (io_s.ok()) {
    // Add the checksum information to file metadata.
    meta->file_checksum = sub_compact->outfile->GetFileChecksum();
    meta->file_checksum_func_name =
        sub_compact->outfile->GetFileChecksumFuncName();
  }
1339 1340 1341
  if (!io_s.ok()) {
    io_status_ = io_s;
    s = io_s;
I
Igor Canadi 已提交
1342
  }
1343
  sub_compact->outfile.reset();
I
Igor Canadi 已提交
1344

1345 1346 1347 1348 1349 1350
  TableProperties tp;
  if (s.ok()) {
    tp = sub_compact->builder->GetTableProperties();
  }

  if (s.ok() && current_entries == 0 && tp.num_range_deletions == 0) {
Z
zhangjinpeng1987 已提交
1351 1352 1353
    // 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.
1354 1355 1356
    std::string fname =
        TableFileName(sub_compact->compaction->immutable_cf_options()->cf_paths,
                      meta->fd.GetNumber(), meta->fd.GetPathId());
Z
zhangjinpeng1987 已提交
1357 1358 1359 1360 1361 1362
    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();
1363
    meta = nullptr;
Z
zhangjinpeng1987 已提交
1364 1365
  }

1366
  if (s.ok() && (current_entries > 0 || tp.num_range_deletions > 0)) {
1367
    // Output to event logger and fire events.
1368 1369 1370 1371 1372 1373 1374 1375
    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 已提交
1376
  }
S
Siying Dong 已提交
1377 1378
  std::string fname;
  FileDescriptor output_fd;
1379
  uint64_t oldest_blob_file_number = kInvalidBlobFileNumber;
S
Siying Dong 已提交
1380
  if (meta != nullptr) {
1381 1382 1383
    fname =
        TableFileName(sub_compact->compaction->immutable_cf_options()->cf_paths,
                      meta->fd.GetNumber(), meta->fd.GetPathId());
S
Siying Dong 已提交
1384
    output_fd = meta->fd;
1385
    oldest_blob_file_number = meta->oldest_blob_file_number;
S
Siying Dong 已提交
1386 1387 1388
  } else {
    fname = "(nil)";
  }
1389 1390
  EventHelpers::LogAndNotifyTableFileCreationFinished(
      event_logger_, cfd->ioptions()->listeners, dbname_, cfd->GetName(), fname,
1391 1392
      job_id_, output_fd, oldest_blob_file_number, tp,
      TableFileCreationReason::kCompaction, s);
1393

1394
#ifndef ROCKSDB_LITE
1395 1396 1397
  // Report new file to SstFileManagerImpl
  auto sfm =
      static_cast<SstFileManagerImpl*>(db_options_.sst_file_manager.get());
S
Siying Dong 已提交
1398
  if (sfm && meta != nullptr && meta->fd.GetPathId() == 0) {
1399
    sfm->OnAddFile(fname);
1400
    if (sfm->IsMaxAllowedSpaceReached()) {
1401 1402
      // TODO(ajkr): should we return OK() if max space was reached by the final
      // compaction output file (similarly to how flush works when full)?
1403
      s = Status::SpaceLimit("Max allowed space was reached");
1404 1405 1406
      TEST_SYNC_POINT(
          "CompactionJob::FinishCompactionOutputFile:"
          "MaxAllowedSpaceReached");
1407
      InstrumentedMutexLock l(db_mutex_);
1408
      db_error_handler_->SetBGError(s, BackgroundErrorReason::kCompaction);
1409 1410
    }
  }
1411
#endif
1412

1413
  sub_compact->builder.reset();
1414
  sub_compact->current_output_file_size = 0;
I
Igor Canadi 已提交
1415 1416 1417
  return s;
}

I
Igor Canadi 已提交
1418
Status CompactionJob::InstallCompactionResults(
1419 1420
    const MutableCFOptions& mutable_cf_options) {
  db_mutex_->AssertHeld();
I
Igor Canadi 已提交
1421

1422
  auto* compaction = compact_->compaction;
I
Igor Canadi 已提交
1423 1424 1425 1426
  // 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_.
1427 1428 1429
  if (!versions_->VerifyCompactionFileConsistency(compaction)) {
    Compaction::InputLevelSummaryBuffer inputs_summary;

1430 1431 1432
    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 已提交
1433 1434 1435
    return Status::Corruption("Compaction input files inconsistent");
  }

1436 1437
  {
    Compaction::InputLevelSummaryBuffer inputs_summary;
1438 1439
    ROCKS_LOG_INFO(
        db_options_.info_log, "[%s] [JOB %d] Compacted %s => %" PRIu64 " bytes",
1440 1441 1442
        compaction->column_family_data()->GetName().c_str(), job_id_,
        compaction->InputLevelSummary(&inputs_summary), compact_->total_bytes);
  }
I
Igor Canadi 已提交
1443

G
Gihwan Oh 已提交
1444
  // Add compaction inputs
1445
  compaction->AddInputDeletions(compact_->compaction->edit());
1446

1447 1448 1449
  for (const auto& sub_compact : compact_->sub_compact_states) {
    for (const auto& out : sub_compact.outputs) {
      compaction->edit()->AddFile(compaction->output_level(), out.meta);
1450
    }
1451 1452
  }
  return versions_->LogAndApply(compaction->column_family_data(),
I
Igor Canadi 已提交
1453
                                mutable_cf_options, compaction->edit(),
1454
                                db_mutex_, db_directory_);
I
Igor Canadi 已提交
1455 1456 1457 1458
}

void CompactionJob::RecordCompactionIOStats() {
  RecordTick(stats_, COMPACT_READ_BYTES, IOSTATS(bytes_read));
1459 1460
  ThreadStatusUtil::IncreaseThreadOperationProperty(
      ThreadStatus::COMPACTION_BYTES_READ, IOSTATS(bytes_read));
I
Igor Canadi 已提交
1461 1462
  IOSTATS_RESET(bytes_read);
  RecordTick(stats_, COMPACT_WRITE_BYTES, IOSTATS(bytes_written));
1463 1464
  ThreadStatusUtil::IncreaseThreadOperationProperty(
      ThreadStatus::COMPACTION_BYTES_WRITTEN, IOSTATS(bytes_written));
I
Igor Canadi 已提交
1465 1466 1467
  IOSTATS_RESET(bytes_written);
}

1468 1469
Status CompactionJob::OpenCompactionOutputFile(
    SubcompactionState* sub_compact) {
1470 1471
  assert(sub_compact != nullptr);
  assert(sub_compact->builder == nullptr);
1472 1473
  // no need to lock because VersionSet::next_file_number_ is atomic
  uint64_t file_number = versions_->NewFileNumber();
1474 1475 1476
  std::string fname =
      TableFileName(sub_compact->compaction->immutable_cf_options()->cf_paths,
                    file_number, sub_compact->compaction->output_path_id());
1477 1478 1479 1480 1481 1482 1483 1484
  // 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
1485
  std::unique_ptr<FSWritableFile> writable_file;
A
Aliaksei Sandryhaila 已提交
1486
#ifndef NDEBUG
1487
  bool syncpoint_arg = file_options_.use_direct_writes;
1488
  TEST_SYNC_POINT_CALLBACK("CompactionJob::OpenCompactionOutputFile",
1489
                           &syncpoint_arg);
A
Aliaksei Sandryhaila 已提交
1490
#endif
1491
  Status s = NewWritableFile(fs_, fname, &writable_file, file_options_);
I
Igor Canadi 已提交
1492
  if (!s.ok()) {
1493 1494
    ROCKS_LOG_ERROR(
        db_options_.info_log,
1495
        "[%s] [JOB %d] OpenCompactionOutputFiles for table #%" PRIu64
1496
        " fails at NewWritableFile with status %s",
1497 1498
        sub_compact->compaction->column_family_data()->GetName().c_str(),
        job_id_, file_number, s.ToString().c_str());
I
Igor Canadi 已提交
1499
    LogFlush(db_options_.info_log);
1500 1501
    EventHelpers::LogAndNotifyTableFileCreationFinished(
        event_logger_, cfd->ioptions()->listeners, dbname_, cfd->GetName(),
1502 1503
        fname, job_id_, FileDescriptor(), kInvalidBlobFileNumber,
        TableProperties(), TableFileCreationReason::kCompaction, s);
I
Igor Canadi 已提交
1504 1505
    return s;
  }
1506

1507 1508 1509 1510 1511 1512 1513 1514 1515 1516 1517 1518 1519 1520 1521 1522 1523 1524 1525 1526 1527 1528
  // Try to figure out the output file's oldest ancester time.
  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 oldest_ancester_time =
      sub_compact->compaction->MinInputFileOldestAncesterTime();
  if (oldest_ancester_time == port::kMaxUint64) {
    oldest_ancester_time = current_time;
  }

  // Initialize a SubcompactionState::Output and add it to sub_compact->outputs
  {
    SubcompactionState::Output out;
    out.meta.fd = FileDescriptor(file_number,
                                 sub_compact->compaction->output_path_id(), 0);
    out.meta.oldest_ancester_time = oldest_ancester_time;
1529
    out.meta.file_creation_time = current_time;
1530 1531 1532
    out.finished = false;
    sub_compact->outputs.push_back(out);
  }
I
Igor Canadi 已提交
1533

1534
  writable_file->SetIOPriority(Env::IOPriority::IO_LOW);
S
Stream  
Shaohua Li 已提交
1535
  writable_file->SetWriteLifeTimeHint(write_hint_);
1536 1537
  writable_file->SetPreallocationBlockSize(static_cast<size_t>(
      sub_compact->compaction->OutputFilePreallocationSize()));
1538 1539
  const auto& listeners =
      sub_compact->compaction->immutable_cf_options()->listeners;
1540
  sub_compact->outfile.reset(
1541
      new WritableFileWriter(std::move(writable_file), fname, file_options_,
1542
                             env_, db_options_.statistics.get(), listeners,
1543
                             db_options_.file_checksum_gen_factory.get()));
I
Igor Canadi 已提交
1544

1545 1546 1547
  // 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
1548 1549
  bool skip_filters =
      cfd->ioptions()->optimize_filters_for_hits && bottommost_level_;
S
Sagar Vemuri 已提交
1550

1551
  sub_compact->builder.reset(NewTableBuilder(
1552 1553 1554 1555
      *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(),
1556
      0 /*sample_for_compression */,
1557
      sub_compact->compaction->output_compression_opts(),
1558 1559
      sub_compact->compaction->output_level(), skip_filters,
      oldest_ancester_time, 0 /* oldest_key_time */,
1560 1561
      sub_compact->compaction->max_output_file_size(), current_time, db_id_,
      db_session_id_));
I
Igor Canadi 已提交
1562 1563 1564 1565
  LogFlush(db_options_.info_log);
  return s;
}

1566
void CompactionJob::CleanupCompaction() {
1567
  for (SubcompactionState& sub_compact : compact_->sub_compact_states) {
1568
    const auto& sub_status = sub_compact.status;
I
Igor Canadi 已提交
1569

1570 1571 1572 1573 1574 1575 1576
    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);
    }
1577
    for (const auto& out : sub_compact.outputs) {
1578 1579 1580
      // If this file was inserted into the table cache then remove
      // them here because this compaction was not committed.
      if (!sub_status.ok()) {
1581
        TableCache::Evict(table_cache_.get(), out.meta.fd.GetNumber());
1582
      }
I
Igor Canadi 已提交
1583 1584 1585 1586 1587 1588
    }
  }
  delete compact_;
  compact_ = nullptr;
}

1589 1590
#ifndef ROCKSDB_LITE
namespace {
1591
void CopyPrefix(const Slice& src, size_t prefix_length, std::string* dst) {
1592 1593 1594
  assert(prefix_length > 0);
  size_t length = src.size() > prefix_length ? prefix_length : src.size();
  dst->assign(src.data(), length);
1595 1596 1597 1598 1599
}
}  // namespace

#endif  // !ROCKSDB_LITE

A
Andres Notzli 已提交
1600
void CompactionJob::UpdateCompactionStats() {
1601 1602 1603 1604 1605 1606
  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) {
1607
    if (compaction->level(input_level) != compaction->output_level()) {
1608 1609
      UpdateCompactionInputStatsHelper(
          &compaction_stats_.num_input_files_in_non_output_levels,
1610
          &compaction_stats_.bytes_read_non_output_levels, input_level);
1611 1612 1613
    } else {
      UpdateCompactionInputStatsHelper(
          &compaction_stats_.num_input_files_in_output_level,
1614
          &compaction_stats_.bytes_read_output_level, input_level);
1615 1616
    }
  }
A
Andres Notzli 已提交
1617

1618 1619
  uint64_t num_output_records = 0;

1620 1621 1622 1623 1624 1625 1626 1627 1628
  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);

1629 1630
    num_output_records += sub_compact.num_output_records;

1631 1632
    for (const auto& out : sub_compact.outputs) {
      compaction_stats_.bytes_written += out.meta.fd.file_size;
1633
    }
1634 1635 1636 1637 1638
  }

  if (compaction_stats_.num_input_records > num_output_records) {
    compaction_stats_.num_dropped_records =
        compaction_stats_.num_input_records - num_output_records;
A
Andres Notzli 已提交
1639
  }
1640 1641
}

1642 1643 1644
void CompactionJob::UpdateCompactionInputStatsHelper(int* num_files,
                                                     uint64_t* bytes_read,
                                                     int input_level) {
1645 1646 1647 1648 1649 1650 1651 1652 1653 1654 1655 1656
  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);
  }
}

1657 1658 1659 1660 1661 1662 1663 1664
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 =
1665
        stats.bytes_read_non_output_levels + stats.bytes_read_output_level;
1666
    compaction_job_stats_->num_input_records = stats.num_input_records;
1667
    compaction_job_stats_->num_input_files =
1668 1669
        stats.num_input_files_in_non_output_levels +
        stats.num_input_files_in_output_level;
1670
    compaction_job_stats_->num_input_files_at_output_level =
1671
        stats.num_input_files_in_output_level;
1672 1673 1674

    // output information
    compaction_job_stats_->total_output_bytes = stats.bytes_written;
1675
    compaction_job_stats_->num_output_records = compact_->num_output_records;
1676
    compaction_job_stats_->num_output_files = stats.num_output_files;
1677

1678
    if (compact_->NumOutputFiles() > 0U) {
1679 1680 1681 1682 1683 1684
      CopyPrefix(compact_->SmallestUserKey(),
                 CompactionJobStats::kMaxPrefixLength,
                 &compaction_job_stats_->smallest_output_key_prefix);
      CopyPrefix(compact_->LargestUserKey(),
                 CompactionJobStats::kMaxPrefixLength,
                 &compaction_job_stats_->largest_output_key_prefix);
1685 1686
    }
  }
1687 1688
#else
  (void)stats;
1689 1690 1691
#endif  // !ROCKSDB_LITE
}

1692 1693 1694 1695
void CompactionJob::LogCompaction() {
  Compaction* compaction = compact_->compaction;
  ColumnFamilyData* cfd = compaction->column_family_data();

A
Andres Notzli 已提交
1696 1697 1698 1699
  // 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;
1700 1701 1702 1703
    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 已提交
1704 1705
    char scratch[2345];
    compaction->Summary(scratch, sizeof(scratch));
1706 1707
    ROCKS_LOG_INFO(db_options_.info_log, "[%s] Compaction start summary: %s\n",
                   cfd->GetName().c_str(), scratch);
A
Andres Notzli 已提交
1708 1709
    // build event logger report
    auto stream = event_logger_->Log();
1710 1711 1712 1713
    stream << "job" << job_id_ << "event"
           << "compaction_started"
           << "compaction_reason"
           << GetCompactionReasonString(compaction->compaction_reason());
A
Andres Notzli 已提交
1714 1715 1716 1717 1718 1719 1720 1721 1722 1723 1724 1725 1726
    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();
  }
}

1727
}  // namespace ROCKSDB_NAMESPACE