Skip to content

Commit

Permalink
Seperate InternalIterator from Iterator
Browse files Browse the repository at this point in the history
Summary:
Separate a new class InternalIterator from class Iterator, when the look-up is done internally, which also means they operate on key with sequence ID and type.

This change will enable potential future optimizations but for now InternalIterator's functions are still the same as Iterator's.
At the same time, separate the cleanup function to a separate class and let both of InternalIterator and Iterator inherit from it.

Test Plan: Run all existing tests.

Reviewers: igor, yhchiang, anthony, kradhakrishnan, IslamAbdelRahman, rven

Reviewed By: rven

Subscribers: leveldb, dhruba

Differential Revision: https://reviews.facebook.net/D48549
  • Loading branch information
siying committed Oct 13, 2015
1 parent 198ed58 commit 35ad531
Show file tree
Hide file tree
Showing 68 changed files with 557 additions and 377 deletions.
8 changes: 5 additions & 3 deletions db/builder.cc
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,7 @@
#include "rocksdb/options.h"
#include "rocksdb/table.h"
#include "table/block_based_table_builder.h"
#include "table/internal_iterator.h"
#include "util/file_reader_writer.h"
#include "util/iostats_context_imp.h"
#include "util/stop_watch.h"
Expand All @@ -52,8 +53,9 @@ TableBuilder* NewTableBuilder(

Status BuildTable(
const std::string& dbname, Env* env, const ImmutableCFOptions& ioptions,
const EnvOptions& env_options, TableCache* table_cache, Iterator* iter,
FileMetaData* meta, const InternalKeyComparator& internal_comparator,
const EnvOptions& env_options, TableCache* table_cache,
InternalIterator* iter, FileMetaData* meta,
const InternalKeyComparator& internal_comparator,
const std::vector<std::unique_ptr<IntTblPropCollectorFactory>>*
int_tbl_prop_collector_factories,
uint32_t column_family_id, std::vector<SequenceNumber> snapshots,
Expand Down Expand Up @@ -141,7 +143,7 @@ Status BuildTable(

if (s.ok() && !empty) {
// Verify that the table is usable
std::unique_ptr<Iterator> it(table_cache->NewIterator(
std::unique_ptr<InternalIterator> it(table_cache->NewIterator(
ReadOptions(), env_options, internal_comparator, meta->fd, nullptr,
(internal_stats == nullptr) ? nullptr
: internal_stats->GetFileReadHist(0),
Expand Down
6 changes: 4 additions & 2 deletions db/builder.h
Original file line number Diff line number Diff line change
Expand Up @@ -31,6 +31,7 @@ class VersionEdit;
class TableBuilder;
class WritableFileWriter;
class InternalStats;
class InternalIterator;

TableBuilder* NewTableBuilder(
const ImmutableCFOptions& options,
Expand All @@ -49,8 +50,9 @@ TableBuilder* NewTableBuilder(
// zero, and no Table file will be produced.
extern Status BuildTable(
const std::string& dbname, Env* env, const ImmutableCFOptions& options,
const EnvOptions& env_options, TableCache* table_cache, Iterator* iter,
FileMetaData* meta, const InternalKeyComparator& internal_comparator,
const EnvOptions& env_options, TableCache* table_cache,
InternalIterator* iter, FileMetaData* meta,
const InternalKeyComparator& internal_comparator,
const std::vector<std::unique_ptr<IntTblPropCollectorFactory>>*
int_tbl_prop_collector_factories,
uint32_t column_family_id, std::vector<SequenceNumber> snapshots,
Expand Down
3 changes: 2 additions & 1 deletion db/compaction_iterator.cc
Original file line number Diff line number Diff line change
Expand Up @@ -6,11 +6,12 @@
// of patent rights can be found in the PATENTS file in the same directory.

#include "db/compaction_iterator.h"
#include "table/internal_iterator.h"

namespace rocksdb {

CompactionIterator::CompactionIterator(
Iterator* input, const Comparator* cmp, MergeHelper* merge_helper,
InternalIterator* input, const Comparator* cmp, MergeHelper* merge_helper,
SequenceNumber last_sequence, std::vector<SequenceNumber>* snapshots,
Env* env, bool expect_valid_internal_key, Compaction* compaction,
const CompactionFilter* compaction_filter, LogBuffer* log_buffer)
Expand Down
4 changes: 2 additions & 2 deletions db/compaction_iterator.h
Original file line number Diff line number Diff line change
Expand Up @@ -37,7 +37,7 @@ struct CompactionIteratorStats {

class CompactionIterator {
public:
CompactionIterator(Iterator* input, const Comparator* cmp,
CompactionIterator(InternalIterator* input, const Comparator* cmp,
MergeHelper* merge_helper, SequenceNumber last_sequence,
std::vector<SequenceNumber>* snapshots, Env* env,
bool expect_valid_internal_key,
Expand Down Expand Up @@ -84,7 +84,7 @@ class CompactionIterator {
inline SequenceNumber findEarliestVisibleSnapshot(
SequenceNumber in, SequenceNumber* prev_snapshot);

Iterator* input_;
InternalIterator* input_;
const Comparator* cmp_;
MergeHelper* merge_helper_;
const std::vector<SequenceNumber>* snapshots_;
Expand Down
4 changes: 2 additions & 2 deletions db/compaction_job.cc
Original file line number Diff line number Diff line change
Expand Up @@ -585,7 +585,7 @@ Status CompactionJob::Install(const MutableCFOptions& mutable_cf_options,

void CompactionJob::ProcessKeyValueCompaction(SubcompactionState* sub_compact) {
assert(sub_compact != nullptr);
std::unique_ptr<Iterator> input(
std::unique_ptr<InternalIterator> input(
versions_->MakeInputIterator(sub_compact->compaction));

AutoThreadOperationStageUpdater stage_updater(
Expand Down Expand Up @@ -811,7 +811,7 @@ Status CompactionJob::FinishCompactionOutputFile(
if (s.ok() && current_entries > 0) {
// Verify that the table is usable
ColumnFamilyData* cfd = sub_compact->compaction->column_family_data();
Iterator* iter = cfd->table_cache()->NewIterator(
InternalIterator* iter = cfd->table_cache()->NewIterator(
ReadOptions(), env_options_, cfd->internal_comparator(), meta->fd,
nullptr, cfd->internal_stats()->GetFileReadHist(
compact_->compaction->output_level()),
Expand Down
2 changes: 1 addition & 1 deletion db/compaction_job.h
Original file line number Diff line number Diff line change
Expand Up @@ -35,9 +35,9 @@
#include "rocksdb/env.h"
#include "rocksdb/memtablerep.h"
#include "rocksdb/transaction_log.h"
#include "table/scoped_arena_iterator.h"
#include "util/autovector.h"
#include "util/event_logger.h"
#include "util/scoped_arena_iterator.h"
#include "util/stop_watch.h"
#include "util/thread_local.h"

Expand Down
2 changes: 1 addition & 1 deletion db/compaction_job_stats_test.cc
Original file line number Diff line number Diff line change
Expand Up @@ -47,14 +47,14 @@
#include "table/block_based_table_factory.h"
#include "table/mock_table.h"
#include "table/plain_table_factory.h"
#include "table/scoped_arena_iterator.h"
#include "util/compression.h"
#include "util/hash.h"
#include "util/hash_linklist_rep.h"
#include "util/logging.h"
#include "util/mock_env.h"
#include "util/mutexlock.h"
#include "util/rate_limiter.h"
#include "util/scoped_arena_iterator.h"
#include "util/statistics.h"
#include "util/string_util.h"
#include "util/sync_point.h"
Expand Down
23 changes: 12 additions & 11 deletions db/db_impl.cc
Original file line number Diff line number Diff line change
Expand Up @@ -2845,11 +2845,11 @@ static void CleanupIteratorState(void* arg1, void* arg2) {
}
} // namespace

Iterator* DBImpl::NewInternalIterator(const ReadOptions& read_options,
ColumnFamilyData* cfd,
SuperVersion* super_version,
Arena* arena) {
Iterator* internal_iter;
InternalIterator* DBImpl::NewInternalIterator(const ReadOptions& read_options,
ColumnFamilyData* cfd,
SuperVersion* super_version,
Arena* arena) {
InternalIterator* internal_iter;
assert(arena != nullptr);
// Need to create internal iterator from the arena.
MergeIteratorBuilder merge_iter_builder(&cfd->internal_comparator(), arena);
Expand Down Expand Up @@ -3148,7 +3148,8 @@ Status DBImpl::AddFile(ColumnFamilyHandle* column_family,
file_info.num_entries = table_reader->GetTableProperties()->num_entries;

ParsedInternalKey key;
std::unique_ptr<Iterator> iter(table_reader->NewIterator(ReadOptions()));
std::unique_ptr<InternalIterator> iter(
table_reader->NewIterator(ReadOptions()));

// Get first (smallest) key from file
iter->SeekToFirst();
Expand Down Expand Up @@ -3307,8 +3308,8 @@ Status DBImpl::AddFile(ColumnFamilyHandle* column_family,
return status;
}

Iterator* DBImpl::NewInternalIterator(Arena* arena,
ColumnFamilyHandle* column_family) {
InternalIterator* DBImpl::NewInternalIterator(
Arena* arena, ColumnFamilyHandle* column_family) {
ColumnFamilyData* cfd;
if (column_family == nullptr) {
cfd = default_cf_handle_->cfd();
Expand Down Expand Up @@ -3565,7 +3566,7 @@ Iterator* DBImpl::NewIterator(const ReadOptions& read_options,
snapshot, sv->mutable_cf_options.max_sequential_skip_in_iterations,
read_options.iterate_upper_bound);

Iterator* internal_iter =
InternalIterator* internal_iter =
NewInternalIterator(read_options, cfd, sv, db_iter->GetArena());
db_iter->SetIterUnderDBIter(internal_iter);

Expand Down Expand Up @@ -3632,8 +3633,8 @@ Status DBImpl::NewIterators(
ArenaWrappedDBIter* db_iter = NewArenaWrappedDbIterator(
env_, *cfd->ioptions(), cfd->user_comparator(), snapshot,
sv->mutable_cf_options.max_sequential_skip_in_iterations);
Iterator* internal_iter = NewInternalIterator(
read_options, cfd, sv, db_iter->GetArena());
InternalIterator* internal_iter =
NewInternalIterator(read_options, cfd, sv, db_iter->GetArena());
db_iter->SetIterUnderDBIter(internal_iter);
iterators->push_back(db_iter);
}
Expand Down
12 changes: 7 additions & 5 deletions db/db_impl.h
Original file line number Diff line number Diff line change
Expand Up @@ -36,11 +36,11 @@
#include "rocksdb/env.h"
#include "rocksdb/memtablerep.h"
#include "rocksdb/transaction_log.h"
#include "table/scoped_arena_iterator.h"
#include "util/autovector.h"
#include "util/event_logger.h"
#include "util/hash.h"
#include "util/instrumented_mutex.h"
#include "util/scoped_arena_iterator.h"
#include "util/stop_watch.h"
#include "util/thread_local.h"

Expand Down Expand Up @@ -251,8 +251,8 @@ class DBImpl : public DB {
// Return an internal iterator over the current state of the database.
// The keys of this iterator are internal keys (see format.h).
// The returned iterator should be deleted when no longer needed.
Iterator* NewInternalIterator(Arena* arena,
ColumnFamilyHandle* column_family = nullptr);
InternalIterator* NewInternalIterator(
Arena* arena, ColumnFamilyHandle* column_family = nullptr);

#ifndef NDEBUG
// Extra methods (for testing) that are not in the public DB interface
Expand Down Expand Up @@ -370,8 +370,10 @@ class DBImpl : public DB {
const DBOptions db_options_;
Statistics* stats_;

Iterator* NewInternalIterator(const ReadOptions&, ColumnFamilyData* cfd,
SuperVersion* super_version, Arena* arena);
InternalIterator* NewInternalIterator(const ReadOptions&,
ColumnFamilyData* cfd,
SuperVersion* super_version,
Arena* arena);

void NotifyOnFlushCompleted(ColumnFamilyData* cfd, FileMetaData* file_meta,
const MutableCFOptions& mutable_cf_options,
Expand Down
19 changes: 10 additions & 9 deletions db/db_iter.cc
Original file line number Diff line number Diff line change
Expand Up @@ -15,11 +15,12 @@

#include "db/filename.h"
#include "db/dbformat.h"
#include "port/port.h"
#include "rocksdb/env.h"
#include "rocksdb/options.h"
#include "rocksdb/iterator.h"
#include "rocksdb/merge_operator.h"
#include "port/port.h"
#include "table/internal_iterator.h"
#include "util/arena.h"
#include "util/logging.h"
#include "util/mutexlock.h"
Expand Down Expand Up @@ -58,9 +59,9 @@ class DBIter: public Iterator {
kReverse
};

DBIter(Env* env, const ImmutableCFOptions& ioptions,
const Comparator* cmp, Iterator* iter, SequenceNumber s,
bool arena_mode, uint64_t max_sequential_skip_in_iterations,
DBIter(Env* env, const ImmutableCFOptions& ioptions, const Comparator* cmp,
InternalIterator* iter, SequenceNumber s, bool arena_mode,
uint64_t max_sequential_skip_in_iterations,
const Slice* iterate_upper_bound = nullptr)
: arena_mode_(arena_mode),
env_(env),
Expand All @@ -83,10 +84,10 @@ class DBIter: public Iterator {
if (!arena_mode_) {
delete iter_;
} else {
iter_->~Iterator();
iter_->~InternalIterator();
}
}
virtual void SetIter(Iterator* iter) {
virtual void SetIter(InternalIterator* iter) {
assert(iter_ == nullptr);
iter_ = iter;
}
Expand Down Expand Up @@ -142,7 +143,7 @@ class DBIter: public Iterator {
Logger* logger_;
const Comparator* const user_comparator_;
const MergeOperator* const user_merge_operator_;
Iterator* iter_;
InternalIterator* iter_;
SequenceNumber const sequence_;

Status status_;
Expand Down Expand Up @@ -744,7 +745,7 @@ void DBIter::SeekToLast() {

Iterator* NewDBIterator(Env* env, const ImmutableCFOptions& ioptions,
const Comparator* user_key_comparator,
Iterator* internal_iter,
InternalIterator* internal_iter,
const SequenceNumber& sequence,
uint64_t max_sequential_skip_in_iterations,
const Slice* iterate_upper_bound) {
Expand All @@ -757,7 +758,7 @@ ArenaWrappedDBIter::~ArenaWrappedDBIter() { db_iter_->~DBIter(); }

void ArenaWrappedDBIter::SetDBIter(DBIter* iter) { db_iter_ = iter; }

void ArenaWrappedDBIter::SetIterUnderDBIter(Iterator* iter) {
void ArenaWrappedDBIter::SetIterUnderDBIter(InternalIterator* iter) {
static_cast<DBIter*>(db_iter_)->SetIter(iter);
}

Expand Down
19 changes: 10 additions & 9 deletions db/db_iter.h
Original file line number Diff line number Diff line change
Expand Up @@ -10,6 +10,7 @@
#pragma once
#include <stdint.h>
#include "rocksdb/db.h"
#include "rocksdb/iterator.h"
#include "db/dbformat.h"
#include "util/arena.h"
#include "util/autovector.h"
Expand All @@ -18,18 +19,17 @@ namespace rocksdb {

class Arena;
class DBIter;
class InternalIterator;

// Return a new iterator that converts internal keys (yielded by
// "*internal_iter") that were live at the specified "sequence" number
// into appropriate user keys.
extern Iterator* NewDBIterator(
Env* env,
const ImmutableCFOptions& options,
const Comparator *user_key_comparator,
Iterator* internal_iter,
const SequenceNumber& sequence,
uint64_t max_sequential_skip_in_iterations,
const Slice* iterate_upper_bound = nullptr);
extern Iterator* NewDBIterator(Env* env, const ImmutableCFOptions& options,
const Comparator* user_key_comparator,
InternalIterator* internal_iter,
const SequenceNumber& sequence,
uint64_t max_sequential_skip_in_iterations,
const Slice* iterate_upper_bound = nullptr);

// A wrapper iterator which wraps DB Iterator and the arena, with which the DB
// iterator is supposed be allocated. This class is used as an entry point of
Expand All @@ -50,7 +50,7 @@ class ArenaWrappedDBIter : public Iterator {

// Set the internal iterator wrapped inside the DB Iterator. Usually it is
// a merging iterator.
virtual void SetIterUnderDBIter(Iterator* iter);
virtual void SetIterUnderDBIter(InternalIterator* iter);
virtual bool Valid() const override;
virtual void SeekToFirst() override;
virtual void SeekToLast() override;
Expand All @@ -60,6 +60,7 @@ class ArenaWrappedDBIter : public Iterator {
virtual Slice key() const override;
virtual Slice value() const override;
virtual Status status() const override;

void RegisterCleanup(CleanupFunction function, void* arg1, void* arg2);

private:
Expand Down
7 changes: 4 additions & 3 deletions db/db_iter_test.cc
Original file line number Diff line number Diff line change
Expand Up @@ -29,7 +29,7 @@ static uint64_t TestGetTickerCount(const Options& options,
return options.statistics->getTickerCount(ticker_type);
}

class TestIterator : public Iterator {
class TestIterator : public InternalIterator {
public:
explicit TestIterator(const Comparator* comparator)
: initialized_(false),
Expand Down Expand Up @@ -1864,11 +1864,12 @@ class DBIterWithMergeIterTest : public testing::Test {
internal_iter2_->Add("d", kTypeValue, "7", 3u);
internal_iter2_->Finish();

std::vector<Iterator*> child_iters;
std::vector<InternalIterator*> child_iters;
child_iters.push_back(internal_iter1_);
child_iters.push_back(internal_iter2_);
InternalKeyComparator icomp(BytewiseComparator());
Iterator* merge_iter = NewMergingIterator(&icomp_, &child_iters[0], 2u);
InternalIterator* merge_iter =
NewMergingIterator(&icomp_, &child_iters[0], 2u);

db_iter_.reset(NewDBIterator(env_, ImmutableCFOptions(options_),
BytewiseComparator(), merge_iter,
Expand Down
2 changes: 1 addition & 1 deletion db/db_test.cc
Original file line number Diff line number Diff line change
Expand Up @@ -54,6 +54,7 @@
#include "table/block_based_table_factory.h"
#include "table/mock_table.h"
#include "table/plain_table_factory.h"
#include "table/scoped_arena_iterator.h"
#include "util/file_reader_writer.h"
#include "util/hash.h"
#include "util/hash_linklist_rep.h"
Expand All @@ -64,7 +65,6 @@
#include "util/rate_limiter.h"
#include "util/statistics.h"
#include "util/testharness.h"
#include "util/scoped_arena_iterator.h"
#include "util/sync_point.h"
#include "util/testutil.h"
#include "util/mock_env.h"
Expand Down
Loading

0 comments on commit 35ad531

Please sign in to comment.