ClickHouse/dbms/src/DataStreams/IBlockInputStream.h

129 lines
4.3 KiB
C++
Raw Normal View History

2011-08-28 02:22:23 +00:00
#pragma once
#include <vector>
#include <memory>
#include <functional>
#include <boost/noncopyable.hpp>
#include <Core/Block.h>
#include <Core/SortDescription.h>
2010-03-01 16:59:51 +00:00
namespace DB
{
class IBlockInputStream;
using BlockInputStreamPtr = std::shared_ptr<IBlockInputStream>;
using BlockInputStreams = std::vector<BlockInputStreamPtr>;
class TableStructureReadLock;
using TableStructureReadLockPtr = std::shared_ptr<TableStructureReadLock>;
using TableStructureReadLocks = std::vector<TableStructureReadLockPtr>;
using TableStructureReadLocksList = std::list<TableStructureReadLockPtr>;
struct Progress;
2017-04-07 20:21:58 +00:00
namespace ErrorCodes
{
extern const int OUTPUT_IS_NOT_SORTED;
}
2011-08-28 02:22:23 +00:00
2017-05-13 22:19:04 +00:00
/** Callback to track the progress of the query.
* Used in IProfilingBlockInputStream and Context.
* The function takes the number of rows in the last block, the number of bytes in the last block.
* Note that the callback can be called from different threads.
*/
using ProgressCallback = std::function<void(const Progress & progress)>;
2011-08-28 02:22:23 +00:00
2017-05-13 22:19:04 +00:00
/** The stream interface for reading data by blocks from the database.
* Relational operations are supposed to be done also as implementations of this interface.
2010-03-01 16:59:51 +00:00
*/
class IBlockInputStream : private boost::noncopyable
2010-03-01 16:59:51 +00:00
{
public:
IBlockInputStream() {}
2017-05-13 22:19:04 +00:00
/** Read next block.
* If there are no more blocks, return an empty block (for which operator `bool` returns false).
*/
virtual Block read() = 0;
2010-03-01 16:59:51 +00:00
2017-05-13 22:19:04 +00:00
/** Get information about the last block received.
*/
virtual BlockExtraInfo getBlockExtraInfo() const
{
throw Exception("Method getBlockExtraInfo is not supported by the data stream " + getName(), ErrorCodes::NOT_IMPLEMENTED);
}
2015-10-12 14:53:16 +00:00
2017-05-13 22:19:04 +00:00
/** Read something before starting all data or after the end of all data.
* In the `readSuffix` function, you can implement a finalization that can lead to an exception.
* readPrefix() must be called before the first call to read().
* readSuffix() should be called after read() returns an empty block, or after a call to cancel(), but not during read() execution.
*/
virtual void readPrefix() {}
virtual void readSuffix() {}
2011-10-31 06:37:12 +00:00
virtual ~IBlockInputStream() {}
2011-09-04 21:23:19 +00:00
2017-05-13 22:19:04 +00:00
/** To output the data stream transformation tree (query execution plan).
*/
virtual String getName() const = 0;
2011-09-04 21:23:19 +00:00
2017-05-13 22:19:04 +00:00
/** The unique identifier of the pipeline part of the query execution.
* Sources with the same identifier are considered identical
* (producing the same data), and can be replaced by one source
* if several queries are executed simultaneously.
* If the source can not be glued together with any other - return the object's address as an identifier.
*/
virtual String getID() const;
2017-04-10 14:50:55 +00:00
/// If this stream generates data in grouped by some keys, return true.
virtual bool isGroupedOutput() const { return false; }
2017-04-07 20:21:58 +00:00
/// If this stream generates data in order by some keys, return true.
virtual bool isSortedOutput() const { return false; }
2017-04-10 14:50:55 +00:00
/// In case of isGroupedOutput or isSortedOutput, return corresponding SortDescription
2017-04-07 20:21:58 +00:00
virtual const SortDescription & getSortDescription() const { throw Exception("Output of " + getName() + " is not sorted", ErrorCodes::OUTPUT_IS_NOT_SORTED); }
BlockInputStreams & getChildren() { return children; }
void dumpTree(std::ostream & ostr, size_t indent = 0, size_t multiplier = 1);
2011-09-04 21:23:19 +00:00
2017-05-13 22:19:04 +00:00
/// Get leaf sources (not including this one).
BlockInputStreams getLeaves();
2012-03-09 04:45:27 +00:00
2017-05-13 22:19:04 +00:00
/// Get the number of rows and bytes read in the leaf sources.
void getLeafRowsBytes(size_t & rows, size_t & bytes);
2012-08-23 23:49:28 +00:00
2017-05-13 22:19:04 +00:00
/** Check the depth of the pipeline.
* If max_depth is specified and the `depth` is greater - throw an exception.
*/
size_t checkDepth(size_t max_depth) const;
2017-05-13 22:19:04 +00:00
/** Do not allow to change the table while the blocks stream is alive.
*/
void addTableLock(const TableStructureReadLockPtr & lock) { table_locks.push_back(lock); }
2013-05-04 05:20:07 +00:00
protected:
TableStructureReadLocks table_locks;
2012-03-09 04:45:27 +00:00
BlockInputStreams children;
2012-03-09 04:45:27 +00:00
private:
void getLeavesImpl(BlockInputStreams & res, BlockInputStreamPtr this_shared_ptr = nullptr);
size_t checkDepthImpl(size_t max_depth, size_t level) const;
2013-05-04 05:20:07 +00:00
2017-05-13 22:19:04 +00:00
/** Get text that identifies this source and the entire subtree.
* Unlike getID - without taking into account the parameters.
*/
String getTreeID() const;
2010-03-01 16:59:51 +00:00
};
2011-08-28 02:22:23 +00:00
2010-03-01 16:59:51 +00:00
}