ClickHouse/dbms/Storages/MergeTree/MergeTreeReaderWide.cpp

268 lines
10 KiB
C++
Raw Normal View History

2019-10-10 16:30:30 +00:00
#include <Storages/MergeTree/MergeTreeReaderWide.h>
2020-02-25 09:49:45 +00:00
#include <Columns/ColumnArray.h>
2020-02-25 09:49:45 +00:00
#include <DataTypes/DataTypeArray.h>
#include <DataTypes/NestedUtils.h>
2020-02-25 08:53:14 +00:00
#include <Interpreters/inplaceBlockConversions.h>
2020-02-25 09:49:45 +00:00
#include <Storages/MergeTree/IMergeTreeReader.h>
#include <Storages/MergeTree/MergeTreeDataPartWide.h>
2020-02-25 09:49:45 +00:00
#include <Common/escapeForFileName.h>
2017-07-13 20:58:19 +00:00
#include <Common/typeid_cast.h>
2016-07-19 10:57:57 +00:00
namespace DB
{
namespace
{
using OffsetColumns = std::map<std::string, ColumnPtr>;
constexpr auto DATA_FILE_EXTENSION = ".bin";
2016-07-19 10:57:57 +00:00
}
namespace ErrorCodes
{
extern const int LOGICAL_ERROR;
extern const int MEMORY_LIMIT_EXCEEDED;
2016-11-20 12:43:20 +00:00
}
MergeTreeReaderWide::MergeTreeReaderWide(
DataPartWidePtr data_part_,
NamesAndTypesList columns_,
UncompressedCache * uncompressed_cache_,
MarkCache * mark_cache_,
MarkRanges mark_ranges_,
MergeTreeReaderSettings settings_,
IMergeTreeDataPart::ValueSizeMap avg_value_size_hints_,
const ReadBufferFromFileBase::ProfileCallback & profile_callback_,
clockid_t clock_type_)
2020-02-25 09:49:45 +00:00
: IMergeTreeReader(
std::move(data_part_), std::move(columns_), uncompressed_cache_, std::move(mark_cache_),
std::move(mark_ranges_), std::move(settings_), std::move(avg_value_size_hints_))
2016-07-19 10:57:57 +00:00
{
try
{
2020-02-25 09:49:45 +00:00
for (const NameAndTypePair & column_from_part : data_part->getColumns())
2020-02-25 08:53:14 +00:00
columns_from_part[column_from_part.name] = column_from_part.type;
for (const NameAndTypePair & column : columns)
2020-02-25 08:53:14 +00:00
{
if (columns_from_part.count(column.name))
2020-03-30 16:03:08 +00:00
{
2020-02-25 08:53:14 +00:00
addStreams(column.name, *columns_from_part[column.name], profile_callback_, clock_type_);
2020-03-30 16:03:08 +00:00
}
2020-02-25 08:53:14 +00:00
else
2020-03-30 16:03:08 +00:00
{
auto renamed_it = alter_conversions.rename_map.find(column.name);
if (renamed_it != alter_conversions.rename_map.end()
&& columns_from_part.count(renamed_it->second))
addStreams(renamed_it->second, *columns_from_part[renamed_it->second], profile_callback_, clock_type_);
else
addStreams(column.name, *column.type, profile_callback_, clock_type_);
}
2020-02-25 08:53:14 +00:00
}
}
catch (...)
{
storage.reportBrokenPart(data_part->name);
throw;
}
2016-07-19 10:57:57 +00:00
}
2019-11-13 01:57:45 +00:00
2019-12-19 13:10:57 +00:00
size_t MergeTreeReaderWide::readRows(size_t from_mark, bool continue_reading, size_t max_rows_to_read, Columns & res_columns)
2016-07-19 10:57:57 +00:00
{
size_t read_rows = 0;
try
{
2019-09-23 19:22:02 +00:00
size_t num_columns = columns.size();
if (res_columns.size() != num_columns)
throw Exception("invalid number of columns passed to MergeTreeReader::readRows. "
"Expected " + toString(num_columns) + ", "
"got " + toString(res_columns.size()), ErrorCodes::LOGICAL_ERROR);
/// Pointers to offset columns that are common to the nested data structure columns.
/// If append is true, then the value will be equal to nullptr and will be used only to
/// check that the offsets column has been already read.
OffsetColumns offset_columns;
2019-09-23 19:22:02 +00:00
auto name_and_type = columns.begin();
for (size_t pos = 0; pos < num_columns; ++pos, ++name_and_type)
{
2020-03-30 16:03:08 +00:00
String name = name_and_type->name;
if (alter_conversions.rename_map.count(name))
{
String original_name = alter_conversions.rename_map[name];
if (!columns_from_part.count(name) && columns_from_part.count(original_name))
name = original_name;
}
2020-02-25 08:53:14 +00:00
DataTypePtr type;
if (columns_from_part.count(name))
type = columns_from_part[name];
else
type = name_and_type->type;
2019-09-23 19:22:02 +00:00
/// The column is already present in the block so we will append the values to the end.
2019-09-23 19:22:02 +00:00
bool append = res_columns[pos] != nullptr;
if (!append)
2020-02-25 08:53:14 +00:00
res_columns[pos] = type->createColumn();
/// To keep offsets shared. TODO Very dangerous. Get rid of this.
2019-09-23 19:22:02 +00:00
MutableColumnPtr column = res_columns[pos]->assumeMutable();
bool read_offsets = true;
/// For nested data structures collect pointers to offset columns.
2019-09-23 19:22:02 +00:00
if (const auto * type_arr = typeid_cast<const DataTypeArray *>(type.get()))
{
2019-09-23 19:22:02 +00:00
String table_name = Nested::extractTableName(name);
2019-09-23 19:22:02 +00:00
auto it_inserted = offset_columns.emplace(table_name, nullptr);
/// offsets have already been read on the previous iteration and we don't need to read it again
if (!it_inserted.second)
read_offsets = false;
/// need to create new offsets
if (it_inserted.second && !append)
it_inserted.first->second = ColumnArray::ColumnOffsets::create();
/// share offsets in all elements of nested structure
if (!append)
2018-03-20 14:17:09 +00:00
column = ColumnArray::create(type_arr->getNestedType()->createColumn(),
it_inserted.first->second)->assumeMutable();
}
try
{
size_t column_size_before_reading = column->size();
2019-09-23 19:22:02 +00:00
readData(name, *type, *column, from_mark, continue_reading, max_rows_to_read, read_offsets);
/// For elements of Nested, column_size_before_reading may be greater than column size
/// if offsets are not empty and were already read, but elements are empty.
2019-09-23 19:22:02 +00:00
if (!column->empty())
read_rows = std::max(read_rows, column->size() - column_size_before_reading);
}
catch (Exception & e)
{
/// Better diagnostics.
2019-09-23 19:22:02 +00:00
e.addMessage("(while reading column " + name + ")");
throw;
}
2019-09-23 19:22:02 +00:00
if (column->empty())
res_columns[pos] = nullptr;
else
2019-09-23 19:22:02 +00:00
res_columns[pos] = std::move(column);
}
2019-09-23 19:22:02 +00:00
/// NOTE: positions for all streams must be kept in sync.
/// In particular, even if for some streams there are no rows to be read,
/// you must ensure that no seeks are skipped and at this point they all point to to_mark.
}
catch (Exception & e)
{
if (e.code() != ErrorCodes::MEMORY_LIMIT_EXCEEDED)
storage.reportBrokenPart(data_part->name);
/// Better diagnostics.
e.addMessage("(while reading from part " + data_part->getFullPath() + " "
2019-09-23 19:22:02 +00:00
"from mark " + toString(from_mark) + " "
"with max_rows_to_read = " + toString(max_rows_to_read) + ")");
throw;
}
catch (...)
{
storage.reportBrokenPart(data_part->name);
throw;
}
return read_rows;
2016-07-19 10:57:57 +00:00
}
2019-10-10 16:30:30 +00:00
void MergeTreeReaderWide::addStreams(const String & name, const IDataType & type,
const ReadBufferFromFileBase::ProfileCallback & profile_callback, clockid_t clock_type)
2016-07-19 10:57:57 +00:00
{
IDataType::StreamCallback callback = [&] (const IDataType::SubstreamPath & substream_path)
{
String stream_name = IDataType::getFileNameForStream(name, substream_path);
if (streams.count(stream_name))
return;
bool data_file_exists = data_part->checksums.files.count(stream_name + DATA_FILE_EXTENSION);
/** If data file is missing then we will not try to open it.
* It is necessary since it allows to add new column to structure of the table without creating new files for old parts.
*/
if (!data_file_exists)
return;
Data Skipping Indices (#4143) * made index parser * added index parsing * some fixes * added index interface and factory * fixed compilation * ptrs * added indexParts * indextypes * index condition * IndexCondition * added indexes in selectexecutor * fix * changed comment * fix * added granularity * comments * fix * fix * added writing indexes * removed indexpart class * fix * added setSkipIndexes * add rw for MergeTreeIndexes * fixes * upd error * fix * fix * reading * test index * fixed nullptr error * fixed * fix * unique names * asts -> exprlist * minmax index * fix * fixed select * fixed merging * fixed mutation * working minmax * removed test index * fixed style * added indexes to checkDataPart * added tests for minmax index * fixed constructor * fix style * fixed includes * fixed setSkipIndexes * added indexes meta to zookeeper * added parsing * removed throw * alter cmds parse * fix * added alter * fix * alters fix * fix alters * fix "after" * fixed alter * alter fix + test * fixes * upd setSkipIndexes * fixed alter bug with drop all indices * fix metadata editing * new test and repl fix * rm test files * fixed repl alter * fix * fix * indices * MTReadStream * upd test for bug * fix * added useful parsers and ast classes * fix * fix comments * replaced columns * fix * fixed parsing * fixed printing * fix err * basic IndicesDescription * go to IndicesDescr * moved indices * go to indicesDescr * fix test minmax_index* * fixed MT alter * fixed bug with replMT indices storing in zk * rename * refactoring * docs ru * docs ru * docs en * refactor * rename tests * fix docs * refactoring * fix * fix * fix * fixed style * unique idx * unique * fix * better minmax calculation * upd * added getBlock * unique_condition * added termForAST * unique * fixed not * uniqueCondition::mayBeTrueOnGranule * fix * fixed bug with double column * is always true * fix * key set * spaces * test * tests * fix * unique * fix * fix * fixed bug with duplicate column * removed unused data * fix * fixes * __bitSwapLastTwo * fix
2019-02-05 14:50:25 +00:00
streams.emplace(stream_name, std::make_unique<MergeTreeReaderStream>(
data_part->disk, data_part->getFullRelativePath() + stream_name, DATA_FILE_EXTENSION,
data_part->getMarksCount(), all_mark_ranges, settings, mark_cache,
2019-02-27 20:02:48 +00:00
uncompressed_cache, data_part->getFileSizeOrZero(stream_name + DATA_FILE_EXTENSION),
2019-06-19 10:07:56 +00:00
&data_part->index_granularity_info,
2018-11-15 14:06:54 +00:00
profile_callback, clock_type));
};
IDataType::SubstreamPath substream_path;
type.enumerateStreams(callback, substream_path);
2016-07-19 10:57:57 +00:00
}
2019-10-10 16:30:30 +00:00
void MergeTreeReaderWide::readData(
const String & name, const IDataType & type, IColumn & column,
size_t from_mark, bool continue_reading, size_t max_rows_to_read,
bool with_offsets)
2016-07-19 10:57:57 +00:00
{
auto get_stream_getter = [&](bool stream_for_prefix) -> IDataType::InputStreamGetter
{
return [&, stream_for_prefix](const IDataType::SubstreamPath & substream_path) -> ReadBuffer *
{
/// If offsets for arrays have already been read.
if (!with_offsets && substream_path.size() == 1 && substream_path[0].type == IDataType::Substream::ArraySizes)
return nullptr;
String stream_name = IDataType::getFileNameForStream(name, substream_path);
auto it = streams.find(stream_name);
if (it == streams.end())
return nullptr;
Data Skipping Indices (#4143) * made index parser * added index parsing * some fixes * added index interface and factory * fixed compilation * ptrs * added indexParts * indextypes * index condition * IndexCondition * added indexes in selectexecutor * fix * changed comment * fix * added granularity * comments * fix * fix * added writing indexes * removed indexpart class * fix * added setSkipIndexes * add rw for MergeTreeIndexes * fixes * upd error * fix * fix * reading * test index * fixed nullptr error * fixed * fix * unique names * asts -> exprlist * minmax index * fix * fixed select * fixed merging * fixed mutation * working minmax * removed test index * fixed style * added indexes to checkDataPart * added tests for minmax index * fixed constructor * fix style * fixed includes * fixed setSkipIndexes * added indexes meta to zookeeper * added parsing * removed throw * alter cmds parse * fix * added alter * fix * alters fix * fix alters * fix "after" * fixed alter * alter fix + test * fixes * upd setSkipIndexes * fixed alter bug with drop all indices * fix metadata editing * new test and repl fix * rm test files * fixed repl alter * fix * fix * indices * MTReadStream * upd test for bug * fix * added useful parsers and ast classes * fix * fix comments * replaced columns * fix * fixed parsing * fixed printing * fix err * basic IndicesDescription * go to IndicesDescr * moved indices * go to indicesDescr * fix test minmax_index* * fixed MT alter * fixed bug with replMT indices storing in zk * rename * refactoring * docs ru * docs ru * docs en * refactor * rename tests * fix docs * refactoring * fix * fix * fix * fixed style * unique idx * unique * fix * better minmax calculation * upd * added getBlock * unique_condition * added termForAST * unique * fixed not * uniqueCondition::mayBeTrueOnGranule * fix * fixed bug with double column * is always true * fix * key set * spaces * test * tests * fix * unique * fix * fix * fixed bug with duplicate column * removed unused data * fix * fixes * __bitSwapLastTwo * fix
2019-02-05 14:50:25 +00:00
MergeTreeReaderStream & stream = *it->second;
if (stream_for_prefix)
{
stream.seekToStart();
continue_reading = false;
}
else if (!continue_reading)
stream.seekToMark(from_mark);
return stream.data_buffer;
};
};
double & avg_value_size_hint = avg_value_size_hints[name];
2019-12-12 18:55:19 +00:00
IDataType::DeserializeBinaryBulkSettings deserialize_settings;
deserialize_settings.avg_value_size_hint = avg_value_size_hint;
if (deserialize_binary_bulk_state_map.count(name) == 0)
{
2019-12-12 18:55:19 +00:00
deserialize_settings.getter = get_stream_getter(true);
type.deserializeBinaryBulkStatePrefix(deserialize_settings, deserialize_binary_bulk_state_map[name]);
}
2019-12-12 18:55:19 +00:00
deserialize_settings.getter = get_stream_getter(false);
deserialize_settings.continuous_reading = continue_reading;
auto & deserialize_state = deserialize_binary_bulk_state_map[name];
2019-12-12 18:55:19 +00:00
type.deserializeBinaryBulkWithMultipleStreams(column, max_rows_to_read, deserialize_settings, deserialize_state);
IDataType::updateAvgValueSizeHint(column, avg_value_size_hint);
2016-07-19 10:57:57 +00:00
}
}