Impala
Impalaistheopensource,nativeanalyticdatabaseforApacheHadoop.
|
#include <data-source-scan-node.h>
Public Member Functions | |
DataSourceScanNode (ObjectPool *pool, const TPlanNode &tnode, const DescriptorTbl &descs) | |
~DataSourceScanNode () | |
virtual Status | Prepare (RuntimeState *state) |
Load the data source library and create the ExternalDataSourceExecutor. More... | |
virtual Status | Open (RuntimeState *state) |
Open the data source and initialize the first row batch. More... | |
virtual Status | GetNext (RuntimeState *state, RowBatch *row_batch, bool *eos) |
Fill the next row batch, calls GetNext() on the external scanner. More... | |
virtual Status | Reset (RuntimeState *state) |
NYI. More... | |
virtual void | Close (RuntimeState *state) |
Close the scanner, and report errors. More... | |
void | SetScanRanges (const std::vector< TScanRangeParams > &scan_range_params) |
virtual bool | IsScanNode () const |
RuntimeProfile::Counter * | bytes_read_counter () const |
RuntimeProfile::Counter * | rows_read_counter () const |
RuntimeProfile::Counter * | read_timer () const |
RuntimeProfile::Counter * | total_throughput_counter () const |
RuntimeProfile::Counter * | per_read_thread_throughput_counter () const |
RuntimeProfile::Counter * | materialize_tuple_timer () const |
RuntimeProfile::Counter * | scan_ranges_complete_counter () const |
RuntimeProfile::ThreadCounters * | scanner_thread_counters () const |
RuntimeProfile::Counter & | active_scanner_thread_counter () |
RuntimeProfile::Counter * | average_scanner_thread_concurrency () const |
virtual Status | Init (const TPlanNode &tnode) |
void | CollectNodes (TPlanNodeType::type node_type, std::vector< ExecNode * > *nodes) |
void | CollectScanNodes (std::vector< ExecNode * > *nodes) |
Collect all scan node types. More... | |
std::string | DebugString () const |
Returns a string representation in DFS order of the plan rooted at this. More... | |
const std::vector< ExprContext * > & | conjunct_ctxs () const |
int | id () const |
TPlanNodeType::type | type () const |
const RowDescriptor & | row_desc () const |
int64_t | rows_returned () const |
int64_t | limit () const |
bool | ReachedLimit () |
RuntimeProfile * | runtime_profile () |
MemTracker * | mem_tracker () |
MemTracker * | expr_mem_tracker () |
Static Public Member Functions | |
static Status | CreateTree (ObjectPool *pool, const TPlan &plan, const DescriptorTbl &descs, ExecNode **root) |
static void | SetDebugOptions (int node_id, TExecNodePhase::type phase, TDebugAction::type action, ExecNode *tree) |
Set debug action for node with given id in 'tree'. More... | |
static bool | EvalConjuncts (ExprContext *const *ctxs, int num_ctxs, TupleRow *row) |
static llvm::Function * | CodegenEvalConjuncts (RuntimeState *state, const std::vector< ExprContext * > &conjunct_ctxs, const char *name="EvalConjuncts") |
static int | GetNodeIdFromProfile (RuntimeProfile *p) |
Extract node id from p->name(). More... | |
Static Public Attributes | |
static const std::string | BYTES_READ_COUNTER = "BytesRead" |
names of ScanNode common counters More... | |
static const std::string | ROWS_READ_COUNTER = "RowsRead" |
static const std::string | TOTAL_HDFS_READ_TIMER = "TotalRawHdfsReadTime(*)" |
static const std::string | TOTAL_HBASE_READ_TIMER = "TotalRawHBaseReadTime(*)" |
static const std::string | TOTAL_THROUGHPUT_COUNTER = "TotalReadThroughput" |
static const std::string | PER_READ_THREAD_THROUGHPUT_COUNTER |
static const std::string | NUM_DISKS_ACCESSED_COUNTER = "NumDisksAccessed" |
static const std::string | MATERIALIZE_TUPLE_TIMER = "MaterializeTupleTime(*)" |
static const std::string | SCAN_RANGES_COMPLETE_COUNTER = "ScanRangesComplete" |
static const std::string | SCANNER_THREAD_COUNTERS_PREFIX = "ScannerThreads" |
static const std::string | SCANNER_THREAD_TOTAL_WALLCLOCK_TIME |
static const std::string | AVERAGE_SCANNER_THREAD_CONCURRENCY |
static const std::string | AVERAGE_HDFS_READ_THREAD_CONCURRENCY |
static const std::string | NUM_SCANNER_THREADS_STARTED |
static const std::string | ROW_THROUGHPUT_COUNTER = "RowsReturnedRate" |
Names of counters shared by all exec nodes. More... | |
Protected Member Functions | |
virtual void | DebugString (int indentation_level, std::stringstream *out) const |
Write debug string of this into out. More... | |
ExecNode * | child (int i) |
bool | is_closed () |
void | InitRuntimeProfile (const std::string &name) |
Status | ExecDebugAction (TExecNodePhase::type phase, RuntimeState *state) |
void | AddRuntimeExecOption (const std::string &option) |
Appends option to 'runtime_exec_options_'. More... | |
virtual Status | QueryMaintenance (RuntimeState *state) |
void | AddExprCtxToFree (ExprContext *ctx) |
void | AddExprCtxsToFree (const std::vector< ExprContext * > &ctxs) |
void | AddExprCtxsToFree (const SortExecExprs &sort_exec_exprs) |
Static Protected Member Functions | |
static Status | CreateNode (ObjectPool *pool, const TPlanNode &tnode, const DescriptorTbl &descs, ExecNode **node) |
Create a single exec node derived from thrift node; place exec node in 'pool'. More... | |
static Status | CreateTreeHelper (ObjectPool *pool, const std::vector< TPlanNode > &tnodes, const DescriptorTbl &descs, ExecNode *parent, int *node_idx, ExecNode **root) |
Private Member Functions | |
Status | MaterializeNextRow (MemPool *mem_pool) |
Materializes the next row (next_row_idx_) into tuple_. More... | |
Status | GetNextInputBatch () |
Gets the next batch from the data source, stored in input_batch_. More... | |
Status | ValidateRowBatchSize () |
bool | InputBatchHasNext () |
True if input_batch_ has more rows. More... | |
Private Attributes | |
boost::scoped_ptr < ExternalDataSourceExecutor > | data_source_executor_ |
Used to call the external data source. More... | |
const TDataSourceScanNode | data_src_node_ |
Thrift structure describing the data source scan node. More... | |
const TupleDescriptor * | tuple_desc_ |
Descriptor of tuples read. More... | |
int | tuple_idx_ |
Tuple index in tuple row. More... | |
Tuple * | tuple_ |
Current tuple. More... | |
std::vector< SlotDescriptor * > | materialized_slots_ |
std::string | scan_handle_ |
The opaque handle returned by the data source for the scan. More... | |
boost::scoped_ptr < extdatasource::TGetNextResult > | input_batch_ |
int | num_rows_ |
size_t | next_row_idx_ |
std::vector< int > | cols_next_val_idx_ |
Scan node for external data sources. The external data source jar is loaded in Prepare() (via an ExternalDataSourceExecutor), and then the data source is called to receive row batches when necessary. This node converts the rows stored in a thrift structure to RowBatches. The external data source is closed in Close().
Definition at line 38 of file data-source-scan-node.h.
impala::DataSourceScanNode::DataSourceScanNode | ( | ObjectPool * | pool, |
const TPlanNode & | tnode, | ||
const DescriptorTbl & | descs | ||
) |
Definition at line 62 of file data-source-scan-node.cc.
impala::DataSourceScanNode::~DataSourceScanNode | ( | ) |
Definition at line 71 of file data-source-scan-node.cc.
|
inlineinherited |
Definition at line 113 of file scan-node.h.
References impala::ScanNode::active_scanner_thread_counter_.
|
protectedinherited |
Referenced by impala::ExecNode::AddExprCtxsToFree(), impala::UnionNode::Prepare(), impala::SortNode::Prepare(), impala::TopNNode::Prepare(), impala::ExchangeNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::ExecNode::Prepare(), impala::PartitionedAggregationNode::Prepare(), and impala::HdfsScanNode::Prepare().
|
protectedinherited |
Definition at line 410 of file exec-node.cc.
References impala::ExecNode::AddExprCtxsToFree(), impala::SortExecExprs::lhs_ordering_expr_ctxs(), impala::SortExecExprs::rhs_ordering_expr_ctxs(), and impala::SortExecExprs::sort_tuple_slot_expr_ctxs().
|
inlineprotectedinherited |
Add an ExprContext to have its local allocations freed by QueryMaintenance(). Exprs that are evaluated in the main execution thread should be added. Exprs evaluated in a separate thread are generally not safe to add, since a local allocation may be freed while it's being used. Rather than using this mechanism, threads should call FreeLocalAllocations() on local ExprContexts periodically.
Definition at line 276 of file exec-node.h.
References impala::ExecNode::expr_ctxs_to_free_.
Referenced by impala::AnalyticEvalNode::Prepare().
|
protectedinherited |
Appends option to 'runtime_exec_options_'.
Definition at line 188 of file exec-node.cc.
References impala::RuntimeProfile::AddInfoString(), impala::ExecNode::exec_options_lock_, impala::ExecNode::runtime_exec_options_, and impala::ExecNode::runtime_profile().
Referenced by impala::PartitionedHashJoinNode::AttachProbeFilters(), impala::HashJoinNode::ConstructBuildSide(), impala::BlockingJoinNode::Open(), impala::HashJoinNode::Prepare(), impala::AggregationNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::PartitionedAggregationNode::Prepare(), impala::HdfsScanNode::Prepare(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
inlineinherited |
Definition at line 116 of file scan-node.h.
References impala::ScanNode::average_scanner_thread_concurrency_.
|
inlineinherited |
Definition at line 95 of file scan-node.h.
References impala::ScanNode::bytes_read_counter_.
Referenced by impala::HBaseTableScanner::GetFamily(), impala::HBaseTableScanner::GetQualifier(), impala::HBaseTableScanner::GetRowKey(), impala::HBaseTableScanner::GetValue(), impala::HdfsScanNode::Open(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
inlineprotectedinherited |
Definition at line 241 of file exec-node.h.
References impala::ExecNode::children_.
Referenced by impala::CrossJoinNode::BuildListDebugString(), impala::BlockingJoinNode::BuildSideThread(), impala::HashJoinNode::CodegenCreateOutputRow(), impala::PartitionedHashJoinNode::CodegenCreateOutputRow(), impala::CrossJoinNode::ConstructBuildSide(), impala::HashJoinNode::ConstructBuildSide(), impala::PartitionedHashJoinNode::ConstructBuildSide(), impala::BlockingJoinNode::GetLeftChildRowString(), impala::SelectNode::GetNext(), impala::UnionNode::GetNext(), impala::CrossJoinNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::PartitionedAggregationNode::Partition::InitStreams(), impala::HashJoinNode::LeftJoinGetNext(), impala::PartitionedHashJoinNode::NextProbeRowBatch(), impala::SelectNode::Open(), impala::SortNode::Open(), impala::TopNNode::Open(), impala::BlockingJoinNode::Open(), impala::AggregationNode::Open(), impala::AnalyticEvalNode::Open(), impala::PartitionedAggregationNode::Open(), impala::UnionNode::OpenCurrentChild(), impala::SelectNode::Prepare(), impala::SortNode::Prepare(), impala::UnionNode::Prepare(), impala::TopNNode::Prepare(), impala::BlockingJoinNode::Prepare(), impala::HashJoinNode::Prepare(), impala::AggregationNode::Prepare(), impala::AnalyticEvalNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::PartitionedAggregationNode::Prepare(), impala::PartitionedHashJoinNode::ProcessBuildInput(), impala::AnalyticEvalNode::ProcessChildBatches(), and impala::SortNode::SortInput().
|
virtual |
Close the scanner, and report errors.
Reimplemented from impala::ExecNode.
Definition at line 349 of file data-source-scan-node.cc.
References impala::RuntimeState::LogError(), impala::Status::msg(), impala::Status::ok(), and SCOPED_TIMER.
|
staticinherited |
Returns a codegen'd version of EvalConjuncts(), or NULL if the function couldn't be codegen'd. The codegen'd version uses inlined, codegen'd GetBooleanVal() functions.
Definition at line 452 of file exec-node.cc.
References impala::LlvmCodeGen::FnPrototype::AddArgument(), impala::LlvmCodeGen::context(), impala::CodegenAnyVal::CreateCallWrapped(), impala::LlvmCodeGen::false_value(), impala::LlvmCodeGen::FinalizeFunction(), impala::RuntimeState::GetCodegen(), impala::Status::GetDetail(), impala::CodegenAnyVal::GetIsNull(), impala::LlvmCodeGen::GetType(), impala::CodegenAnyVal::GetVal(), impala::TupleRow::LLVM_CLASS_NAME, impala::ExprContext::LLVM_CLASS_NAME, impala::Status::ok(), impala::LlvmCodeGen::true_value(), impala::ExecNode::type(), impala::TYPE_BOOLEAN, impala::TYPE_INT, and VLOG_QUERY.
Referenced by impala::HdfsAvroScanner::CodegenDecodeAvroData(), impala::HashJoinNode::CodegenProcessProbeBatch(), and impala::PartitionedHashJoinNode::CodegenProcessProbeBatch().
|
inherited |
Collect all nodes of given 'node_type' that are part of this subtree, and return in 'nodes'.
Definition at line 359 of file exec-node.cc.
References impala::ExecNode::children_, and impala::ExecNode::type_.
Referenced by impala::ExecNode::CollectScanNodes(), and impala::PlanFragmentExecutor::Prepare().
|
inherited |
Collect all scan node types.
Definition at line 366 of file exec-node.cc.
References impala::ExecNode::CollectNodes().
Referenced by impala::PlanFragmentExecutor::Prepare().
|
inlineinherited |
Definition at line 152 of file exec-node.h.
References impala::ExecNode::conjunct_ctxs_.
Referenced by impala::HdfsScanNode::ComputeSlotMaterializationOrder(), impala::SelectNode::CopyRows(), impala::UnionNode::EvalAndMaterializeExprs(), impala::HashJoinNode::GetNext(), impala::PartitionedHashJoinNode::OutputUnmatchedBuild(), impala::HashJoinNode::ProcessProbeBatch(), and impala::PartitionedHashJoinNode::ProcessProbeBatch().
|
staticprotectedinherited |
Create a single exec node derived from thrift node; place exec node in 'pool'.
Definition at line 260 of file exec-node.cc.
References impala::ObjectPool::Add(), impala::Status::OK, and RETURN_IF_ERROR.
Referenced by impala::ExecNode::CreateTreeHelper().
|
staticinherited |
Creates exec node tree from list of nodes contained in plan via depth-first traversal. All nodes are placed in pool. Returns error if 'plan' is corrupted, otherwise success.
Definition at line 199 of file exec-node.cc.
References impala::ExecNode::CreateTreeHelper(), impala::Status::OK, and impala::Status::ok().
Referenced by impala::PlanFragmentExecutor::Prepare().
|
staticprotectedinherited |
Definition at line 218 of file exec-node.cc.
References impala::RuntimeProfile::AddChild(), impala::ExecNode::children_, impala::ExecNode::CreateNode(), impala::Status::OK, RETURN_IF_ERROR, and impala::ExecNode::runtime_profile().
Referenced by impala::ExecNode::CreateTree().
|
protectedvirtual |
Write debug string of this into out.
Reimplemented from impala::ExecNode.
Definition at line 363 of file data-source-scan-node.cc.
|
inherited |
Returns a string representation in DFS order of the plan rooted at this.
Definition at line 345 of file exec-node.cc.
Referenced by impala::SortNode::DebugString(), impala::TopNNode::DebugString(), impala::ExchangeNode::DebugString(), impala::AggregationNode::DebugString(), impala::AnalyticEvalNode::DebugString(), impala::PartitionedAggregationNode::DebugString(), impala::BlockingJoinNode::DebugString(), and impala::PlanFragmentExecutor::Prepare().
|
staticinherited |
Evaluate ExprContexts over row. Returns true if all exprs return true. TODO: This doesn't use the vector<Expr*> signature because I haven't figured out how to deal with declaring a templated std:vector type in IR
Definition at line 393 of file exec-node.cc.
References impala::ExprContext::GetBooleanVal(), impala_udf::AnyVal::is_null, and impala_udf::BooleanVal::val.
Referenced by impala::SelectNode::CopyRows(), impala::UnionNode::EvalAndMaterializeExprs(), impala::HdfsScanner::EvalConjuncts(), EvalOtherJoinConjuncts(), EvalOtherJoinConjuncts2(), impala::PartitionedHashJoinNode::EvaluateNullProbe(), impala::HBaseScanNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::PartitionedAggregationNode::GetNext(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::PartitionedHashJoinNode::OutputNullAwareProbeRows(), impala::PartitionedHashJoinNode::OutputUnmatchedBuild(), impala::CrossJoinNode::ProcessLeftChildBatch(), impala::HashJoinNode::ProcessProbeBatch(), and impala::PartitionedHashJoinNode::ProcessProbeBatch().
|
protectedinherited |
Executes debug_action_ if phase matches debug_phase_. 'phase' must not be INVALID.
Definition at line 378 of file exec-node.cc.
References impala::Status::CANCELLED, impala::ExecNode::debug_action_, impala::ExecNode::debug_phase_, impala::RuntimeState::is_cancelled(), and impala::Status::OK.
Referenced by impala::SelectNode::GetNext(), impala::SortNode::GetNext(), impala::UnionNode::GetNext(), impala::HBaseScanNode::GetNext(), impala::TopNNode::GetNext(), impala::ExchangeNode::GetNext(), impala::CrossJoinNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::AnalyticEvalNode::GetNext(), impala::PartitionedHashJoinNode::GetNext(), impala::PartitionedAggregationNode::GetNext(), impala::HdfsScanNode::GetNextInternal(), impala::ExecNode::Open(), and impala::ExecNode::Prepare().
|
inlineinherited |
Definition at line 163 of file exec-node.h.
References impala::ExecNode::expr_mem_tracker_.
Referenced by impala::PartitionedAggregationNode::Partition::InitStreams(), impala::SortNode::Prepare(), impala::UnionNode::Prepare(), impala::TopNNode::Prepare(), impala::ExchangeNode::Prepare(), impala::HashJoinNode::Prepare(), impala::AggregationNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::AnalyticEvalNode::Prepare(), impala::ExecNode::Prepare(), impala::PartitionedAggregationNode::Prepare(), and impala::HdfsScanNode::Prepare().
|
virtual |
Fill the next row batch, calls GetNext() on the external scanner.
Implements impala::ExecNode.
Definition at line 292 of file data-source-scan-node.cc.
References impala::RowBatch::AddRow(), impala::MemPool::Allocate(), impala::RowBatch::AtCapacity(), impala::RowBatch::CommitLastRow(), COUNTER_SET, impala::RowBatch::GetRow(), impala::RowBatch::MaxTupleBufferSize(), impala::OK, RETURN_IF_CANCELLED, RETURN_IF_ERROR, SCOPED_TIMER, impala::TupleRow::SetTuple(), and impala::RowBatch::tuple_data_pool().
|
private |
Gets the next batch from the data source, stored in input_batch_.
Definition at line 144 of file data-source-scan-node.cc.
References cols_next_val_idx_, data_source_executor_, data_src_node_, input_batch_, InputBatchHasNext(), next_row_idx_, impala::Status::OK, RETURN_IF_ERROR, scan_handle_, ValidateRowBatchSize(), and VLOG_QUERY.
Referenced by Open().
|
staticinherited |
Extract node id from p->name().
Definition at line 62 of file exec-node.cc.
References impala::RuntimeProfile::metadata().
|
inlineinherited |
Definition at line 154 of file exec-node.h.
References impala::ExecNode::id_.
Referenced by impala::AnalyticEvalNode::AddResultTuple(), impala::AnalyticEvalNode::AddRow(), impala::AnalyticEvalNode::AnalyticEvalNode(), impala::AnalyticEvalNode::GetNext(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::HdfsScanner::InitializeWriteTuplesFn(), impala::HdfsAvroScanner::InitNewRange(), impala::AnalyticEvalNode::InitNextPartition(), impala::PartitionedAggregationNode::MoveHashPartitions(), impala::PartitionedHashJoinNode::NodeDebugString(), impala::AnalyticEvalNode::Open(), impala::HdfsScanNode::Open(), impala::PlanFragmentExecutor::Prepare(), impala::PartitionedHashJoinNode::ProcessBuildInput(), impala::AnalyticEvalNode::ProcessChildBatch(), impala::HdfsScanNode::ScannerThread(), impala::AnalyticEvalNode::TryAddRemainingResults(), impala::AnalyticEvalNode::TryAddResultTupleForCurrRow(), impala::AnalyticEvalNode::TryAddResultTupleForPrevRow(), and impala::AnalyticEvalNode::TryRemoveRowsBeforeWindow().
|
virtualinherited |
Initializes this object from the thrift tnode desc. The subclass should do any initialization that can fail in Init() rather than the ctor. If overridden in subclass, must first call superclass's Init().
Reimplemented in impala::PartitionedAggregationNode, impala::AnalyticEvalNode, impala::PartitionedHashJoinNode, impala::AggregationNode, impala::HashJoinNode, impala::BlockingJoinNode, impala::ExchangeNode, impala::TopNNode, impala::SortNode, and impala::UnionNode.
Definition at line 124 of file exec-node.cc.
References impala::ExecNode::conjunct_ctxs_, impala::Expr::CreateExprTrees(), impala::Status::OK, impala::ExecNode::pool_, and RETURN_IF_ERROR.
Referenced by impala::SortNode::Init(), impala::UnionNode::Init(), impala::TopNNode::Init(), impala::ExchangeNode::Init(), impala::BlockingJoinNode::Init(), impala::AggregationNode::Init(), impala::AnalyticEvalNode::Init(), and impala::PartitionedAggregationNode::Init().
|
protectedinherited |
Definition at line 371 of file exec-node.cc.
References impala::ExecNode::id_, impala::ExecNode::pool_, and impala::ExecNode::runtime_profile_.
Referenced by impala::ExecNode::ExecNode().
|
inlineprivate |
True if input_batch_ has more rows.
Definition at line 115 of file data-source-scan-node.h.
References input_batch_, next_row_idx_, and num_rows_.
Referenced by GetNextInputBatch().
|
inlineprotectedinherited |
Definition at line 242 of file exec-node.h.
References impala::ExecNode::is_closed_.
Referenced by impala::SelectNode::Close(), impala::SortNode::Close(), impala::UnionNode::Close(), impala::TopNNode::Close(), impala::ExchangeNode::Close(), impala::HBaseScanNode::Close(), impala::CrossJoinNode::Close(), impala::HashJoinNode::Close(), impala::AggregationNode::Close(), impala::BlockingJoinNode::Close(), impala::AnalyticEvalNode::Close(), impala::PartitionedHashJoinNode::Close(), impala::PartitionedAggregationNode::Close(), impala::HdfsScanNode::Close(), impala::PartitionedAggregationNode::Partition::Close(), impala::PartitionedHashJoinNode::Partition::Close(), impala::PartitionedHashJoinNode::ReserveTupleStreamBlocks(), impala::PartitionedHashJoinNode::SpillPartition(), impala::PartitionedAggregationNode::SpillPartition(), and impala::PartitionedHashJoinNode::Partition::~Partition().
|
inlinevirtualinherited |
Reimplemented from impala::ExecNode.
Definition at line 93 of file scan-node.h.
|
inlineinherited |
Definition at line 158 of file exec-node.h.
References impala::ExecNode::limit_.
Referenced by impala::CrossJoinNode::GetNext(), and impala::HashJoinNode::LeftJoinGetNext().
|
inlineinherited |
Definition at line 104 of file scan-node.h.
References impala::ScanNode::materialize_tuple_timer_.
Referenced by impala::HBaseScanNode::GetNext(), impala::HdfsSequenceScanner::ProcessDecompressedBlock(), impala::HdfsTextScanner::ProcessRange(), impala::HdfsAvroScanner::ProcessRange(), impala::HdfsSequenceScanner::ProcessRange(), impala::HdfsRCFileScanner::ProcessRange(), and impala::HdfsTextScanner::WriteFields().
Materializes the next row (next_row_idx_) into tuple_.
Definition at line 193 of file data-source-scan-node.cc.
References impala::MemPool::Allocate(), impala::TupleDescriptor::byte_size(), cols_next_val_idx_, impala::ERROR_INVALID_COL_DATA, impala::ERROR_INVALID_TIMESTAMP, impala::Tuple::GetSlot(), impala::Tuple::Init(), input_batch_, materialized_slots_, next_row_idx_, impala::SlotDescriptor::null_indicator_offset(), RETURN_IF_ERROR, impala::SetDecimalVal(), impala::Tuple::SetNull(), impala::TIMESTAMP_SIZE, tuple_, tuple_desc_, impala::SlotDescriptor::tuple_offset(), impala::ColumnType::type, impala::SlotDescriptor::type(), impala::TYPE_BIGINT, impala::TYPE_BOOLEAN, impala::TYPE_DECIMAL, impala::TYPE_DOUBLE, impala::TYPE_FLOAT, impala::TYPE_INT, impala::TYPE_SMALLINT, impala::TYPE_STRING, impala::TYPE_TIMESTAMP, and impala::TYPE_TINYINT.
|
inlineinherited |
Definition at line 162 of file exec-node.h.
References impala::ExecNode::mem_tracker_.
Referenced by impala::ExecNode::Close(), impala::CrossJoinNode::ConstructBuildSide(), impala::HashJoinNode::ConstructBuildSide(), impala::HdfsScanNode::EnoughMemoryForScannerThread(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::SortNode::Open(), impala::TopNNode::Open(), impala::AggregationNode::Open(), impala::AnalyticEvalNode::Open(), impala::PartitionedAggregationNode::Open(), impala::HdfsScanNode::Open(), impala::UnionNode::OpenCurrentChild(), impala::SelectNode::Prepare(), impala::HBaseScanNode::Prepare(), impala::TopNNode::Prepare(), impala::BlockingJoinNode::Prepare(), impala::HashJoinNode::Prepare(), impala::AggregationNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::AnalyticEvalNode::Prepare(), impala::PartitionedAggregationNode::Prepare(), impala::HdfsScanNode::Prepare(), impala::PartitionedHashJoinNode::PrepareNextPartition(), impala::PartitionedHashJoinNode::ProcessBuildInput(), impala::PartitionedAggregationNode::ProcessStream(), impala::HdfsRCFileScanner::ReadRowGroup(), impala::HdfsAvroScanner::ResolveSchemas(), impala::SortNode::SortInput(), and impala::HdfsScanner::StartNewRowBatch().
|
virtual |
Open the data source and initialize the first row batch.
Reimplemented from impala::ExecNode.
Definition at line 92 of file data-source-scan-node.cc.
References impala::TableDescriptor::col_names(), impala::SlotDescriptor::col_pos(), data_source_executor_, data_src_node_, impala::RuntimeState::effective_user(), GetNextInputBatch(), materialized_slots_, impala::TableDescriptor::name(), impala::ExecNode::Open(), impala::RuntimeState::query_id(), RETURN_IF_CANCELLED, RETURN_IF_ERROR, impala::ExecNode::runtime_profile_, scan_handle_, SCOPED_TIMER, impala::TupleDescriptor::table_desc(), impala::ColumnType::ToThrift(), tuple_desc_, and impala::SlotDescriptor::type().
|
inlineinherited |
Definition at line 101 of file scan-node.h.
References impala::ScanNode::per_read_thread_throughput_counter_.
|
virtual |
Load the data source library and create the ExternalDataSourceExecutor.
Reimplemented from impala::ScanNode.
Definition at line 74 of file data-source-scan-node.cc.
References cols_next_val_idx_, data_source_executor_, data_src_node_, impala::RuntimeState::desc_tbl(), impala::DescriptorTbl::GetTupleDescriptor(), impala::SlotDescriptor::is_materialized(), materialized_slots_, impala::Status::OK, impala::ScanNode::Prepare(), RETURN_IF_ERROR, impala::TupleDescriptor::slots(), and tuple_desc_.
|
protectedvirtualinherited |
Frees any local allocations made by expr_ctxs_to_free_ and returns the result of state->CheckQueryState(). Nodes should call this periodically, e.g. once per input row batch. This should not be called outside the main execution thread. Nodes may override this to add extra periodic cleanup, e.g. freeing other local allocations. ExecNodes overriding this function should return ExecNode::QueryMaintenance().
Reimplemented in impala::PartitionedAggregationNode, and impala::AnalyticEvalNode.
Definition at line 401 of file exec-node.cc.
References impala::RuntimeState::CheckQueryState(), impala::ExecNode::expr_ctxs_to_free_, and impala::ExprContext::FreeLocalAllocations().
Referenced by impala::CrossJoinNode::ConstructBuildSide(), impala::HashJoinNode::ConstructBuildSide(), impala::SelectNode::GetNext(), impala::SortNode::GetNext(), impala::UnionNode::GetNext(), impala::HBaseScanNode::GetNext(), impala::TopNNode::GetNext(), impala::ExchangeNode::GetNext(), impala::CrossJoinNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::PartitionedHashJoinNode::GetNext(), impala::HdfsScanNode::GetNextInternal(), impala::HBaseScanNode::Open(), impala::SortNode::Open(), impala::TopNNode::Open(), impala::BlockingJoinNode::Open(), impala::AggregationNode::Open(), impala::PartitionedHashJoinNode::ProcessBuildInput(), impala::AnalyticEvalNode::QueryMaintenance(), impala::PartitionedAggregationNode::QueryMaintenance(), and impala::SortNode::SortInput().
|
inlineinherited |
Definition at line 159 of file exec-node.h.
References impala::ExecNode::limit_, and impala::ExecNode::num_rows_returned_.
Referenced by impala::HdfsParquetScanner::AssembleRows(), impala::SelectNode::CopyRows(), impala::UnionNode::EvalAndMaterializeExprs(), impala::HdfsTextScanner::FinishScanRange(), impala::SelectNode::GetNext(), impala::UnionNode::GetNext(), impala::SortNode::GetNext(), impala::HBaseScanNode::GetNext(), impala::ExchangeNode::GetNext(), impala::CrossJoinNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::PartitionedHashJoinNode::GetNext(), impala::AnalyticEvalNode::GetNext(), impala::PartitionedAggregationNode::GetNext(), impala::HdfsScanNode::GetNextInternal(), impala::ExchangeNode::GetNextMerging(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::HashJoinNode::LeftJoinGetNext(), impala::HdfsSequenceScanner::ProcessBlockCompressedScanRange(), impala::HdfsTextScanner::ProcessRange(), impala::HdfsAvroScanner::ProcessRange(), impala::HdfsSequenceScanner::ProcessRange(), impala::HdfsRCFileScanner::ProcessRange(), and impala::PlanFragmentExecutor::ReachedLimit().
|
inlineinherited |
Definition at line 97 of file scan-node.h.
References impala::ScanNode::read_timer_.
Referenced by impala::HBaseTableScanner::Next(), and impala::HdfsScanNode::Open().
|
virtual |
|
inlineinherited |
Definition at line 156 of file exec-node.h.
References impala::ExecNode::row_descriptor_.
Referenced by impala::CrossJoinNode::BuildListDebugString(), impala::HashJoinNode::CodegenCreateOutputRow(), impala::PartitionedHashJoinNode::CodegenCreateOutputRow(), impala::CrossJoinNode::ConstructBuildSide(), impala::HashJoinNode::ConstructBuildSide(), impala::BlockingJoinNode::GetLeftChildRowString(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::PartitionedAggregationNode::Partition::InitStreams(), impala::TopNNode::Open(), impala::AggregationNode::Open(), impala::AnalyticEvalNode::Open(), impala::PartitionedAggregationNode::Open(), impala::UnionNode::OpenCurrentChild(), impala::SelectNode::Prepare(), impala::SortNode::Prepare(), impala::UnionNode::Prepare(), impala::TopNNode::Prepare(), impala::BlockingJoinNode::Prepare(), impala::HashJoinNode::Prepare(), impala::AggregationNode::Prepare(), impala::AnalyticEvalNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::ExecNode::Prepare(), impala::PlanFragmentExecutor::Prepare(), impala::PartitionedAggregationNode::Prepare(), impala::HdfsScanNode::Prepare(), impala::PartitionedHashJoinNode::ProcessBuildInput(), impala::PartitionedAggregationNode::ProcessStream(), impala::PlanFragmentExecutor::row_desc(), impala::SortNode::SortInput(), and impala::HdfsScanner::StartNewRowBatch().
|
inlineinherited |
Definition at line 96 of file scan-node.h.
References impala::ScanNode::rows_read_counter_.
Referenced by impala::HdfsParquetScanner::AssembleRows(), impala::HdfsTextScanner::FinishScanRange(), impala::HdfsSequenceScanner::ProcessDecompressedBlock(), impala::HdfsParquetScanner::ProcessFooter(), impala::HdfsTextScanner::ProcessRange(), impala::HdfsAvroScanner::ProcessRange(), impala::HdfsSequenceScanner::ProcessRange(), and impala::HdfsRCFileScanner::ProcessRange().
|
inlineinherited |
Definition at line 157 of file exec-node.h.
References impala::ExecNode::num_rows_returned_.
Referenced by impala::CrossJoinNode::GetNext(), impala::AnalyticEvalNode::GetNext(), impala::HashJoinNode::LeftJoinGetNext(), impala::PartitionedAggregationNode::Open(), impala::HdfsSequenceScanner::ProcessDecompressedBlock(), impala::CrossJoinNode::ProcessLeftChildBatch(), impala::HashJoinNode::ProcessProbeBatch(), and impala::HdfsTextScanner::WriteFields().
|
inlineinherited |
Definition at line 161 of file exec-node.h.
References impala::ExecNode::runtime_profile_.
Referenced by impala::ExecNode::AddRuntimeExecOption(), impala::BlockingJoinNode::BuildSideThread(), impala::ExecNode::CreateTreeHelper(), impala::PartitionedAggregationNode::Partition::InitStreams(), impala::SortNode::Open(), impala::AnalyticEvalNode::Open(), impala::PartitionedAggregationNode::Open(), impala::HdfsScanNode::Open(), impala::HdfsTextScanner::Prepare(), impala::HBaseScanNode::Prepare(), impala::BaseSequenceScanner::Prepare(), impala::ExchangeNode::Prepare(), impala::HdfsParquetScanner::Prepare(), impala::BlockingJoinNode::Prepare(), impala::HashJoinNode::Prepare(), impala::AggregationNode::Prepare(), impala::AnalyticEvalNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::ExecNode::Prepare(), impala::ScanNode::Prepare(), impala::PlanFragmentExecutor::Prepare(), impala::PartitionedAggregationNode::Prepare(), impala::HdfsScanner::Prepare(), impala::HdfsScanNode::Prepare(), and impala::PartitionedHashJoinNode::ProcessBuildInput().
|
inlineinherited |
Definition at line 107 of file scan-node.h.
References impala::ScanNode::scan_ranges_complete_counter_.
|
inlineinherited |
Definition at line 110 of file scan-node.h.
References impala::ScanNode::scanner_thread_counters_.
Referenced by impala::HBaseScanNode::GetNext(), and impala::HdfsScanNode::ScannerThread().
|
staticinherited |
Set debug action for node with given id in 'tree'.
Definition at line 332 of file exec-node.cc.
References impala::ExecNode::children_, impala::ExecNode::debug_action_, impala::ExecNode::debug_phase_, and impala::ExecNode::id_.
Referenced by impala::PlanFragmentExecutor::Prepare().
|
inlineinherited |
This should be called before Prepare(), and the argument must be not destroyed until after Prepare().
Definition at line 89 of file scan-node.h.
References impala::ScanNode::scan_range_params_.
Referenced by impala::PlanFragmentExecutor::Prepare().
|
inlineinherited |
Definition at line 98 of file scan-node.h.
References impala::ScanNode::total_throughput_counter_.
Referenced by impala::HBaseScanNode::Close(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
inlineinherited |
Definition at line 155 of file exec-node.h.
References impala::ExecNode::type_.
Referenced by impala::ExecNode::CodegenEvalConjuncts(), impala::PartitionedAggregationNode::CodegenUpdateTuple(), and impala::PlanFragmentExecutor::Prepare().
|
private |
Validate row_batch_ contains the correct number of columns and that columns contain the same number of rows.
Definition at line 124 of file data-source-scan-node.cc.
References impala::ERROR_MISMATCHED_COL_SIZES, impala::ERROR_NUM_COLUMNS, input_batch_, materialized_slots_, num_rows_, and impala::Status::OK.
Referenced by GetNextInputBatch().
|
protectedinherited |
The number of active hdfs reading threads reading for this node.
Definition at line 164 of file scan-node.h.
Referenced by impala::HdfsScanNode::Close(), and impala::HdfsScanNode::Open().
|
protectedinherited |
The number of active scanner threads that are not blocked by IO.
Definition at line 157 of file scan-node.h.
Referenced by impala::ScanNode::active_scanner_thread_counter(), impala::HdfsScanNode::Close(), impala::HdfsScanNode::EnoughMemoryForScannerThread(), impala::HdfsScanNode::Open(), impala::HdfsScanNode::ScannerThread(), and impala::HdfsScanNode::ThreadTokenAvailableCb().
|
staticinherited |
Definition at line 133 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open().
|
protectedinherited |
Average number of active hdfs reading threads This should be created in Open and stopped when all the scanner threads are done.
Definition at line 168 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
staticinherited |
Definition at line 132 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open().
|
protectedinherited |
Average number of active scanner threads This should be created in Open and stopped when all the scanner threads are done.
Definition at line 161 of file scan-node.h.
Referenced by impala::ScanNode::average_scanner_thread_concurrency(), impala::HdfsScanNode::Open(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
staticinherited |
names of ScanNode common counters
Definition at line 121 of file scan-node.h.
Referenced by impala::ScanNode::Prepare().
|
protectedinherited |
Definition at line 140 of file scan-node.h.
Referenced by impala::ScanNode::bytes_read_counter(), impala::HdfsScanNode::Open(), and impala::ScanNode::Prepare().
|
protectedinherited |
Time series of the bytes_read_counter_.
Definition at line 142 of file scan-node.h.
Referenced by impala::HBaseScanNode::Close(), impala::ScanNode::Prepare(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
protectedinherited |
Definition at line 214 of file exec-node.h.
Referenced by impala::ExecNode::child(), impala::ExecNode::Close(), impala::ExecNode::CollectNodes(), impala::ExecNode::CreateTreeHelper(), impala::HBaseScanNode::DebugString(), impala::UnionNode::GetNext(), impala::UnionNode::Open(), impala::AggregationNode::Open(), impala::PartitionedAggregationNode::Open(), impala::UnionNode::OpenCurrentChild(), impala::ExecNode::Prepare(), impala::PartitionedAggregationNode::ProcessStream(), impala::ExecNode::Reset(), and impala::ExecNode::SetDebugOptions().
|
private |
The indexes of the next non-null value in the row batch, per column. Should always contain materialized_slots_.size() integers. All values are reset to 0 when getting the next row batch.
Definition at line 102 of file data-source-scan-node.h.
Referenced by GetNextInputBatch(), MaterializeNextRow(), and Prepare().
|
protectedinherited |
Definition at line 212 of file exec-node.h.
Referenced by impala::ExecNode::Close(), impala::HashJoinNode::CodegenProcessProbeBatch(), impala::PartitionedHashJoinNode::CodegenProcessProbeBatch(), impala::ExecNode::conjunct_ctxs(), impala::SelectNode::CopyRows(), impala::UnionNode::EvalAndMaterializeExprs(), impala::HBaseScanNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::PartitionedAggregationNode::GetNext(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::TopNNode::Init(), impala::ExecNode::Init(), impala::PartitionedHashJoinNode::Init(), impala::ExecNode::Open(), impala::PartitionedHashJoinNode::OutputUnmatchedBuild(), impala::ExecNode::Prepare(), impala::CrossJoinNode::ProcessLeftChildBatch(), impala::HashJoinNode::ProcessProbeBatch(), and impala::PartitionedHashJoinNode::ProcessProbeBatch().
|
private |
Used to call the external data source.
Definition at line 65 of file data-source-scan-node.h.
Referenced by GetNextInputBatch(), Open(), and Prepare().
|
private |
Thrift structure describing the data source scan node.
Definition at line 68 of file data-source-scan-node.h.
Referenced by GetNextInputBatch(), Open(), and Prepare().
|
protectedinherited |
Definition at line 220 of file exec-node.h.
Referenced by impala::ExecNode::ExecDebugAction(), and impala::ExecNode::SetDebugOptions().
|
protectedinherited |
debug-only: if debug_action_ is not INVALID, node will perform action in debug_phase_
Definition at line 219 of file exec-node.h.
Referenced by impala::ExecNode::ExecDebugAction(), and impala::ExecNode::SetDebugOptions().
|
protectedinherited |
Execution options that are determined at runtime. This is added to the runtime profile at Close(). Examples for options logged here would be "Codegen Enabled"
Definition at line 238 of file exec-node.h.
Referenced by impala::ExecNode::AddRuntimeExecOption().
|
protectedinherited |
MemTracker that should be used for ExprContexts.
Definition at line 233 of file exec-node.h.
Referenced by impala::ExecNode::expr_mem_tracker(), and impala::ExecNode::Prepare().
|
protectedinherited |
HDFS read thread concurrency bucket: bucket[i] refers to the number of sample taken where there are i concurrent hdfs read thread running
Definition at line 174 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
protectedinherited |
Definition at line 209 of file exec-node.h.
Referenced by impala::PartitionedAggregationNode::CreateHashPartitions(), impala::ExecNode::id(), impala::ExecNode::InitRuntimeProfile(), impala::PartitionedAggregationNode::NextPartition(), impala::ExchangeNode::Prepare(), impala::PartitionedHashJoinNode::PrepareNextPartition(), impala::PartitionedHashJoinNode::ProcessBuildInput(), and impala::ExecNode::SetDebugOptions().
|
private |
The current result from calling GetNext() on the data source. Contains the thrift representation of the rows.
Definition at line 89 of file data-source-scan-node.h.
Referenced by GetNextInputBatch(), InputBatchHasNext(), MaterializeNextRow(), and ValidateRowBatchSize().
|
protectedinherited |
Definition at line 222 of file exec-node.h.
Referenced by impala::SortNode::GetNext(), impala::HdfsScanNode::GetNextInternal(), impala::ExchangeNode::GetNextMerging(), impala::TopNNode::InsertTupleRow(), impala::HdfsScanNode::limit(), impala::ExecNode::limit(), impala::TopNNode::Open(), and impala::ExecNode::ReachedLimit().
|
staticinherited |
Definition at line 128 of file scan-node.h.
Referenced by impala::ScanNode::Prepare().
|
protectedinherited |
Definition at line 151 of file scan-node.h.
Referenced by impala::ScanNode::materialize_tuple_timer(), and impala::ScanNode::Prepare().
|
private |
Vector containing slot descriptors for all materialized slots. These descriptors are sorted in order of increasing col_pos. TODO: Refactor to base class. HdfsScanNode has this and other nodes could use it.
Definition at line 82 of file data-source-scan-node.h.
Referenced by MaterializeNextRow(), Open(), Prepare(), and ValidateRowBatchSize().
|
protectedinherited |
Account for peak memory used by this node.
Definition at line 230 of file exec-node.h.
Referenced by impala::ExecNode::mem_tracker(), and impala::ExecNode::Prepare().
|
private |
The index of the next row in input_batch_, i.e. the index into TColumnData.is_null.
Definition at line 97 of file data-source-scan-node.h.
Referenced by GetNextInputBatch(), InputBatchHasNext(), and MaterializeNextRow().
|
staticinherited |
Definition at line 127 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open().
|
protectedinherited |
Definition at line 150 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
private |
The number of rows in input_batch_->rows. The data source should have set TRowBatch.num_rows, but we compute it just in case they haven't.
Definition at line 93 of file data-source-scan-node.h.
Referenced by InputBatchHasNext(), and ValidateRowBatchSize().
|
protectedinherited |
Definition at line 223 of file exec-node.h.
Referenced by impala::ExecNode::Close(), impala::SelectNode::CopyRows(), impala::UnionNode::EvalAndMaterializeExprs(), impala::SortNode::GetNext(), impala::HBaseScanNode::GetNext(), impala::TopNNode::GetNext(), impala::ExchangeNode::GetNext(), impala::CrossJoinNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::PartitionedHashJoinNode::GetNext(), impala::AnalyticEvalNode::GetNext(), impala::PartitionedAggregationNode::GetNext(), impala::HdfsScanNode::GetNextInternal(), impala::ExchangeNode::GetNextMerging(), impala::AnalyticEvalNode::GetNextOutputBatch(), impala::HashJoinNode::LeftJoinGetNext(), impala::PartitionedHashJoinNode::OutputUnmatchedBuild(), impala::ExecNode::ReachedLimit(), and impala::ExecNode::rows_returned().
|
staticinherited |
Definition at line 134 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open().
|
protectedinherited |
Definition at line 170 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open(), and impala::HdfsScanNode::ThreadTokenAvailableCb().
|
staticinherited |
Definition at line 126 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open().
|
protectedinherited |
Per thread read throughput [bytes/sec].
Definition at line 149 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open(), and impala::ScanNode::per_read_thread_throughput_counter().
|
protectedinherited |
Definition at line 211 of file exec-node.h.
Referenced by impala::SortNode::Init(), impala::UnionNode::Init(), impala::TopNNode::Init(), impala::ExchangeNode::Init(), impala::HashJoinNode::Init(), impala::AggregationNode::Init(), impala::ExecNode::Init(), impala::PartitionedHashJoinNode::Init(), impala::AnalyticEvalNode::Init(), impala::PartitionedAggregationNode::Init(), impala::ExecNode::InitRuntimeProfile(), impala::HdfsScanNode::Open(), impala::PartitionedHashJoinNode::Prepare(), and impala::PartitionedHashJoinNode::ProcessBuildInput().
|
protectedinherited |
Definition at line 145 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open(), impala::HBaseScanNode::Prepare(), and impala::ScanNode::read_timer().
|
protectedinherited |
Definition at line 215 of file exec-node.h.
Referenced by impala::SortNode::Open(), impala::SortNode::Prepare(), impala::TopNNode::Prepare(), impala::ExchangeNode::Prepare(), and impala::ExecNode::row_desc().
|
staticinherited |
Names of counters shared by all exec nodes.
Definition at line 169 of file exec-node.h.
Referenced by impala::ExecNode::Prepare().
|
staticinherited |
Definition at line 122 of file scan-node.h.
Referenced by impala::ScanNode::Prepare().
|
protectedinherited |
Definition at line 144 of file scan-node.h.
Referenced by impala::HBaseScanNode::GetNext(), impala::ScanNode::Prepare(), and impala::ScanNode::rows_read_counter().
|
protectedinherited |
Definition at line 226 of file exec-node.h.
Referenced by impala::ExecNode::Close(), impala::SelectNode::CopyRows(), impala::UnionNode::EvalAndMaterializeExprs(), impala::SortNode::GetNext(), impala::HBaseScanNode::GetNext(), impala::TopNNode::GetNext(), impala::ExchangeNode::GetNext(), impala::CrossJoinNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::AnalyticEvalNode::GetNext(), impala::PartitionedHashJoinNode::GetNext(), impala::PartitionedAggregationNode::GetNext(), impala::HdfsScanNode::GetNextInternal(), impala::ExchangeNode::GetNextMerging(), impala::HashJoinNode::LeftJoinGetNext(), impala::PartitionedHashJoinNode::OutputUnmatchedBuild(), and impala::ExecNode::Prepare().
|
protectedinherited |
Definition at line 227 of file exec-node.h.
Referenced by impala::ExecNode::Prepare().
|
protectedinherited |
Definition at line 239 of file exec-node.h.
Referenced by impala::ExecNode::AddRuntimeExecOption().
|
protectedinherited |
Definition at line 225 of file exec-node.h.
Referenced by impala::HBaseScanNode::Close(), impala::SelectNode::GetNext(), impala::SortNode::GetNext(), impala::UnionNode::GetNext(), impala::HBaseScanNode::GetNext(), impala::TopNNode::GetNext(), impala::ExchangeNode::GetNext(), impala::CrossJoinNode::GetNext(), impala::HashJoinNode::GetNext(), impala::AggregationNode::GetNext(), impala::AnalyticEvalNode::GetNext(), impala::PartitionedHashJoinNode::GetNext(), impala::PartitionedAggregationNode::GetNext(), impala::HdfsScanNode::GetNext(), impala::ExecNode::InitRuntimeProfile(), impala::SelectNode::Open(), impala::HBaseScanNode::Open(), impala::UnionNode::Open(), impala::SortNode::Open(), impala::TopNNode::Open(), impala::ExchangeNode::Open(), Open(), impala::BlockingJoinNode::Open(), impala::AggregationNode::Open(), impala::AnalyticEvalNode::Open(), impala::PartitionedAggregationNode::Open(), impala::SelectNode::Prepare(), impala::SortNode::Prepare(), impala::UnionNode::Prepare(), impala::TopNNode::Prepare(), impala::BlockingJoinNode::Prepare(), impala::HashJoinNode::Prepare(), impala::AggregationNode::Prepare(), impala::PartitionedHashJoinNode::Prepare(), impala::AnalyticEvalNode::Prepare(), impala::ExecNode::Prepare(), impala::ScanNode::Prepare(), impala::PartitionedAggregationNode::Prepare(), impala::HdfsScanNode::Prepare(), impala::ExecNode::runtime_profile(), and impala::HdfsScanNode::StopAndFinalizeCounters().
|
private |
The opaque handle returned by the data source for the scan.
Definition at line 85 of file data-source-scan-node.h.
Referenced by GetNextInputBatch(), and Open().
|
protectedinherited |
The scan ranges this scan node is responsible for. Not owned.
Definition at line 138 of file scan-node.h.
Referenced by impala::HBaseScanNode::Prepare(), impala::HdfsScanNode::Prepare(), and impala::ScanNode::SetScanRanges().
|
staticinherited |
Definition at line 129 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open().
|
protectedinherited |
Definition at line 152 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open(), and impala::ScanNode::scan_ranges_complete_counter().
|
protectedinherited |
Aggregated scanner thread counters.
Definition at line 154 of file scan-node.h.
Referenced by impala::ScanNode::Prepare(), and impala::ScanNode::scanner_thread_counters().
|
staticinherited |
Definition at line 130 of file scan-node.h.
Referenced by impala::ScanNode::Prepare().
|
staticinherited |
Definition at line 131 of file scan-node.h.
Referenced by impala::HdfsTextScanner::Prepare(), and impala::ScanNode::Prepare().
|
staticinherited |
Definition at line 124 of file scan-node.h.
Referenced by impala::HBaseScanNode::Prepare().
|
staticinherited |
Definition at line 123 of file scan-node.h.
Referenced by impala::HdfsScanNode::Open().
|
staticinherited |
Definition at line 125 of file scan-node.h.
Referenced by impala::ScanNode::Prepare().
|
protectedinherited |
Wall based aggregate read throughput [bytes/sec].
Definition at line 147 of file scan-node.h.
Referenced by impala::ScanNode::Prepare(), and impala::ScanNode::total_throughput_counter().
|
private |
Current tuple.
Definition at line 77 of file data-source-scan-node.h.
Referenced by MaterializeNextRow().
|
private |
Descriptor of tuples read.
Definition at line 71 of file data-source-scan-node.h.
Referenced by MaterializeNextRow(), Open(), and Prepare().
|
private |
Tuple index in tuple row.
Definition at line 74 of file data-source-scan-node.h.
|
protectedinherited |
Definition at line 210 of file exec-node.h.
Referenced by impala::ExecNode::CollectNodes(), and impala::ExecNode::type().