This is an automated email from the ASF dual-hosted git repository. dataroaring pushed a commit to branch branch-3.0 in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/branch-3.0 by this push: new 1a6a3c09a86 [improve](columns)replace logFatal with exception and del some useless code in columns #38035 (#38863) 1a6a3c09a86 is described below commit 1a6a3c09a86c62e513f3ff60810f0ea1d6c312bf Author: amory <wangqian...@selectdb.com> AuthorDate: Mon Aug 5 15:59:48 2024 +0800 [improve](columns)replace logFatal with exception and del some useless code in columns #38035 (#38863) ## Proposed changes backport: https://github.com/apache/doris/pull/38035 Issue Number: close #xxx <!--Describe your changes.--> --- be/src/common/exception.h | 23 +++++ be/src/olap/compaction.cpp | 27 ++---- be/src/olap/schema_change.h | 4 +- be/src/vec/columns/column.cpp | 25 ----- be/src/vec/columns/column.h | 102 ++++++++++++--------- be/src/vec/columns/column_array.cpp | 36 +++++--- be/src/vec/columns/column_array.h | 3 +- be/src/vec/columns/column_complex.h | 16 ++-- be/src/vec/columns/column_const.cpp | 8 +- be/src/vec/columns/column_decimal.cpp | 4 +- be/src/vec/columns/column_dictionary.h | 62 +++++++++---- be/src/vec/columns/column_dummy.h | 23 +++-- be/src/vec/columns/column_impl.h | 5 +- be/src/vec/columns/column_map.cpp | 29 +++--- be/src/vec/columns/column_map.h | 3 +- be/src/vec/columns/column_nullable.cpp | 10 +- be/src/vec/columns/column_object.cpp | 92 +++++++------------ be/src/vec/columns/column_object.h | 17 ---- be/src/vec/columns/column_string.cpp | 3 +- be/src/vec/columns/column_string.h | 20 +--- be/src/vec/columns/column_struct.cpp | 18 ++-- be/src/vec/columns/column_struct.h | 9 +- be/src/vec/columns/column_vector.cpp | 4 +- be/src/vec/columns/column_vector.h | 3 +- be/src/vec/columns/predicate_column.h | 72 ++++++++++----- .../vec/functions/array/function_arrays_overlap.h | 1 + .../data/nereids_p0/join/test_join_on.out | 5 + .../suites/nereids_p0/join/test_join_on.groovy | 53 +++++++++++ 28 files changed, 391 insertions(+), 286 deletions(-) diff --git a/be/src/common/exception.h b/be/src/common/exception.h index ce44e658749..e8b6d3567a7 100644 --- a/be/src/common/exception.h +++ b/be/src/common/exception.h @@ -139,3 +139,26 @@ inline const std::string& Exception::to_string() const { } \ } \ } while (0); + +#define HANDLE_EXCEPTION_IF_CATCH_EXCEPTION(stmt, exception_handler) \ + do { \ + try { \ + doris::enable_thread_catch_bad_alloc++; \ + Defer defer {[&]() { doris::enable_thread_catch_bad_alloc--; }}; \ + { \ + Status _status_ = (stmt); \ + if (UNLIKELY(!_status_.ok())) { \ + exception_handler(doris::Exception()); \ + return _status_; \ + } \ + } \ + } catch (const doris::Exception& e) { \ + exception_handler(e); \ + if (e.code() == doris::ErrorCode::MEM_ALLOC_FAILED) { \ + return Status::MemoryLimitExceeded(fmt::format( \ + "PreCatch error code:{}, {}, __FILE__:{}, __LINE__:{}, __FUNCTION__:{}", \ + e.code(), e.to_string(), __FILE__, __LINE__, __PRETTY_FUNCTION__)); \ + } \ + return Status::Error<false>(e.code(), e.to_string()); \ + } \ + } while (0); diff --git a/be/src/olap/compaction.cpp b/be/src/olap/compaction.cpp index fbfc0c1caca..8c109eec1c1 100644 --- a/be/src/olap/compaction.cpp +++ b/be/src/olap/compaction.cpp @@ -399,15 +399,14 @@ Status CompactionMixin::execute_compact() { data_dir->disks_compaction_score_increment(permits); data_dir->disks_compaction_num_increment(1); - Status st = execute_compact_impl(permits); - _tablet->compaction_count.fetch_add(1, std::memory_order_relaxed); - - data_dir->disks_compaction_score_increment(-permits); - data_dir->disks_compaction_num_increment(-1); + auto record_compaction_stats = [&](const doris::Exception& ex) { + _tablet->compaction_count.fetch_add(1, std::memory_order_relaxed); + data_dir->disks_compaction_score_increment(-permits); + data_dir->disks_compaction_num_increment(-1); + }; - if (!st.ok()) { - return st; - } + HANDLE_EXCEPTION_IF_CATCH_EXCEPTION(execute_compact_impl(permits), record_compaction_stats); + record_compaction_stats(doris::Exception()); if (enable_compaction_checksum) { EngineChecksumTask checksum_task(_engine, _tablet->tablet_id(), _tablet->schema_hash(), @@ -1182,16 +1181,10 @@ Status CloudCompactionMixin::execute_compact_impl(int64_t permits) { Status CloudCompactionMixin::execute_compact() { TEST_INJECTION_POINT("Compaction::do_compaction"); int64_t permits = get_compaction_permits(); - Status st = execute_compact_impl(permits); - if (!st.ok()) { - LOG(WARNING) << "failed to do " << compaction_name() << ". res=" << st - << ", tablet=" << _tablet->tablet_id() - << ", output_version=" << _output_version; - garbage_collection(); - return st; - } + HANDLE_EXCEPTION_IF_CATCH_EXCEPTION(execute_compact_impl(permits), + [&](const doris::Exception& ex) { garbage_collection(); }); _load_segment_to_cache(); - return st; + return Status::OK(); } Status CloudCompactionMixin::modify_rowsets() { diff --git a/be/src/olap/schema_change.h b/be/src/olap/schema_change.h index eb0f046270d..64ab0c724d0 100644 --- a/be/src/olap/schema_change.h +++ b/be/src/olap/schema_change.h @@ -117,8 +117,8 @@ public: _filtered_rows = 0; _merged_rows = 0; - RETURN_IF_ERROR(_inner_process(rowset_reader, rowset_writer, new_tablet, base_tablet_schema, - new_tablet_schema)); + RETURN_IF_ERROR_OR_CATCH_EXCEPTION(_inner_process(rowset_reader, rowset_writer, new_tablet, + base_tablet_schema, new_tablet_schema)); // Check row num changes if (!_check_row_nums(rowset_reader, *rowset_writer)) { diff --git a/be/src/vec/columns/column.cpp b/be/src/vec/columns/column.cpp index 85e36d163e4..2760604f0ba 100644 --- a/be/src/vec/columns/column.cpp +++ b/be/src/vec/columns/column.cpp @@ -87,29 +87,4 @@ bool is_column_const(const IColumn& column) { return check_column<ColumnConst>(column); } -ColumnPtr IColumn::create_with_offsets(const Offsets64& offsets, const Field& default_field, - size_t total_rows, size_t shift) const { - if (offsets.size() + shift != size()) { - LOG(FATAL) << fmt::format( - "Incompatible sizes of offsets ({}), shift ({}) and size of column {}", - offsets.size(), shift, size()); - } - auto res = clone_empty(); - res->reserve(total_rows); - ssize_t current_offset = -1; - for (size_t i = 0; i < offsets.size(); ++i) { - ssize_t offsets_diff = static_cast<ssize_t>(offsets[i]) - current_offset; - current_offset = offsets[i]; - if (offsets_diff > 1) { - res->insert_many(default_field, offsets_diff - 1); - } - res->insert_from(*this, i + shift); - } - ssize_t offsets_diff = static_cast<ssize_t>(total_rows) - current_offset; - if (offsets_diff > 1) { - res->insert_many(default_field, offsets_diff - 1); - } - return res; -} - } // namespace doris::vectorized diff --git a/be/src/vec/columns/column.h b/be/src/vec/columns/column.h index fd8d1688c33..35869955d44 100644 --- a/be/src/vec/columns/column.h +++ b/be/src/vec/columns/column.h @@ -124,13 +124,15 @@ public: /// If size is less current size, then data is cut. /// If size is greater, than default values are appended. virtual MutablePtr clone_resized(size_t s) const { - LOG(FATAL) << "Cannot clone_resized() column " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method clone_resized is not supported for " + get_name()); return nullptr; } // shrink the end zeros for CHAR type or ARRAY<CHAR> type virtual MutablePtr get_shrinked_column() { - LOG(FATAL) << "Cannot get_shrinked_column() column " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method get_shrinked_column is not supported for " + get_name()); return nullptr; } @@ -164,7 +166,8 @@ public: virtual StringRef get_data_at(size_t n) const = 0; virtual Int64 get_int(size_t /*n*/) const { - LOG(FATAL) << "Method get_int is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method get_int is not supported for " + get_name()); return 0; } @@ -175,7 +178,8 @@ public: * Otherwise throw an exception. */ virtual bool get_bool(size_t /*n*/) const { - LOG(FATAL) << "Method get_bool is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method get_bool is not supported for " + get_name()); return false; } @@ -230,42 +234,51 @@ public: virtual void insert_data(const char* pos, size_t length) = 0; virtual void insert_many_fix_len_data(const char* pos, size_t num) { - LOG(FATAL) << "Method insert_many_fix_len_data is not supported for " << get_name(); + throw doris::Exception( + ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method insert_many_fix_len_data is not supported for " + get_name()); } // todo(zeno) Use dict_args temp object to cover all arguments virtual void insert_many_dict_data(const int32_t* data_array, size_t start_index, const StringRef* dict, size_t data_num, uint32_t dict_num = 0) { - LOG(FATAL) << "Method insert_many_dict_data is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method insert_many_dict_data is not supported for " + get_name()); } virtual void insert_many_binary_data(char* data_array, uint32_t* len_array, uint32_t* start_offset_array, size_t num) { - LOG(FATAL) << "Method insert_many_binary_data is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method insert_many_binary_data is not supported for " + get_name()); } /// Insert binary data into column from a continuous buffer, the implementation maybe copy all binary data /// in one single time. virtual void insert_many_continuous_binary_data(const char* data, const uint32_t* offsets, const size_t num) { - LOG(FATAL) << "Method insert_many_continuous_binary_data is not supported for " - << get_name(); + throw doris::Exception( + ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method insert_many_continuous_binary_data is not supported for " + get_name()); } virtual void insert_many_strings(const StringRef* strings, size_t num) { - LOG(FATAL) << "Method insert_many_binary_data is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method insert_many_strings is not supported for " + get_name()); } virtual void insert_many_strings_overflow(const StringRef* strings, size_t num, size_t max_length) { - LOG(FATAL) << "Method insert_many_strings_overflow is not supported for " << get_name(); + throw doris::Exception( + ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method insert_many_strings_overflow is not supported for " + get_name()); } // Here `pos` points to the memory data type is the same as the data type of the column. // This function is used by `insert_keys_into_columns` in AggregationNode. virtual void insert_many_raw_data(const char* pos, size_t num) { - LOG(FATAL) << "Method insert_many_raw_data is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method insert_many_raw_data is not supported for " + get_name()); } void insert_many_data(const char* pos, size_t length, size_t data_num) { @@ -309,32 +322,39 @@ public: /// Return the size of largest row. /// This is for calculating the memory size for vectorized serialization of aggregation keys. virtual size_t get_max_row_byte_size() const { - LOG(FATAL) << "get_max_row_byte_size not supported"; + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method get_max_row_byte_size is not supported for " + get_name()); return 0; } virtual void serialize_vec(std::vector<StringRef>& keys, size_t num_rows, size_t max_row_byte_size) const { - LOG(FATAL) << "serialize_vec not supported"; + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method serialize_vec is not supported for " + get_name()); __builtin_unreachable(); } virtual void serialize_vec_with_null_map(std::vector<StringRef>& keys, size_t num_rows, const uint8_t* null_map) const { - LOG(FATAL) << "serialize_vec_with_null_map not supported"; + throw doris::Exception( + ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method serialize_vec_with_null_map is not supported for " + get_name()); __builtin_unreachable(); } // This function deserializes group-by keys into column in the vectorized way. virtual void deserialize_vec(std::vector<StringRef>& keys, const size_t num_rows) { - LOG(FATAL) << "deserialize_vec not supported"; + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method deserialize_vec is not supported for " + get_name()); __builtin_unreachable(); } // Used in ColumnNullable::deserialize_vec virtual void deserialize_vec_with_null_map(std::vector<StringRef>& keys, const size_t num_rows, const uint8_t* null_map) { - LOG(FATAL) << "deserialize_vec_with_null_map not supported"; + throw doris::Exception( + ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method deserialize_vec_with_null_map is not supported for " + get_name()); __builtin_unreachable(); } @@ -343,7 +363,8 @@ public: /// On subsequent calls of this method for sequence of column values of arbitrary types, /// passed bytes to hash must identify sequence of values unambiguously. virtual void update_hash_with_value(size_t n, SipHash& hash) const { - LOG(FATAL) << get_name() << " update_hash_with_value siphash not supported"; + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method update_hash_with_value is not supported for " + get_name()); } /// Update state of hash function with value of n elements to avoid the virtual function call @@ -352,13 +373,17 @@ public: /// do xxHash here, faster than other sip hash virtual void update_hashes_with_value(uint64_t* __restrict hashes, const uint8_t* __restrict null_data = nullptr) const { - LOG(FATAL) << get_name() << " update_hashes_with_value xxhash not supported"; + throw doris::Exception( + ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method update_hashes_with_value is not supported for " + get_name()); } // use range for one hash value to avoid virtual function call in loop virtual void update_xxHash_with_value(size_t start, size_t end, uint64_t& hash, const uint8_t* __restrict null_data) const { - LOG(FATAL) << get_name() << " update_hash_with_value xxhash not supported"; + throw doris::Exception( + ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method update_xxHash_with_value is not supported for " + get_name()); } /// Update state of crc32 hash function with value of n elements to avoid the virtual function call @@ -367,13 +392,15 @@ public: virtual void update_crcs_with_value(uint32_t* __restrict hash, PrimitiveType type, uint32_t rows, uint32_t offset = 0, const uint8_t* __restrict null_data = nullptr) const { - LOG(FATAL) << get_name() << "update_crcs_with_value not supported"; + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method update_crcs_with_value is not supported for " + get_name()); } // use range for one hash value to avoid virtual function call in loop virtual void update_crc_with_value(size_t start, size_t end, uint32_t& hash, const uint8_t* __restrict null_data) const { - LOG(FATAL) << get_name() << " update_crc_with_value not supported"; + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method update_crc_with_value is not supported for " + get_name()); } /** Removes elements that don't match the filter. @@ -397,12 +424,14 @@ public: * happends in filter_by_selector because of mem-reuse logic or ColumnNullable, I think this is meaningless; * So using raw ptr directly here. * NOTICE: only column_nullable and predict_column, column_dictionary now support filter_by_selector + * // nullable -> predict_column + * // string (dictionary) -> column_dictionary */ virtual Status filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) { - LOG(FATAL) << get_name() - << " do not support filter_by_selector, only column_nullable, column_dictionary " - "and predict_column " - "support"; + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method filter_by_selector is not supported for {}, only " + "column_nullable, column_dictionary and predict_column support", + get_name()); __builtin_unreachable(); } @@ -463,14 +492,6 @@ public: } } - /// Returns column with @total_size elements. - /// In result column values from current column are at positions from @offsets. - /// Other values are filled by @default_value. - /// @shift means how much rows to skip from the beginning of current column. - /// Used to create full column from sparse. - virtual Ptr create_with_offsets(const Offsets64& offsets, const Field& default_field, - size_t total_rows, size_t shift) const; - /** Split column to smaller columns. Each value goes to column index, selected by corresponding element of 'selector'. * Selector must contain values from 0 to num_columns - 1. * For default implementation, see scatter_impl. @@ -514,7 +535,8 @@ public: /// Columns have equal structure. /// If true - you can use "compare_at", "insert_from", etc. methods. virtual bool structure_equals(const IColumn&) const { - LOG(FATAL) << "Method structure_equals is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method structure_equals is not supported for " + get_name()); return false; } @@ -550,10 +572,6 @@ public: virtual bool is_hll() const { return false; } - virtual bool is_variant() const { return false; } - - virtual bool is_quantile_state() const { return false; } - // true if column has null element virtual bool has_null() const { return false; } @@ -587,13 +605,15 @@ public: /// If is_fixed_and_contiguous, returns the underlying data array, otherwise throws an exception. virtual StringRef get_raw_data() const { - LOG(FATAL) << fmt::format("Column {} is not a contiguous block of memory", get_name()); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Column {} is not a contiguous block of memory", get_name()); return StringRef {}; } /// If values_have_fixed_size, returns size of value, otherwise throw an exception. virtual size_t size_of_value_if_fixed() const { - LOG(FATAL) << fmt::format("Values of column {} are not fixed size.", get_name()); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Values of column {} are not fixed size.", get_name()); return 0; } diff --git a/be/src/vec/columns/column_array.cpp b/be/src/vec/columns/column_array.cpp index 518aabeec06..450ed7d5eb6 100644 --- a/be/src/vec/columns/column_array.cpp +++ b/be/src/vec/columns/column_array.cpp @@ -46,7 +46,7 @@ ColumnArray::ColumnArray(MutableColumnPtr&& nested_column, MutableColumnPtr&& of const auto* offsets_concrete = typeid_cast<const ColumnOffsets*>(offsets.get()); if (!offsets_concrete) { - LOG(FATAL) << "offsets_column must be a ColumnUInt64"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "offsets_column must be a ColumnUInt64"); __builtin_unreachable(); } @@ -55,8 +55,10 @@ ColumnArray::ColumnArray(MutableColumnPtr&& nested_column, MutableColumnPtr&& of /// This will also prevent possible overflow in offset. if (data->size() != last_offset) { - LOG(FATAL) << "nested_column's size " << data->size() - << " is not consistent with offsets_column's " << last_offset; + throw doris::Exception( + ErrorCode::INTERNAL_ERROR, + "nested_column's size {}, is not consistent with offsets_column's {}", + data->size(), last_offset); } } @@ -68,7 +70,8 @@ ColumnArray::ColumnArray(MutableColumnPtr&& nested_column, MutableColumnPtr&& of ColumnArray::ColumnArray(MutableColumnPtr&& nested_column) : data(std::move(nested_column)) { if (!data->empty()) { - LOG(FATAL) << "Not empty data passed to ColumnArray, but no offsets passed"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Not empty data passed to ColumnArray, but no offsets passed"); __builtin_unreachable(); } @@ -126,8 +129,10 @@ Field ColumnArray::operator[](size_t n) const { size_t size = size_at(n); if (size > max_array_size_as_field) - LOG(FATAL) << "Array of size " << size << " is too large to be manipulated as single field," - << "maximum size " << max_array_size_as_field; + throw doris::Exception( + ErrorCode::INTERNAL_ERROR, + "Array of size {}, is too large to be manipulated as single field, maximum size {}", + size, max_array_size_as_field); Array res(size); @@ -141,8 +146,10 @@ void ColumnArray::get(size_t n, Field& res) const { size_t size = size_at(n); if (size > max_array_size_as_field) - LOG(FATAL) << "Array of size " << size << " is too large to be manipulated as single field," - << " maximum size " << max_array_size_as_field; + throw doris::Exception( + ErrorCode::INTERNAL_ERROR, + "Array of size {}, is too large to be manipulated as single field, maximum size {}", + size, max_array_size_as_field); res = Array(size); Array& res_arr = doris::vectorized::get<Array&>(res); @@ -181,8 +188,11 @@ bool ColumnArray::is_default_at(size_t n) const { void ColumnArray::insert_data(const char* pos, size_t length) { /** Similarly - only for arrays of fixed length values. */ - if (!data->is_fixed_and_contiguous()) - LOG(FATAL) << "Method insert_data is not supported for " << get_name(); + if (!data->is_fixed_and_contiguous()) { + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Method insert_data should have_fixed_size, {} is not suitable", + get_name()); + } size_t field_size = data->size_of_value_if_fixed(); @@ -194,7 +204,8 @@ void ColumnArray::insert_data(const char* pos, size_t length) { data->insert_data(pos, field_size); if (pos != end) - LOG(FATAL) << "Incorrect length argument for method ColumnArray::insert_data"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Incorrect length argument for method ColumnArray::insert_data"); __builtin_unreachable(); } @@ -1053,7 +1064,8 @@ ColumnPtr ColumnArray::permute(const Permutation& perm, size_t limit) const { limit = std::min(size, limit); } if (perm.size() < limit) { - LOG(FATAL) << "Size of permutation is less than required."; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of permutation is less than required."); __builtin_unreachable(); } if (limit == 0) { diff --git a/be/src/vec/columns/column_array.h b/be/src/vec/columns/column_array.h index 54fef626ac3..7839503faa4 100644 --- a/be/src/vec/columns/column_array.h +++ b/be/src/vec/columns/column_array.h @@ -213,7 +213,8 @@ public: const uint32_t* indices_end) override; void replace_column_data(const IColumn& rhs, size_t row, size_t self_row = 0) override { - LOG(FATAL) << "Method replace_column_data is not supported for " << get_name(); + throw doris::Exception(ErrorCode::NOT_IMPLEMENTED_ERROR, + "Method replace_column_data is not supported for " + get_name()); } void clear() override { diff --git a/be/src/vec/columns/column_complex.h b/be/src/vec/columns/column_complex.h index 6b570dddb72..c380ee1d0dd 100644 --- a/be/src/vec/columns/column_complex.h +++ b/be/src/vec/columns/column_complex.h @@ -50,7 +50,6 @@ public: bool is_bitmap() const override { return std::is_same_v<T, BitmapValue>; } bool is_hll() const override { return std::is_same_v<T, HyperLogLog>; } - bool is_quantile_state() const override { return std::is_same_v<T, QuantileState>; } size_t size() const override { return data.size(); } @@ -81,7 +80,7 @@ public: } else if constexpr (std::is_same_v<T, QuantileState>) { pvalue->deserialize(Slice(pos, length)); } else { - LOG(FATAL) << "Unexpected type in column complex"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Unexpected type in column complex"); __builtin_unreachable(); } } @@ -145,12 +144,12 @@ public: } [[noreturn]] bool get_bool(size_t n) const override { - LOG(FATAL) << "get field not implemented"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "get field not implemented"); __builtin_unreachable(); } [[noreturn]] Int64 get_int(size_t n) const override { - LOG(FATAL) << "get field not implemented"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "get field not implemented"); __builtin_unreachable(); } @@ -177,12 +176,14 @@ public: // it's impossible to use ComplexType as key , so we don't have to implement them [[noreturn]] StringRef serialize_value_into_arena(size_t n, Arena& arena, char const*& begin) const override { - LOG(FATAL) << "serialize_value_into_arena not implemented"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "serialize_value_into_arena not implemented"); __builtin_unreachable(); } [[noreturn]] const char* deserialize_and_insert_from_arena(const char* pos) override { - LOG(FATAL) << "deserialize_and_insert_from_arena not implemented"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "deserialize_and_insert_from_arena not implemented"); __builtin_unreachable(); } @@ -323,7 +324,8 @@ ColumnPtr ColumnComplexType<T>::permute(const IColumn::Permutation& perm, size_t limit = limit ? std::min(size, limit) : size; if (perm.size() < limit) { - LOG(FATAL) << "Size of permutation is less than required."; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of permutation is less than required."); __builtin_unreachable(); } diff --git a/be/src/vec/columns/column_const.cpp b/be/src/vec/columns/column_const.cpp index 061c8650c24..e1a5244bc65 100644 --- a/be/src/vec/columns/column_const.cpp +++ b/be/src/vec/columns/column_const.cpp @@ -43,7 +43,8 @@ ColumnConst::ColumnConst(const ColumnPtr& data_, size_t s_) : data(data_), s(s_) } if (data->size() != 1) { - LOG(FATAL) << fmt::format( + throw doris::Exception( + ErrorCode::INTERNAL_ERROR, "Incorrect size of nested column in constructor of ColumnConst: {}, must be 1.", data->size()); } @@ -86,8 +87,9 @@ ColumnPtr ColumnConst::permute(const Permutation& perm, size_t limit) const { } if (perm.size() < limit) { - LOG(FATAL) << fmt::format("Size of permutation ({}) is less than required ({})", - perm.size(), limit); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of permutation ({}) is less than required ({})", perm.size(), + limit); } return ColumnConst::create(data, limit); diff --git a/be/src/vec/columns/column_decimal.cpp b/be/src/vec/columns/column_decimal.cpp index 420984bf83c..67f25ffa867 100644 --- a/be/src/vec/columns/column_decimal.cpp +++ b/be/src/vec/columns/column_decimal.cpp @@ -226,7 +226,9 @@ template <typename T> ColumnPtr ColumnDecimal<T>::permute(const IColumn::Permutation& perm, size_t limit) const { size_t size = limit ? std::min(data.size(), limit) : data.size(); if (perm.size() < size) { - LOG(FATAL) << "Size of permutation is less than required."; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of permutation ({}) is less than required ({})", perm.size(), + limit); __builtin_unreachable(); } diff --git a/be/src/vec/columns/column_dictionary.h b/be/src/vec/columns/column_dictionary.h index b74457730c7..66a26da80bd 100644 --- a/be/src/vec/columns/column_dictionary.h +++ b/be/src/vec/columns/column_dictionary.h @@ -66,35 +66,39 @@ public: size_t size() const override { return _codes.size(); } [[noreturn]] StringRef get_data_at(size_t n) const override { - LOG(FATAL) << "get_data_at not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "get_data_at not supported in ColumnDictionary"); __builtin_unreachable(); } void insert_from(const IColumn& src, size_t n) override { - LOG(FATAL) << "insert_from not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "insert_from not supported in ColumnDictionary"); __builtin_unreachable(); } void insert_range_from(const IColumn& src, size_t start, size_t length) override { - LOG(FATAL) << "insert_range_from not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "insert_range_from not supported in ColumnDictionary"); __builtin_unreachable(); } void insert_indices_from(const IColumn& src, const uint32_t* indices_begin, const uint32_t* indices_end) override { - LOG(FATAL) << "insert_indices_from not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "insert_indices_from not supported in ColumnDictionary"); __builtin_unreachable(); } - void pop_back(size_t n) override { LOG(FATAL) << "pop_back not supported in ColumnDictionary"; } - void update_hash_with_value(size_t n, SipHash& hash) const override { - LOG(FATAL) << "update_hash_with_value not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "update_hash_with_value not supported in ColumnDictionary"); __builtin_unreachable(); } void insert_data(const char* pos, size_t /*length*/) override { - LOG(FATAL) << "insert_data not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "insert_data not supported in ColumnDictionary"); __builtin_unreachable(); } @@ -111,6 +115,11 @@ public: size_t allocated_bytes() const override { return byte_size(); } + void pop_back(size_t n) override { + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "pop_back not supported in ColumnDictionary"); + } + void reserve(size_t n) override { _codes.reserve(n); } const char* get_family_name() const override { return "ColumnDictionary"; } @@ -121,7 +130,8 @@ public: } void insert(const Field& x) override { - LOG(FATAL) << "insert not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "insert not supported in ColumnDictionary"); __builtin_unreachable(); } @@ -136,12 +146,15 @@ public: // it's impossible to use ComplexType as key , so we don't have to implement them [[noreturn]] StringRef serialize_value_into_arena(size_t n, Arena& arena, char const*& begin) const override { - LOG(FATAL) << "serialize_value_into_arena not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "serialize_value_into_arena not supported in ColumnDictionary"); __builtin_unreachable(); } [[noreturn]] const char* deserialize_and_insert_from_arena(const char* pos) override { - LOG(FATAL) << "deserialize_and_insert_from_arena not supported in ColumnDictionary"; + throw doris::Exception( + ErrorCode::INTERNAL_ERROR, + "deserialize_and_insert_from_arena not supported in ColumnDictionary"); __builtin_unreachable(); } @@ -150,45 +163,53 @@ public: size_t size_of_value_if_fixed() const override { return sizeof(T); } [[noreturn]] StringRef get_raw_data() const override { - LOG(FATAL) << "get_raw_data not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "get_raw_data not supported in ColumnDictionary"); __builtin_unreachable(); } [[noreturn]] bool structure_equals(const IColumn& rhs) const override { - LOG(FATAL) << "structure_equals not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "structure_equals not supported in ColumnDictionary"); __builtin_unreachable(); } [[noreturn]] ColumnPtr filter(const IColumn::Filter& filt, ssize_t result_size_hint) const override { - LOG(FATAL) << "filter not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "filter not supported in ColumnDictionary"); __builtin_unreachable(); } [[noreturn]] size_t filter(const IColumn::Filter&) override { - LOG(FATAL) << "filter not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "filter not supported in ColumnDictionary"); __builtin_unreachable(); } [[noreturn]] ColumnPtr permute(const IColumn::Permutation& perm, size_t limit) const override { - LOG(FATAL) << "permute not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "permute not supported in ColumnDictionary"); __builtin_unreachable(); } [[noreturn]] ColumnPtr replicate(const IColumn::Offsets& replicate_offsets) const override { - LOG(FATAL) << "replicate not supported in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "replicate not supported in ColumnDictionary"); __builtin_unreachable(); } void append_data_by_selector(MutableColumnPtr& res, const IColumn::Selector& selector) const override { - LOG(FATAL) << "append_data_by_selector is not supported in ColumnDictionary!"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "append_data_by_selector is not supported in ColumnDictionary!"); __builtin_unreachable(); } void append_data_by_selector(MutableColumnPtr& res, const IColumn::Selector& selector, size_t begin, size_t end) const override { - LOG(FATAL) << "append_data_by_selector is not supported in ColumnDictionary!"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "append_data_by_selector is not supported in ColumnDictionary!"); __builtin_unreachable(); } @@ -209,7 +230,8 @@ public: } void replace_column_data(const IColumn&, size_t row, size_t self_row = 0) override { - LOG(FATAL) << "should not call replace_column_data in ColumnDictionary"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call replace_column_data in ColumnDictionary"); __builtin_unreachable(); } diff --git a/be/src/vec/columns/column_dummy.h b/be/src/vec/columns/column_dummy.h index 3b7dd6c4dda..07bcf55bad8 100644 --- a/be/src/vec/columns/column_dummy.h +++ b/be/src/vec/columns/column_dummy.h @@ -47,16 +47,17 @@ public: int compare_at(size_t, size_t, const IColumn&, int) const override { return 0; } [[noreturn]] Field operator[](size_t) const override { - LOG(FATAL) << "Cannot get value from " << get_name(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Cannot get value from {}", get_name()); __builtin_unreachable(); } void get(size_t, Field&) const override { - LOG(FATAL) << "Cannot get value from " << get_name(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Cannot get value from {}", get_name()); } void insert(const Field&) override { - LOG(FATAL) << "Cannot insert element into " << get_name(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Cannot insert element into {}", + get_name()); } StringRef get_data_at(size_t) const override { return {}; } @@ -98,7 +99,8 @@ public: ColumnPtr permute(const Permutation& perm, size_t limit) const override { if (s != perm.size()) { - LOG(FATAL) << "Size of permutation doesn't match size of column."; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of permutation doesn't match size of column."); __builtin_unreachable(); } @@ -122,8 +124,9 @@ public: size_t num_rows = size(); if (num_rows < selector.size()) { - LOG(FATAL) << fmt::format("Size of selector: {}, is larger than size of column:{}", - selector.size(), num_rows); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of selector: {}, is larger than size of column:{}", + selector.size(), num_rows); } res->reserve(num_rows); @@ -136,8 +139,9 @@ public: size_t num_rows = size(); if (num_rows < selector.size()) { - LOG(FATAL) << fmt::format("Size of selector: {}, is larger than size of column:{}", - selector.size(), num_rows); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of selector: {}, is larger than size of column:{}", + selector.size(), num_rows); } res->reserve(num_rows); @@ -148,7 +152,8 @@ public: void addSize(size_t delta) { s += delta; } void replace_column_data(const IColumn& rhs, size_t row, size_t self_row = 0) override { - LOG(FATAL) << "should not call the method in column dummy"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call the method in column dummy"); __builtin_unreachable(); } diff --git a/be/src/vec/columns/column_impl.h b/be/src/vec/columns/column_impl.h index 1c994592e73..f0a157f4197 100644 --- a/be/src/vec/columns/column_impl.h +++ b/be/src/vec/columns/column_impl.h @@ -38,8 +38,9 @@ void IColumn::append_data_by_selector_impl(MutablePtr& res, const Selector& sele size_t num_rows = size(); if (num_rows < selector.size()) { - LOG(FATAL) << fmt::format("Size of selector: {}, is larger than size of column:{}", - selector.size(), num_rows); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Size of selector: {} is larger than size of column: {}", + selector.size(), num_rows); } res->reserve(num_rows); diff --git a/be/src/vec/columns/column_map.cpp b/be/src/vec/columns/column_map.cpp index 4b32eef3c1d..a8fa5b2b516 100644 --- a/be/src/vec/columns/column_map.cpp +++ b/be/src/vec/columns/column_map.cpp @@ -50,8 +50,8 @@ ColumnMap::ColumnMap(MutableColumnPtr&& keys, MutableColumnPtr&& values, Mutable const COffsets* offsets_concrete = typeid_cast<const COffsets*>(offsets_column.get()); if (!offsets_concrete) { - LOG(FATAL) << "offsets_column must be a ColumnUInt64"; - __builtin_unreachable(); + throw doris::Exception(doris::ErrorCode::INTERNAL_ERROR, + "offsets_column must be a ColumnUInt64."); } if (!offsets_concrete->empty() && keys_column && values_column) { @@ -59,12 +59,16 @@ ColumnMap::ColumnMap(MutableColumnPtr&& keys, MutableColumnPtr&& values, Mutable /// This will also prevent possible overflow in offset. if (keys_column->size() != last_offset) { - LOG(FATAL) << "offsets_column has data inconsistent with key_column " - << keys_column->size() << " " << last_offset; + throw doris::Exception( + doris::ErrorCode::INTERNAL_ERROR, + "offsets_column size {} has data inconsistent with key_column {}", last_offset, + keys_column->size()); } if (values_column->size() != last_offset) { - LOG(FATAL) << "offsets_column has data inconsistent with value_column " - << values_column->size() << " " << last_offset; + throw doris::Exception( + doris::ErrorCode::INTERNAL_ERROR, + "offsets_column size {} has data inconsistent with value_column {}", + last_offset, values_column->size()); } } } @@ -106,9 +110,10 @@ Field ColumnMap::operator[](size_t n) const { size_t element_size = size_at(n); if (element_size > max_array_size_as_field) { - LOG(FATAL) << "element size " << start_offset - << " is too large to be manipulated as single map field," - << "maximum size " << max_array_size_as_field; + throw doris::Exception(doris::ErrorCode::INTERNAL_ERROR, + "element size {} is too large to be manipulated as single map " + "field, maximum size {}", + element_size, max_array_size_as_field); } Array k(element_size), v(element_size); @@ -127,11 +132,13 @@ void ColumnMap::get(size_t n, Field& res) const { } StringRef ColumnMap::get_data_at(size_t n) const { - LOG(FATAL) << "Method get_data_at is not supported for " << get_name(); + throw doris::Exception(doris::ErrorCode::INTERNAL_ERROR, + "Method get_data_at is not supported for {}", get_name()); } void ColumnMap::insert_data(const char*, size_t) { - LOG(FATAL) << "Method insert_data is not supported for " << get_name(); + throw doris::Exception(doris::ErrorCode::INTERNAL_ERROR, + "Method insert_data is not supported for {}", get_name()); } void ColumnMap::insert(const Field& x) { diff --git a/be/src/vec/columns/column_map.h b/be/src/vec/columns/column_map.h index 73e367d0462..9f2862da09b 100644 --- a/be/src/vec/columns/column_map.h +++ b/be/src/vec/columns/column_map.h @@ -136,7 +136,8 @@ public: } void replace_column_data(const IColumn& rhs, size_t row, size_t self_row = 0) override { - LOG(FATAL) << "Method replace_column_data is not supported for " << get_name(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Method replace_column_data is not supported for " + get_name()); } ColumnArray::Offsets64& ALWAYS_INLINE get_offsets() { diff --git a/be/src/vec/columns/column_nullable.cpp b/be/src/vec/columns/column_nullable.cpp index c516b96b72f..9c82a2be8c7 100644 --- a/be/src/vec/columns/column_nullable.cpp +++ b/be/src/vec/columns/column_nullable.cpp @@ -43,7 +43,8 @@ ColumnNullable::ColumnNullable(MutableColumnPtr&& nested_column_, MutableColumnP } if (is_column_const(*null_map)) { - LOG(FATAL) << "ColumnNullable cannot have constant null map"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "ColumnNullable cannot have constant null map"); __builtin_unreachable(); } _need_update_has_null = true; @@ -572,7 +573,8 @@ void ColumnNullable::apply_null_map_impl(const ColumnUInt8& map) { const NullMap& arr2 = map.get_data(); if (arr1.size() != arr2.size()) { - LOG(FATAL) << "Inconsistent sizes of ColumnNullable objects"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Inconsistent sizes of ColumnNullable objects"); __builtin_unreachable(); } @@ -595,8 +597,8 @@ void ColumnNullable::apply_null_map(const ColumnNullable& other) { void ColumnNullable::check_consistency() const { if (null_map->size() != get_nested_column().size()) { - LOG(FATAL) << "Logical error: Sizes of nested column and null map of Nullable column are " - "not equal"; + throw Exception(ErrorCode::INTERNAL_ERROR, + "Sizes of nested column and null map of Nullable column are not equal"); } } diff --git a/be/src/vec/columns/column_object.cpp b/be/src/vec/columns/column_object.cpp index 65c0a5dcd89..b5b272dbeec 100644 --- a/be/src/vec/columns/column_object.cpp +++ b/be/src/vec/columns/column_object.cpp @@ -416,7 +416,12 @@ void ColumnObject::Subcolumn::insert(Field field, FieldInfo info) { } void ColumnObject::Subcolumn::insertRangeFrom(const Subcolumn& src, size_t start, size_t length) { - assert(start + length <= src.size()); + if (start + length > src.size()) { + throw doris::Exception( + ErrorCode::OUT_OF_BOUND, + "Invalid range for insertRangeFrom: start={}, length={}, src.size={}", start, + length, src.size()); + } size_t end = start + length; // num_rows += length; if (data.empty()) { @@ -438,7 +443,12 @@ void ColumnObject::Subcolumn::insertRangeFrom(const Subcolumn& src, size_t start } auto insert_from_part = [&](const auto& column, const auto& column_type, size_t from, size_t n) { - assert(from + n <= column->size()); + if (from + n > column->size()) { + throw doris::Exception( + ErrorCode::OUT_OF_BOUND, + "Invalid range for insertRangeFrom: from={}, n={}, column.size={}", from, n, + column->size()); + } if (column_type->equals(*least_common_type.get())) { data.back()->insert_range_from(*column, from, n); return; @@ -596,7 +606,10 @@ void ColumnObject::Subcolumn::insertManyDefaults(size_t length) { } void ColumnObject::Subcolumn::pop_back(size_t n) { - assert(n <= size()); + if (n > size()) { + throw doris::Exception(ErrorCode::OUT_OF_BOUND, + "Invalid number of elements to pop: {}, size: {}", n, size()); + } size_t num_removed = 0; for (auto it = data.rbegin(); it != data.rend(); ++it) { if (n == 0) { @@ -617,37 +630,38 @@ void ColumnObject::Subcolumn::pop_back(size_t n) { num_of_defaults_in_prefix -= n; } -Field ColumnObject::Subcolumn::get_last_field() const { - if (data.empty()) { - return Field(); - } - const auto& last_part = data.back(); - assert(!last_part->empty()); - return (*last_part)[last_part->size() - 1]; -} - IColumn& ColumnObject::Subcolumn::get_finalized_column() { - assert(is_finalized()); + if (!is_finalized()) { + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Subcolumn is not finalized"); + } return *data[0]; } const IColumn& ColumnObject::Subcolumn::get_finalized_column() const { - assert(is_finalized()); + if (!is_finalized()) { + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Subcolumn is not finalized"); + } return *data[0]; } const ColumnPtr& ColumnObject::Subcolumn::get_finalized_column_ptr() const { - assert(is_finalized()); + if (!is_finalized()) { + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Subcolumn is not finalized"); + } return data[0]; } ColumnPtr& ColumnObject::Subcolumn::get_finalized_column_ptr() { - assert(is_finalized()); + if (!is_finalized()) { + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Subcolumn is not finalized"); + } return data[0]; } void ColumnObject::Subcolumn::remove_nullable() { - assert(is_finalized()); + if (!is_finalized()) { + throw doris::Exception(ErrorCode::INTERNAL_ERROR, "Subcolumn is not finalized"); + } data[0] = doris::vectorized::remove_nullable(data[0]); least_common_type.remove_nullable(); } @@ -874,19 +888,6 @@ void ColumnObject::get(size_t n, Field& res) const { } } -Status ColumnObject::try_insert_indices_from(const IColumn& src, const int* indices_begin, - const int* indices_end) { - for (auto x = indices_begin; x != indices_end; ++x) { - if (*x == -1) { - ColumnObject::insert_default(); - } else { - ColumnObject::insert_from(src, *x); - } - } - finalize(); - return Status::OK(); -} - void ColumnObject::insert_range_from(const IColumn& src, size_t start, size_t length) { #ifndef NDEBUG check_consistency(); @@ -1449,17 +1450,6 @@ ColumnPtr get_base_column_of_array(const ColumnPtr& column) { return column; } -void ColumnObject::strip_outer_array() { - assert(is_finalized()); - Subcolumns new_subcolumns; - for (auto&& entry : subcolumns) { - auto base_column = get_base_column_of_array(entry->data.get_finalized_column_ptr()); - new_subcolumns.add(entry->path, Subcolumn {base_column->assume_mutable(), is_nullable}); - num_rows = base_column->size(); - } - std::swap(subcolumns, new_subcolumns); -} - ColumnPtr ColumnObject::filter(const Filter& filter, ssize_t count) const { if (!is_finalized()) { auto finalized = clone_finalized(); @@ -1551,15 +1541,6 @@ void ColumnObject::clear() { _prev_positions.clear(); } -void ColumnObject::revise_to(int target_num_rows) { - for (auto&& entry : subcolumns) { - if (entry->data.size() > target_num_rows) { - entry->data.pop_back(entry->data.size() - target_num_rows); - } - } - num_rows = target_num_rows; -} - void ColumnObject::create_root() { auto type = is_nullable ? make_nullable(std::make_shared<MostCommonType>()) : std::make_shared<MostCommonType>(); @@ -1606,17 +1587,6 @@ DataTypePtr ColumnObject::get_root_type() const { subcolumns.get_root()->data.get_least_common_type()->get_name(), path.get_path()); \ } -Status ColumnObject::extract_root(const PathInData& path) { - SANITIZE_ROOT(); - if (!path.empty()) { - MutableColumnPtr extracted; - RETURN_IF_ERROR(schema_util::extract(subcolumns.get_root()->data.get_finalized_column_ptr(), - path, extracted)); - subcolumns.get_mutable_root()->data.data[0] = extracted->get_ptr(); - } - return Status::OK(); -} - Status ColumnObject::extract_root(const PathInData& path, MutableColumnPtr& dst) const { SANITIZE_ROOT(); if (!path.empty()) { diff --git a/be/src/vec/columns/column_object.h b/be/src/vec/columns/column_object.h index e9b6eb7dfd8..a3d89394959 100644 --- a/be/src/vec/columns/column_object.h +++ b/be/src/vec/columns/column_object.h @@ -149,9 +149,6 @@ public: bool check_if_sparse_column(size_t num_rows); - /// Returns last inserted field. - Field get_last_field() const; - /// Returns single column if subcolumn in finalizes. /// Otherwise -- undefined behaviour. IColumn& get_finalized_column(); @@ -289,8 +286,6 @@ public: DataTypePtr get_root_type() const; - bool is_variant() const override { return true; } - // return null if not found const Subcolumn* get_subcolumn(const PathInData& key) const; @@ -412,9 +407,6 @@ public: void insert_default() override; - // Revise this column to specified num_rows - void revise_to(int num_rows); - ColumnPtr replicate(const Offsets& offsets) const override; void pop_back(size_t length) override; @@ -423,9 +415,6 @@ public: void get(size_t n, Field& res) const override; - Status try_insert_indices_from(const IColumn& src, const int* indices_begin, - const int* indices_end); - void update_hash_with_value(size_t n, SipHash& hash) const override; ColumnPtr filter(const Filter&, ssize_t) const override; @@ -443,15 +432,9 @@ public: void for_each_imutable_subcolumn(ImutableColumnCallback callback) const; - // Extract path from root column and replace root with new extracted column, - // root must be jsonb type - Status extract_root(const PathInData& path); - // Extract path from root column and output to dst Status extract_root(const PathInData& path, MutableColumnPtr& dst) const; - void strip_outer_array(); - bool empty() const; // Check if all columns and types are aligned diff --git a/be/src/vec/columns/column_string.cpp b/be/src/vec/columns/column_string.cpp index 919854a42d9..8e142208061 100644 --- a/be/src/vec/columns/column_string.cpp +++ b/be/src/vec/columns/column_string.cpp @@ -297,7 +297,8 @@ ColumnPtr ColumnStr<T>::permute(const IColumn::Permutation& perm, size_t limit) } if (perm.size() < limit) { - LOG(FATAL) << "Size of permutation is less than required."; + throw doris::Exception(doris::ErrorCode::INTERNAL_ERROR, + "Size of permutation is less than required."); __builtin_unreachable(); } diff --git a/be/src/vec/columns/column_string.h b/be/src/vec/columns/column_string.h index 22dcd612d3a..182b8cbefde 100644 --- a/be/src/vec/columns/column_string.h +++ b/be/src/vec/columns/column_string.h @@ -100,8 +100,9 @@ private: chars(src.chars.begin(), src.chars.end()) {} public: - void sanity_check() const; bool is_variable_length() const override { return true; } + // used in string ut testd + void sanity_check() const; const char* get_family_name() const override { return "String"; } size_t size() const override { return offsets.size(); } @@ -309,10 +310,6 @@ public: } } - // template <typename T, size_t copy_length> - // void insert_many_strings_fixed_length(const StringRef* strings, size_t num) - // __attribute__((noinline)); - template <size_t copy_length> void insert_many_strings_fixed_length(const StringRef* strings, size_t num) { size_t new_size = 0; @@ -542,7 +539,8 @@ public: } void replace_column_data(const IColumn& rhs, size_t row, size_t self_row = 0) override { - LOG(FATAL) << "Method replace_column_data is not supported for ColumnString"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Method replace_column_data is not supported for ColumnString"); __builtin_unreachable(); } @@ -550,16 +548,6 @@ public: int direction, std::vector<uint8>& cmp_res, uint8* __restrict filter) const override; - MutableColumnPtr get_shinked_column() const { - auto shrinked_column = ColumnStr<T>::create(); - for (int i = 0; i < size(); i++) { - StringRef str = get_data_at(i); - reinterpret_cast<ColumnStr<T>*>(shrinked_column.get()) - ->insert_data(str.data, strnlen(str.data, str.size)); - } - return shrinked_column; - } - ColumnPtr convert_column_if_overflow() override; }; diff --git a/be/src/vec/columns/column_struct.cpp b/be/src/vec/columns/column_struct.cpp index 86503601942..78250bc952d 100644 --- a/be/src/vec/columns/column_struct.cpp +++ b/be/src/vec/columns/column_struct.cpp @@ -53,7 +53,8 @@ ColumnStruct::ColumnStruct(MutableColumns&& mutable_columns) { columns.reserve(mutable_columns.size()); for (auto& column : mutable_columns) { if (is_column_const(*column)) { - LOG(FATAL) << "ColumnStruct cannot have ColumnConst as its element"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "ColumnStruct cannot have ColumnConst as its element"); __builtin_unreachable(); } columns.push_back(std::move(column)); @@ -63,7 +64,8 @@ ColumnStruct::ColumnStruct(MutableColumns&& mutable_columns) { ColumnStruct::Ptr ColumnStruct::create(const Columns& columns) { for (const auto& column : columns) { if (is_column_const(*column)) { - LOG(FATAL) << "ColumnStruct cannot have ColumnConst as its element"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "ColumnStruct cannot have ColumnConst as its element"); __builtin_unreachable(); } } @@ -75,7 +77,8 @@ ColumnStruct::Ptr ColumnStruct::create(const Columns& columns) { ColumnStruct::Ptr ColumnStruct::create(const TupleColumns& tuple_columns) { for (const auto& column : tuple_columns) { if (is_column_const(*column)) { - LOG(FATAL) << "ColumnStruct cannot have ColumnConst as its element"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "ColumnStruct cannot have ColumnConst as its element"); __builtin_unreachable(); } } @@ -124,8 +127,10 @@ void ColumnStruct::insert(const Field& x) { const auto& tuple = x.get<const Tuple&>(); const size_t tuple_size = columns.size(); if (tuple.size() != tuple_size) { - LOG(FATAL) << "Cannot insert value of different size into tuple. field tuple size" - << tuple.size() << ", columns size " << tuple_size; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Cannot insert value of different size into tuple. field tuple size " + "{}, columns size {}", + tuple.size(), tuple_size); } for (size_t i = 0; i < tuple_size; ++i) { @@ -138,7 +143,8 @@ void ColumnStruct::insert_from(const IColumn& src_, size_t n) { const size_t tuple_size = columns.size(); if (src.columns.size() != tuple_size) { - LOG(FATAL) << "Cannot insert value of different size into tuple."; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Cannot insert value of different size into tuple."); __builtin_unreachable(); } diff --git a/be/src/vec/columns/column_struct.h b/be/src/vec/columns/column_struct.h index 8001b721fcb..599e413f14a 100644 --- a/be/src/vec/columns/column_struct.h +++ b/be/src/vec/columns/column_struct.h @@ -94,11 +94,13 @@ public: void get(size_t n, Field& res) const override; [[noreturn]] StringRef get_data_at(size_t n) const override { - LOG(FATAL) << "Method get_data_at is not supported for " + get_name(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Method get_data_at is not supported for " + get_name()); __builtin_unreachable(); } [[noreturn]] void insert_data(const char* pos, size_t length) override { - LOG(FATAL) << "Method insert_data is not supported for " + get_name(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Method insert_data is not supported for " + get_name()); __builtin_unreachable(); } void insert(const Field& x) override; @@ -132,7 +134,8 @@ public: return append_data_by_selector_impl<ColumnStruct>(res, selector, begin, end); } void replace_column_data(const IColumn& rhs, size_t row, size_t self_row = 0) override { - LOG(FATAL) << "Method replace_column_data is not supported for " << get_name(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Method replace_column_data is not supported for " + get_name()); } void insert_range_from(const IColumn& src, size_t start, size_t length) override; diff --git a/be/src/vec/columns/column_vector.cpp b/be/src/vec/columns/column_vector.cpp index 14d52045943..4812befe3ed 100644 --- a/be/src/vec/columns/column_vector.cpp +++ b/be/src/vec/columns/column_vector.cpp @@ -480,7 +480,9 @@ ColumnPtr ColumnVector<T>::permute(const IColumn::Permutation& perm, size_t limi limit = std::min(size, limit); if (perm.size() < limit) { - LOG(FATAL) << "Size of permutation is less than required."; + throw doris::Exception(doris::ErrorCode::INTERNAL_ERROR, + "Size of permutation ({}) is less than required ({})", perm.size(), + limit); __builtin_unreachable(); } diff --git a/be/src/vec/columns/column_vector.h b/be/src/vec/columns/column_vector.h index 240229d2a1f..d9e59b0fa86 100644 --- a/be/src/vec/columns/column_vector.h +++ b/be/src/vec/columns/column_vector.h @@ -182,7 +182,8 @@ public: data[old_size + i] = begin + i; } } else { - LOG(FATAL) << "double column not support insert_range_of_integer"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "double column not support insert_range_of_integer"); __builtin_unreachable(); } } diff --git a/be/src/vec/columns/predicate_column.h b/be/src/vec/columns/predicate_column.h index c1101038870..fccebc783d4 100644 --- a/be/src/vec/columns/predicate_column.h +++ b/be/src/vec/columns/predicate_column.h @@ -114,34 +114,41 @@ public: } return res; } else { - LOG(FATAL) << "should not call get_data_at in predicate column except for string type"; + throw doris::Exception( + ErrorCode::INTERNAL_ERROR, + "should not call get_data_at in predicate column except for string type"); __builtin_unreachable(); } } void insert_from(const IColumn& src, size_t n) override { - LOG(FATAL) << "insert_from not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call insert_from in predicate column"); __builtin_unreachable(); } void insert_range_from(const IColumn& src, size_t start, size_t length) override { - LOG(FATAL) << "insert_range_from not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call insert_range_from in predicate column"); __builtin_unreachable(); } void insert_indices_from(const IColumn& src, const uint32_t* indices_begin, const uint32_t* indices_end) override { - LOG(FATAL) << "insert_indices_from not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call insert_indices_from in predicate column"); __builtin_unreachable(); } void pop_back(size_t n) override { - LOG(FATAL) << "pop_back not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call pop_back in predicate column"); __builtin_unreachable(); } void update_hash_with_value(size_t n, SipHash& hash) const override { - LOG(FATAL) << "update_hash_with_value not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call update_hash_with_value in predicate column"); __builtin_unreachable(); } @@ -300,7 +307,8 @@ public: CHECK(destination - org_dst == total_mem_size) << "Copied size not equal to expected size"; } else { - LOG(FATAL) << "Method insert_many_binary_data is not supported"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "Method insert_many_binary_data is not supported"); __builtin_unreachable(); } } @@ -328,39 +336,46 @@ public: } void insert(const Field& x) override { - LOG(FATAL) << "insert not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "insert not supported in PredicateColumnType"); __builtin_unreachable(); } [[noreturn]] Field operator[](size_t n) const override { - LOG(FATAL) << "operator[] not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "operator[] not supported in PredicateColumnType"); __builtin_unreachable(); } void get(size_t n, Field& res) const override { - LOG(FATAL) << "get field not supported in PredicateColumnType"; - __builtin_unreachable(); + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "get field not supported in PredicateColumnType"); } [[noreturn]] bool get_bool(size_t n) const override { - LOG(FATAL) << "get field not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "get field not supported in PredicateColumnType"); __builtin_unreachable(); } [[noreturn]] Int64 get_int(size_t n) const override { - LOG(FATAL) << "get field not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "get field not supported in PredicateColumnType"); __builtin_unreachable(); } // it's impossible to use ComplexType as key , so we don't have to implement them [[noreturn]] StringRef serialize_value_into_arena(size_t n, Arena& arena, char const*& begin) const override { - LOG(FATAL) << "serialize_value_into_arena not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "serialize_value_into_arena not supported in PredicateColumnType"); __builtin_unreachable(); } [[noreturn]] const char* deserialize_and_insert_from_arena(const char* pos) override { - LOG(FATAL) << "deserialize_and_insert_from_arena not supported in PredicateColumnType"; + throw doris::Exception( + ErrorCode::INTERNAL_ERROR, + "deserialize_and_insert_from_arena not supported in PredicateColumnType"); __builtin_unreachable(); } @@ -368,28 +383,33 @@ public: size_t size_of_value_if_fixed() const override { return sizeof(T); } [[noreturn]] StringRef get_raw_data() const override { - LOG(FATAL) << "get_raw_data not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "get_raw_data not supported in PredicateColumnType"); __builtin_unreachable(); } [[noreturn]] bool structure_equals(const IColumn& rhs) const override { - LOG(FATAL) << "structure_equals not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "structure_equals not supported in PredicateColumnType"); __builtin_unreachable(); } [[noreturn]] ColumnPtr filter(const IColumn::Filter& filt, ssize_t result_size_hint) const override { - LOG(FATAL) << "filter not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "filter not supported in PredicateColumnType"); __builtin_unreachable(); } [[noreturn]] size_t filter(const IColumn::Filter&) override { - LOG(FATAL) << "filter not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "filter not supported in PredicateColumnType"); __builtin_unreachable(); } [[noreturn]] ColumnPtr permute(const IColumn::Permutation& perm, size_t limit) const override { - LOG(FATAL) << "permute not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "permute not supported in PredicateColumnType"); __builtin_unreachable(); } @@ -398,18 +418,21 @@ public: const Container& get_data() const { return data; } [[noreturn]] ColumnPtr replicate(const IColumn::Offsets& replicate_offsets) const override { - LOG(FATAL) << "replicate not supported in PredicateColumnType"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "replicate not supported in PredicateColumnType"); __builtin_unreachable(); } void append_data_by_selector(MutableColumnPtr& res, const IColumn::Selector& selector) const override { - LOG(FATAL) << "append_data_by_selector is not supported in PredicateColumnType!"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "append_data_by_selector is not supported in PredicateColumnType!"); __builtin_unreachable(); } void append_data_by_selector(MutableColumnPtr& res, const IColumn::Selector& selector, size_t begin, size_t end) const override { - LOG(FATAL) << "append_data_by_selector is not supported in PredicateColumnType!"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "append_data_by_selector is not supported in PredicateColumnType!"); __builtin_unreachable(); } @@ -426,7 +449,8 @@ public: } void replace_column_data(const IColumn&, size_t row, size_t self_row = 0) override { - LOG(FATAL) << "should not call replace_column_data in predicate column"; + throw doris::Exception(ErrorCode::INTERNAL_ERROR, + "should not call replace_column_data in predicate column"); __builtin_unreachable(); } diff --git a/be/src/vec/functions/array/function_arrays_overlap.h b/be/src/vec/functions/array/function_arrays_overlap.h index 682f99acb40..74872b6cd55 100644 --- a/be/src/vec/functions/array/function_arrays_overlap.h +++ b/be/src/vec/functions/array/function_arrays_overlap.h @@ -26,6 +26,7 @@ #include <utility> #include "common/status.h" +#include "function_array_index.h" #include "vec/columns/column.h" #include "vec/columns/column_nullable.h" #include "vec/columns/column_vector.h" diff --git a/regression-test/data/nereids_p0/join/test_join_on.out b/regression-test/data/nereids_p0/join/test_join_on.out new file mode 100644 index 00000000000..b14a14dcc75 --- /dev/null +++ b/regression-test/data/nereids_p0/join/test_join_on.out @@ -0,0 +1,5 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !sql -- +1 [1, 2] \N \N +2 [2, 3] \N \N + diff --git a/regression-test/suites/nereids_p0/join/test_join_on.groovy b/regression-test/suites/nereids_p0/join/test_join_on.groovy new file mode 100644 index 00000000000..752467d3028 --- /dev/null +++ b/regression-test/suites/nereids_p0/join/test_join_on.groovy @@ -0,0 +1,53 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you under the Apache License, Version 2.0 (the +// "License"); you may not use this file except in compliance +// with the License. You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, +// software distributed under the License is distributed on an +// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY +// KIND, either express or implied. See the License for the +// specific language governing permissions and limitations +// under the License. + +suite("test_join_on", "nereids_p0") { + sql "SET enable_nereids_planner=true" + sql "SET enable_fallback_to_original_planner=false" + + sql "DROP TABLE IF EXISTS join_on" + sql """ + CREATE TABLE join_on ( + `k1` int(11) NULL, + d_array ARRAY<int> , + hll_col HLL , + `k3` bitmap, + ) ENGINE=OLAP + DUPLICATE KEY(`k1`) + COMMENT 'OLAP' + DISTRIBUTED BY HASH(`k1`) BUCKETS 4 + PROPERTIES ( + "replication_allocation" = "tag.location.default: 1" + ); + """ + sql """insert into join_on values (1, [1, 2], hll_hash(1), bitmap_from_string('1, 3, 5, 7, 9, 11, 13, 99, 19910811, 20150402')); """ + sql """insert into join_on values (2, [2, 3], hll_hash(2), bitmap_from_string('2, 4, 6, 8, 10, 12, 14, 100, 19910812, 20150403')); """ + qt_sql """ select * from join_on order by k1; """ + test { + sql """ select * from join_on as j1 inner join join_on as j2 on j1.d_array = j2.d_array; """ + exception "Method get_max_row_byte_size is not supported for Array" + } + test { + sql """ select * from join_on as j1 inner join join_on as j2 on j1.hll_col = j2.hll_col; """ + exception "data type HLL could not used in ComparisonPredicate" + } + + test { + sql """ select * from join_on as j1 inner join join_on as j2 on j1.k3 = j2.k3; """ + exception "data type BITMAP could not used in ComparisonPredicate" + } +} --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org