2017-04-01 09:19:00 +00:00
|
|
|
#include <Core/Defines.h>
|
|
|
|
#include <Core/Block.h>
|
2012-08-26 11:14:52 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <IO/WriteHelpers.h>
|
|
|
|
#include <IO/VarInt.h>
|
|
|
|
#include <IO/CompressedWriteBuffer.h>
|
2011-08-19 19:18:15 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <Columns/ColumnArray.h>
|
|
|
|
#include <Columns/ColumnNullable.h>
|
|
|
|
#include <Columns/ColumnsNumber.h>
|
2012-08-26 11:14:52 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <DataTypes/DataTypeArray.h>
|
|
|
|
#include <DataTypes/DataTypeNullable.h>
|
|
|
|
#include <DataTypes/DataTypesNumber.h>
|
2012-03-19 12:57:56 +00:00
|
|
|
|
2017-04-01 09:19:00 +00:00
|
|
|
#include <DataStreams/MarkInCompressedFile.h>
|
|
|
|
#include <DataStreams/NativeBlockOutputStream.h>
|
2011-08-19 19:18:15 +00:00
|
|
|
|
2017-07-13 20:58:19 +00:00
|
|
|
#include <Common/typeid_cast.h>
|
2011-08-19 19:18:15 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2015-08-16 07:01:41 +00:00
|
|
|
NativeBlockOutputStream::NativeBlockOutputStream(
|
2017-04-01 07:20:54 +00:00
|
|
|
WriteBuffer & ostr_, UInt64 client_revision_,
|
|
|
|
WriteBuffer * index_ostr_, size_t initial_size_of_file_)
|
|
|
|
: ostr(ostr_), client_revision(client_revision_),
|
|
|
|
index_ostr(index_ostr_), initial_size_of_file(initial_size_of_file_)
|
2015-08-16 07:01:41 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
if (index_ostr)
|
|
|
|
{
|
|
|
|
ostr_concrete = typeid_cast<CompressedWriteBuffer *>(&ostr);
|
|
|
|
if (!ostr_concrete)
|
|
|
|
throw Exception("When need to write index for NativeBlockOutputStream, ostr must be CompressedWriteBuffer.", ErrorCodes::LOGICAL_ERROR);
|
|
|
|
}
|
2015-08-16 07:01:41 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2016-08-13 01:57:35 +00:00
|
|
|
void NativeBlockOutputStream::flush()
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
ostr.next();
|
2016-08-13 01:57:35 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2015-04-16 10:48:35 +00:00
|
|
|
void NativeBlockOutputStream::writeData(const IDataType & type, const ColumnPtr & column, WriteBuffer & ostr, size_t offset, size_t limit)
|
2012-08-26 11:14:52 +00:00
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
/** If there are columns-constants - then we materialize them.
|
|
|
|
* (Since the data type does not know how to serialize / deserialize constants.)
|
|
|
|
*/
|
|
|
|
ColumnPtr full_column;
|
|
|
|
|
|
|
|
if (auto converted = column->convertToFullColumnIfConst())
|
|
|
|
full_column = converted;
|
|
|
|
else
|
|
|
|
full_column = column;
|
|
|
|
|
|
|
|
if (type.isNullable())
|
|
|
|
{
|
|
|
|
const DataTypeNullable & nullable_type = static_cast<const DataTypeNullable &>(type);
|
|
|
|
const IDataType & nested_type = *nullable_type.getNestedType();
|
|
|
|
|
|
|
|
const ColumnNullable & nullable_col = static_cast<const ColumnNullable &>(*full_column.get());
|
|
|
|
const ColumnPtr & nested_col = nullable_col.getNestedColumn();
|
|
|
|
|
|
|
|
const IColumn & null_map = nullable_col.getNullMapConcreteColumn();
|
|
|
|
DataTypeUInt8{}.serializeBinaryBulk(null_map, ostr, offset, limit);
|
|
|
|
|
|
|
|
writeData(nested_type, nested_col, ostr, offset, limit);
|
|
|
|
}
|
|
|
|
else if (const DataTypeArray * type_arr = typeid_cast<const DataTypeArray *>(&type))
|
|
|
|
{
|
2017-07-21 06:35:58 +00:00
|
|
|
/** For arrays, we serialize the offsets first, and then the values.
|
2017-04-01 07:20:54 +00:00
|
|
|
*/
|
|
|
|
const ColumnArray & column_array = typeid_cast<const ColumnArray &>(*full_column);
|
|
|
|
type_arr->getOffsetsType()->serializeBinaryBulk(*column_array.getOffsetsColumn(), ostr, offset, limit);
|
|
|
|
|
2017-07-21 06:35:58 +00:00
|
|
|
if (!column_array.getData().empty())
|
2017-04-01 07:20:54 +00:00
|
|
|
{
|
|
|
|
const ColumnArray::Offsets_t & offsets = column_array.getOffsets();
|
|
|
|
|
|
|
|
if (offset > offsets.size())
|
|
|
|
return;
|
|
|
|
|
|
|
|
/** offset - from which array to write.
|
|
|
|
* limit - how many arrays should be written, or 0, if you write everything that is.
|
|
|
|
* end - up to which array written part finishes.
|
|
|
|
*
|
|
|
|
* nested_offset - from which nested element to write.
|
|
|
|
* nested_limit - how many nested elements to write, or 0, if you write everything that is.
|
|
|
|
*/
|
|
|
|
|
|
|
|
size_t end = std::min(offset + limit, offsets.size());
|
|
|
|
|
|
|
|
size_t nested_offset = offset ? offsets[offset - 1] : 0;
|
|
|
|
size_t nested_limit = limit
|
|
|
|
? offsets[end - 1] - nested_offset
|
|
|
|
: 0;
|
|
|
|
|
|
|
|
const DataTypePtr & nested_type = type_arr->getNestedType();
|
|
|
|
|
|
|
|
if (limit == 0 || nested_limit)
|
2017-07-21 06:35:58 +00:00
|
|
|
writeData(*nested_type, column_array.getDataPtr(), ostr, nested_offset, nested_limit);
|
2017-04-01 07:20:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
else
|
|
|
|
type.serializeBinaryBulk(*full_column, ostr, offset, limit);
|
2012-08-26 11:14:52 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
|
2011-08-19 19:18:15 +00:00
|
|
|
void NativeBlockOutputStream::write(const Block & block)
|
|
|
|
{
|
2017-04-01 07:20:54 +00:00
|
|
|
/// Additional information about the block.
|
|
|
|
if (client_revision >= DBMS_MIN_REVISION_WITH_BLOCK_INFO)
|
|
|
|
block.info.write(ostr);
|
|
|
|
|
|
|
|
/// Dimensions
|
|
|
|
size_t columns = block.columns();
|
|
|
|
size_t rows = block.rows();
|
|
|
|
|
|
|
|
writeVarUInt(columns, ostr);
|
|
|
|
writeVarUInt(rows, ostr);
|
|
|
|
|
|
|
|
/** The index has the same structure as the data stream.
|
|
|
|
* But instead of column values, it contains a mark that points to the location in the data file where this part of the column is located.
|
|
|
|
*/
|
|
|
|
if (index_ostr)
|
|
|
|
{
|
|
|
|
writeVarUInt(columns, *index_ostr);
|
|
|
|
writeVarUInt(rows, *index_ostr);
|
|
|
|
}
|
|
|
|
|
|
|
|
for (size_t i = 0; i < columns; ++i)
|
|
|
|
{
|
|
|
|
/// For the index.
|
|
|
|
MarkInCompressedFile mark;
|
|
|
|
|
|
|
|
if (index_ostr)
|
|
|
|
{
|
|
|
|
ostr_concrete->next(); /// Finish compressed block.
|
|
|
|
mark.offset_in_compressed_file = initial_size_of_file + ostr_concrete->getCompressedBytes();
|
|
|
|
mark.offset_in_decompressed_block = ostr_concrete->getRemainingBytes();
|
|
|
|
}
|
|
|
|
|
|
|
|
const ColumnWithTypeAndName & column = block.safeGetByPosition(i);
|
|
|
|
|
|
|
|
/// Name
|
|
|
|
writeStringBinary(column.name, ostr);
|
|
|
|
|
|
|
|
/// Type
|
|
|
|
writeStringBinary(column.type->getName(), ostr);
|
|
|
|
|
|
|
|
/// Data
|
|
|
|
if (rows) /// Zero items of data is always represented as zero number of bytes.
|
|
|
|
writeData(*column.type, column.column, ostr, 0, 0);
|
|
|
|
|
|
|
|
if (index_ostr)
|
|
|
|
{
|
|
|
|
writeStringBinary(column.name, *index_ostr);
|
|
|
|
writeStringBinary(column.type->getName(), *index_ostr);
|
|
|
|
|
|
|
|
writeBinary(mark.offset_in_compressed_file, *index_ostr);
|
|
|
|
writeBinary(mark.offset_in_decompressed_block, *index_ostr);
|
|
|
|
}
|
|
|
|
}
|
2011-08-19 19:18:15 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
}
|