[FEAT MERGE] sql execution improvements
Co-authored-by: DengzhiLiu <dengzhiliu@gmail.com> Co-authored-by: Zach41 <zach_41@163.com> Co-authored-by: tushicheng <18829573815@163.com>
This commit is contained in:
@ -33,11 +33,11 @@ using AggregateExtras = ObAggregateProcessor::ExtraResult **;
|
||||
// ---------------------------------------------------------------------------------------------------------------------------------
|
||||
// | int64 | <ob_number, int64_tmp_res>| int64 | <char *, in32_t> | <char *, int32> | int32 (idx) | bits |
|
||||
// ---------------------------------------------------------------------------------------------------------------------------------
|
||||
// |
|
||||
// |
|
||||
// | ... | |
|
||||
// |-------------------------------------| |
|
||||
// extra_info array | <distinct_extra *, concat_extra*> |<---------
|
||||
// |
|
||||
// |
|
||||
// | ... | |
|
||||
// |-------------------------------------| |
|
||||
// extra_info array | <distinct_extra *, concat_extra*> |<------------------------
|
||||
// |-------------------------------------|
|
||||
// | ... |
|
||||
//
|
||||
@ -110,6 +110,11 @@ struct AggrRowMeta
|
||||
return *reinterpret_cast<NotNullBitVector *>(agg_row + nullbits_offset_);
|
||||
}
|
||||
|
||||
inline bool is_var_len(const int32_t col_id) const
|
||||
{
|
||||
return (use_var_len_ != nullptr && use_var_len_->at(col_id));
|
||||
}
|
||||
|
||||
TO_STRING_KV(K_(row_size), K_(col_cnt), K_(extra_cnt), K_(nullbits_offset), K_(extra_idx_offset));
|
||||
int32_t row_size_;
|
||||
int32_t col_cnt_;
|
||||
@ -159,16 +164,49 @@ struct RowSelector
|
||||
const int32_t size_;
|
||||
};
|
||||
|
||||
struct RemovalInfo
|
||||
{
|
||||
RemovalInfo() :
|
||||
enable_removal_opt_(false), is_max_min_idx_changed_(false), max_min_index_(-1),
|
||||
is_inverse_agg_(false), null_cnt_(0)
|
||||
{}
|
||||
bool enable_removal_opt_;
|
||||
bool is_max_min_idx_changed_;
|
||||
int64_t max_min_index_; // used to record index of min/max value
|
||||
bool is_inverse_agg_; // used to determine which interface is called: add_batch_rows/remove_batch_rows
|
||||
int32_t null_cnt_;
|
||||
TO_STRING_KV(K_(enable_removal_opt), K_(is_max_min_idx_changed), K_(max_min_index),
|
||||
K_(is_inverse_agg), K_(null_cnt));
|
||||
|
||||
void reset()
|
||||
{
|
||||
enable_removal_opt_ = false;
|
||||
max_min_index_ = -1;
|
||||
is_inverse_agg_ = false;
|
||||
null_cnt_ = 0;
|
||||
is_max_min_idx_changed_ = false;
|
||||
}
|
||||
|
||||
void reset_for_new_frame()
|
||||
{
|
||||
max_min_index_ = -1;
|
||||
is_inverse_agg_ = false;
|
||||
null_cnt_ = 0;
|
||||
is_max_min_idx_changed_ = false;
|
||||
}
|
||||
};
|
||||
|
||||
struct RuntimeContext
|
||||
{
|
||||
RuntimeContext(sql::ObEvalCtx &eval_ctx, uint64_t tenant_id, ObIArray<ObAggrInfo> &aggr_infos,
|
||||
const lib::ObLabel &label, ObIAllocator &struct_allocator) :
|
||||
const lib::ObLabel &label) :
|
||||
eval_ctx_(eval_ctx),
|
||||
aggr_infos_(aggr_infos),
|
||||
allocator_(label, OB_MALLOC_NORMAL_BLOCK_SIZE, tenant_id, ObCtxIds::WORK_AREA),
|
||||
op_monitor_info_(nullptr), io_event_observer_(nullptr), agg_row_meta_(),
|
||||
agg_rows_(ModulePageAllocator(label, tenant_id, ObCtxIds::WORK_AREA)),
|
||||
agg_extras_(ModulePageAllocator(label, tenant_id, ObCtxIds::WORK_AREA))
|
||||
agg_extras_(ModulePageAllocator(label, tenant_id, ObCtxIds::WORK_AREA)),
|
||||
removal_info_(), win_func_agg_(false)
|
||||
{}
|
||||
|
||||
inline const AggrRowMeta &row_meta() const
|
||||
@ -276,6 +314,7 @@ struct RuntimeContext
|
||||
} // end for
|
||||
agg_extras_.reuse();
|
||||
allocator_.reset_remain_one_page();
|
||||
removal_info_.reset();
|
||||
}
|
||||
void destroy()
|
||||
{
|
||||
@ -294,6 +333,22 @@ struct RuntimeContext
|
||||
op_monitor_info_ = nullptr;
|
||||
io_event_observer_ = nullptr;
|
||||
agg_row_meta_.reset();
|
||||
removal_info_.reset();
|
||||
win_func_agg_ = false;
|
||||
}
|
||||
|
||||
inline void enable_removal_opt()
|
||||
{
|
||||
removal_info_.enable_removal_opt_ = true;
|
||||
}
|
||||
inline void set_inverse_agg(bool is_inverse)
|
||||
{
|
||||
removal_info_.is_inverse_agg_ = is_inverse;
|
||||
}
|
||||
|
||||
inline void disable_inverse_agg()
|
||||
{
|
||||
removal_info_.reset();
|
||||
}
|
||||
|
||||
int init_row_meta(ObIArray<ObAggrInfo> &aggr_infos, ObIAllocator &alloc);
|
||||
@ -308,6 +363,8 @@ struct RuntimeContext
|
||||
agg_rows_;
|
||||
ObSegmentArray<AggregateExtras, OB_MALLOC_MIDDLE_BLOCK_SIZE, common::ModulePageAllocator>
|
||||
agg_extras_;
|
||||
RemovalInfo removal_info_;
|
||||
bool win_func_agg_;
|
||||
};
|
||||
|
||||
/*
|
||||
|
||||
89
src/share/aggregate/approx_count_distinct.cpp
Normal file
89
src/share/aggregate/approx_count_distinct.cpp
Normal file
@ -0,0 +1,89 @@
|
||||
/**
|
||||
* Copyright (c) 2021 OceanBase
|
||||
* OceanBase CE is licensed under Mulan PubL v2.
|
||||
* You can use this software according to the terms and conditions of the Mulan PubL v2.
|
||||
* You may obtain a copy of Mulan PubL v2 at:
|
||||
* http://license.coscl.org.cn/MulanPubL-2.0
|
||||
* THIS SOFTWARE IS PROVIDED ON AN "AS IS" BASIS, WITHOUT WARRANTIES OF ANY KIND,
|
||||
* EITHER EXPRESS OR IMPLIED, INCLUDING BUT NOT LIMITED TO NON-INFRINGEMENT,
|
||||
* MERCHANTABILITY OR FIT FOR A PARTICULAR PURPOSE.
|
||||
* See the Mulan PubL v2 for more details.
|
||||
*/
|
||||
#define USING_LOG_PREFIX SQL_ENG
|
||||
|
||||
#include "approx_count_distinct.h"
|
||||
|
||||
namespace oceanbase
|
||||
{
|
||||
namespace share
|
||||
{
|
||||
namespace aggregate
|
||||
{
|
||||
namespace helper
|
||||
{
|
||||
int init_approx_count_distinct_aggregate(RuntimeContext &agg_ctx, const int64_t agg_col_id,
|
||||
ObIAllocator &allocator, IAggregate *&agg)
|
||||
{
|
||||
#define INIT_CASE(func_type, vec_tc) \
|
||||
case (vec_tc): { \
|
||||
if (func_type == T_FUN_APPROX_COUNT_DISTINCT) { \
|
||||
if (lib::is_oracle_mode()) { \
|
||||
ret = init_agg_func<ApproxCountDistinct<func_type, vec_tc, VEC_TC_NUMBER>>( \
|
||||
agg_ctx, agg_col_id, has_distinct, allocator, agg); \
|
||||
} else { \
|
||||
ret = init_agg_func<ApproxCountDistinct<func_type, vec_tc, VEC_TC_INTEGER>>( \
|
||||
agg_ctx, agg_col_id, has_distinct, allocator, agg); \
|
||||
} \
|
||||
} else { \
|
||||
ret = init_agg_func<ApproxCountDistinct<func_type, vec_tc, VEC_TC_STRING>>( \
|
||||
agg_ctx, agg_col_id, has_distinct, allocator, agg); \
|
||||
} \
|
||||
} break
|
||||
|
||||
#define INIT_APP_CNT_DISTINCT_CASE(vec_tc) INIT_CASE(T_FUN_APPROX_COUNT_DISTINCT, vec_tc)
|
||||
#define INIT_APP_CNT_DIST_SYN_CASE(vec_tc) INIT_CASE(T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS, vec_tc)
|
||||
#define INIT_APP_CNT_DIST_MERGE_CASE(vec_tc) INIT_CASE(T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE, vec_tc)
|
||||
|
||||
int ret = OB_SUCCESS;
|
||||
ObAggrInfo &aggr_info = agg_ctx.locate_aggr_info(agg_col_id);
|
||||
bool has_distinct = aggr_info.has_distinct_;
|
||||
if (T_FUN_APPROX_COUNT_DISTINCT == aggr_info.get_expr_type()) {
|
||||
ObDatumMeta ¶m_meta = aggr_info.param_exprs_.at(0)->datum_meta_;
|
||||
VecValueTypeClass in_tc =
|
||||
get_vec_value_tc(param_meta.type_, param_meta.scale_, param_meta.precision_);
|
||||
switch (in_tc) {
|
||||
LST_DO_CODE(INIT_APP_CNT_DISTINCT_CASE, AGG_VEC_TC_LIST);
|
||||
default: {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid param format", K(ret), K(in_tc));
|
||||
}
|
||||
}
|
||||
} else if (T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE == aggr_info.get_expr_type()) {
|
||||
ObDatumMeta ¶m_meta = aggr_info.param_exprs_.at(0)->datum_meta_;
|
||||
VecValueTypeClass in_tc =
|
||||
get_vec_value_tc(param_meta.type_, param_meta.scale_, param_meta.precision_);
|
||||
switch (in_tc) {
|
||||
LST_DO_CODE(INIT_APP_CNT_DIST_MERGE_CASE, AGG_VEC_TC_LIST);
|
||||
default: {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid param format", K(ret), K(in_tc));
|
||||
}
|
||||
}
|
||||
} else if (T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS == aggr_info.get_expr_type()) {
|
||||
VecValueTypeClass vec_tc =
|
||||
get_vec_value_tc(aggr_info.get_first_child_type(), aggr_info.get_first_child_datum_scale(),
|
||||
aggr_info.get_first_child_datum_precision());
|
||||
switch (vec_tc) {
|
||||
LST_DO_CODE(INIT_APP_CNT_DIST_SYN_CASE, AGG_VEC_TC_LIST);
|
||||
default: {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid param format", K(ret), K(vec_tc));
|
||||
}
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
}
|
||||
} // end aggregate
|
||||
} // end share
|
||||
} // end oceanbase
|
||||
436
src/share/aggregate/approx_count_distinct.h
Normal file
436
src/share/aggregate/approx_count_distinct.h
Normal file
@ -0,0 +1,436 @@
|
||||
/**
|
||||
* Copyright (c) 2021 OceanBase
|
||||
* OceanBase CE is licensed under Mulan PubL v2.
|
||||
* You can use this software according to the terms and conditions of the Mulan PubL v2.
|
||||
* You may obtain a copy of Mulan PubL v2 at:
|
||||
* http://license.coscl.org.cn/MulanPubL-2.0
|
||||
* THIS SOFTWARE IS PROVIDED ON AN "AS IS" BASIS, WITHOUT WARRANTIES OF ANY KIND,
|
||||
* EITHER EXPRESS OR IMPLIED, INCLUDING BUT NOT LIMITED TO NON-INFRINGEMENT,
|
||||
* MERCHANTABILITY OR FIT FOR A PARTICULAR PURPOSE.
|
||||
* See the Mulan PubL v2 for more details.
|
||||
*/
|
||||
|
||||
#ifndef OCEANBASE_SHARE_AGGREGATE_APPROX_COUNT_DISTINCT_H
|
||||
#define OCEANBASE_SHARE_AGGREGATE_APPROX_COUNT_DISTINCT_H
|
||||
|
||||
#include "share/aggregate/iaggregate.h"
|
||||
#include "src/sql/engine/expr/ob_expr_estimate_ndv.h"
|
||||
|
||||
namespace oceanbase
|
||||
{
|
||||
namespace share
|
||||
{
|
||||
namespace aggregate
|
||||
{
|
||||
template <ObExprOperatorType agg_func, VecValueTypeClass in_tc, VecValueTypeClass out_tc>
|
||||
class ApproxCountDistinct final
|
||||
: public BatchAggregateWrapper<ApproxCountDistinct<agg_func, in_tc, out_tc>>
|
||||
{
|
||||
static const constexpr int8_t LLC_BUCKET_BITS = 10;
|
||||
static const constexpr int64_t LLC_NUM_BUCKETS = (1 << LLC_BUCKET_BITS);
|
||||
public:
|
||||
static const constexpr VecValueTypeClass IN_TC = in_tc;
|
||||
static const constexpr VecValueTypeClass OUT_TC = out_tc;
|
||||
public:
|
||||
ApproxCountDistinct() {}
|
||||
template <typename ColumnFmt>
|
||||
inline int add_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
if (agg_func == T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE) {
|
||||
char *llc_bitmap_buf = EXTRACT_MEM_ADDR(agg_cell);
|
||||
const char *payload = nullptr;
|
||||
int32_t len = 0;
|
||||
columns.get_payload(row_num, payload, len);
|
||||
if (OB_UNLIKELY(len != LLC_NUM_BUCKETS)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "unexpected length of input", K(ret), K(len));
|
||||
} else if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
// not calculated before, copy from payload
|
||||
llc_bitmap_buf = (char *)agg_ctx.allocator_.alloc(LLC_NUM_BUCKETS);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(WARN, "allocate memory failed", K(ret));
|
||||
} else {
|
||||
MEMSET(llc_bitmap_buf, 0, LLC_NUM_BUCKETS);
|
||||
MEMCPY(llc_bitmap_buf, payload, LLC_NUM_BUCKETS);
|
||||
STORE_MEM_ADDR(llc_bitmap_buf, agg_cell);
|
||||
*reinterpret_cast<int32_t *>(agg_cell + sizeof(char *)) = LLC_NUM_BUCKETS;
|
||||
}
|
||||
} else {
|
||||
for (int i = 0; i < LLC_NUM_BUCKETS; i++) {
|
||||
llc_bitmap_buf[i] =
|
||||
std::max(static_cast<uint8_t>(llc_bitmap_buf[i]), static_cast<uint8_t>(payload[i]));
|
||||
}
|
||||
}
|
||||
} else {
|
||||
const char *payload = nullptr;
|
||||
int32_t len = 0;
|
||||
ObExprHashFuncType hash_func = reinterpret_cast<ObExprHashFuncType>(calc_info);
|
||||
ObDatum tmp_datum;
|
||||
uint64_t hash_val = 0;
|
||||
if (OB_ISNULL(tmp_res)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid null tmp result", K(ret));
|
||||
} else {
|
||||
columns.get_payload(row_num, payload, len);
|
||||
tmp_datum.ptr_ = payload;
|
||||
tmp_datum.pack_ = len;
|
||||
if (OB_FAIL(hash_func(tmp_datum, hash_val, hash_val))) {
|
||||
SQL_LOG(WARN, "hash func failed", K(ret));
|
||||
} else if (OB_FAIL(
|
||||
llc_add_value(hash_val, reinterpret_cast<char *>(tmp_res), LLC_NUM_BUCKETS))) {
|
||||
SQL_LOG(WARN, "llc add value failed", K(ret));
|
||||
}
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
inline int add_nullable_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
if (OB_UNLIKELY(columns.is_null(row_num))) {
|
||||
SQL_LOG(DEBUG, "add null row", K(ret), K(row_num));
|
||||
} else if (OB_FAIL(
|
||||
add_row(agg_ctx, columns, row_num, agg_col_id, agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
} else {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int collect_group_result(RuntimeContext &agg_ctx, const sql::ObExpr &agg_expr,
|
||||
const int32_t agg_col_id, const char *agg_cell,
|
||||
const int32_t agg_cell_len)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
const NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
int64_t output_idx = agg_ctx.eval_ctx_.get_batch_idx();
|
||||
const char *llc_bitmap_buf = nullptr;
|
||||
ColumnFmt *res_vec = static_cast<ColumnFmt *>(agg_expr.get_vector(agg_ctx.eval_ctx_));
|
||||
if (agg_func != T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE) {
|
||||
llc_bitmap_buf = (const char *)get_tmp_res(agg_ctx, agg_col_id, const_cast<char *>(agg_cell));
|
||||
} else {
|
||||
llc_bitmap_buf = reinterpret_cast<const char *>(*reinterpret_cast<const int64_t *>(agg_cell));
|
||||
}
|
||||
if (agg_func == T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE) {
|
||||
if (OB_LIKELY(not_nulls.at(agg_col_id))) {
|
||||
char *res_buf = agg_expr.get_str_res_mem(agg_ctx.eval_ctx_, LLC_NUM_BUCKETS);
|
||||
if (OB_ISNULL(res_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(WARN, "allocate memory failed", K(ret));
|
||||
} else {
|
||||
MEMCPY(res_buf, llc_bitmap_buf, LLC_NUM_BUCKETS);
|
||||
res_vec->set_payload_shallow(output_idx, res_buf, LLC_NUM_BUCKETS);
|
||||
}
|
||||
} else {
|
||||
res_vec->set_null(output_idx);
|
||||
}
|
||||
} else if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN , "invalid null llc bitmap", K(ret));
|
||||
} else if (agg_func == T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS) {
|
||||
char *res_buf = agg_expr.get_str_res_mem(agg_ctx.eval_ctx_, LLC_NUM_BUCKETS);
|
||||
if (OB_ISNULL(res_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(WARN, "allocate memory failed", K(ret));
|
||||
} else {
|
||||
MEMCPY(res_buf, llc_bitmap_buf, LLC_NUM_BUCKETS);
|
||||
res_vec->set_payload_shallow(output_idx, res_buf, LLC_NUM_BUCKETS);
|
||||
}
|
||||
} else if (OB_LIKELY(not_nulls.at(agg_col_id))) {
|
||||
ObString llc_str(LLC_NUM_BUCKETS, llc_bitmap_buf);
|
||||
int64_t tmp_result = OB_INVALID_COUNT;
|
||||
ObExprEstimateNdv::llc_estimate_ndv(tmp_result, llc_str);
|
||||
if (tmp_result >= 0) {
|
||||
if (lib::is_mysql_mode()) {
|
||||
res_vec->set_int(output_idx, tmp_result);
|
||||
} else {
|
||||
ObNumStackOnceAlloc tmp_alloc;
|
||||
number::ObNumber res_nmb;
|
||||
if (OB_FAIL(res_nmb.from(tmp_result, tmp_alloc))) {
|
||||
SQL_LOG(WARN, "convert int to number failed", K(ret));
|
||||
} else {
|
||||
res_vec->set_number(output_idx, res_nmb);
|
||||
}
|
||||
}
|
||||
}
|
||||
} else {
|
||||
if (lib::is_oracle_mode()) {
|
||||
// set zero number
|
||||
number::ObNumber zero_nmb;
|
||||
zero_nmb.set_zero();
|
||||
res_vec->set_payload(output_idx, &zero_nmb, sizeof(ObNumberDesc));
|
||||
} else {
|
||||
res_vec->set_int(output_idx, 0);
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template<typename ColumnFmt>
|
||||
int add_param_batch(RuntimeContext &agg_ctx, const ObBitVector &skip, ObBitVector &pvt_skip,
|
||||
const EvalBound &bound, const RowSelector &row_sel, const int32_t agg_col_id,
|
||||
const int32_t param_id, ColumnFmt ¶m_vec, char *aggr_cell)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
ob_assert(agg_func == T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS);
|
||||
char *llc_bitmap_buf = EXTRACT_MEM_ADDR(aggr_cell);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
llc_bitmap_buf = (char *)agg_ctx.allocator_.alloc(LLC_NUM_BUCKETS);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(WARN, "allocate memory failed", K(ret));
|
||||
} else {
|
||||
MEMSET(llc_bitmap_buf, 0, LLC_NUM_BUCKETS);
|
||||
STORE_MEM_ADDR(llc_bitmap_buf, aggr_cell);
|
||||
}
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
} else if (row_sel.is_empty()) {
|
||||
for (int i = bound.start(); i < bound.end(); i++) {
|
||||
if (pvt_skip.at(i)) { continue; }
|
||||
if (param_vec.is_null(i)) {
|
||||
pvt_skip.set(i);
|
||||
}
|
||||
}
|
||||
} else {
|
||||
for (int i = 0; i < row_sel.size(); i++) {
|
||||
if (param_vec.is_null(row_sel.index(i))) { pvt_skip.set(row_sel.index(i)); }
|
||||
}
|
||||
}
|
||||
if (param_id == agg_ctx.aggr_infos_.at(agg_col_id).param_exprs_.count() - 1) {
|
||||
// last param, calculate hash values if possible
|
||||
ObIArray<ObExpr *> ¶m_exprs = agg_ctx.aggr_infos_.at(agg_col_id).param_exprs_;
|
||||
const char *payload = nullptr;
|
||||
int32_t len = 0;
|
||||
ObDatum tmp_datum;
|
||||
if (row_sel.is_empty()) {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
uint64_t hash_val = 0;
|
||||
if (pvt_skip.at(i)) {
|
||||
continue;
|
||||
}
|
||||
for (int j = 0; OB_SUCC(ret) && j < param_exprs.count(); j++) {
|
||||
ObExpr *expr = param_exprs.at(j);
|
||||
ObExprHashFuncType hash_func = expr->basic_funcs_->murmur_hash_;
|
||||
expr->get_vector(agg_ctx.eval_ctx_)->get_payload(i, payload, len);
|
||||
tmp_datum.ptr_ = payload;
|
||||
tmp_datum.pack_ = len;
|
||||
if (OB_FAIL(hash_func(tmp_datum, hash_val, hash_val))) {
|
||||
SQL_LOG(WARN, "hash calculation failed", K(ret));
|
||||
}
|
||||
}
|
||||
if (OB_SUCC(ret)) {
|
||||
if (OB_FAIL(llc_add_value(hash_val, llc_bitmap_buf, LLC_NUM_BUCKETS))) {
|
||||
SQL_LOG(WARN, "add llc value failed");
|
||||
}
|
||||
}
|
||||
}
|
||||
} else {
|
||||
for (int i = 0; OB_SUCC(ret) && i < row_sel.size(); i++) {
|
||||
if (pvt_skip.at(row_sel.index(i))) {
|
||||
continue;
|
||||
}
|
||||
int64_t row_num = row_sel.index(i);
|
||||
uint64_t hash_val = 0;
|
||||
for (int j = 0; OB_SUCC(ret) && j < param_exprs.count(); j++) {
|
||||
ObExpr *expr = param_exprs.at(j);
|
||||
ObExprHashFuncType hash_func = expr->basic_funcs_->murmur_hash_;
|
||||
expr->get_vector(agg_ctx.eval_ctx_)->get_payload(row_num, payload, len);
|
||||
tmp_datum.ptr_ = payload;
|
||||
tmp_datum.pack_ = len;
|
||||
if (OB_FAIL(hash_func(tmp_datum, hash_val, hash_val))) {
|
||||
SQL_LOG(WARN, "hash calculation failed", K(ret));
|
||||
}
|
||||
}
|
||||
if (OB_SUCC(ret)) {
|
||||
if (OB_FAIL(llc_add_value(hash_val, llc_bitmap_buf, LLC_NUM_BUCKETS))) {
|
||||
SQL_LOG(WARN, "add llc value failed");
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
int add_one_row(RuntimeContext &agg_ctx, int64_t batch_idx, int64_t batch_size,
|
||||
const bool is_null, const char *data, const int32_t data_len, int32_t agg_col_idx,
|
||||
char *agg_cell) override
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
void *llc_bitmap_buf = get_tmp_res(agg_ctx, agg_col_idx, agg_cell);
|
||||
|
||||
if (agg_func != T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE && OB_ISNULL(llc_bitmap_buf)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid null llc bitmap buf", K(ret), K(*this));
|
||||
} else if (OB_LIKELY(agg_ctx.aggr_infos_.at(agg_col_idx).param_exprs_.count() == 1)) {
|
||||
if (agg_func == T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE) {
|
||||
if (OB_UNLIKELY(data_len != LLC_NUM_BUCKETS)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "unexpected input length", K(ret));
|
||||
} else if (!is_null) {
|
||||
char *llc_bitmap_buf = EXTRACT_MEM_ADDR(agg_cell);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
// not calculated before
|
||||
llc_bitmap_buf = (char *)agg_ctx.allocator_.alloc(LLC_NUM_BUCKETS);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(WARN, "allocate memory failed", K(ret));
|
||||
} else {
|
||||
MEMSET(llc_bitmap_buf, 0, LLC_NUM_BUCKETS);
|
||||
MEMCPY(llc_bitmap_buf, data, LLC_NUM_BUCKETS);
|
||||
STORE_MEM_ADDR(llc_bitmap_buf, agg_cell);
|
||||
*reinterpret_cast<int32_t *>(agg_cell + sizeof(char *)) = LLC_NUM_BUCKETS;
|
||||
}
|
||||
} else {
|
||||
for (int i = 0; i < LLC_NUM_BUCKETS; i++) {
|
||||
llc_bitmap_buf[i] =
|
||||
std::max(static_cast<uint8_t>(llc_bitmap_buf[i]), static_cast<uint8_t>(data[i]));
|
||||
}
|
||||
}
|
||||
agg_ctx.locate_notnulls_bitmap(agg_col_idx, agg_cell).set(agg_col_idx);
|
||||
}
|
||||
} else {
|
||||
const ObExpr *param_expr = agg_ctx.aggr_infos_.at(agg_col_idx).param_exprs_.at(0);
|
||||
ObExprHashFuncType hash_func = param_expr->basic_funcs_->murmur_hash_;
|
||||
if (OB_LIKELY(!is_null)) {
|
||||
ObDatum tmp_datum;
|
||||
tmp_datum.ptr_ = data;
|
||||
tmp_datum.pack_ = data_len;
|
||||
uint64_t hash_val = 0;
|
||||
if (OB_FAIL(hash_func(tmp_datum, hash_val, hash_val))) {
|
||||
SQL_LOG(WARN, "hash calculation failed", K(ret));
|
||||
} else if (OB_FAIL(llc_add_value(hash_val, reinterpret_cast<char *>(llc_bitmap_buf),
|
||||
LLC_NUM_BUCKETS))) {
|
||||
SQL_LOG(WARN, "llc add value failed", K(ret));
|
||||
} else {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_idx, agg_cell);
|
||||
not_nulls.set(agg_col_idx);
|
||||
}
|
||||
}
|
||||
}
|
||||
} else if (agg_ctx.aggr_infos_.at(agg_col_idx).param_exprs_.count() > 1) {
|
||||
ob_assert(T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS == agg_func);
|
||||
ObIArray<ObExpr *> ¶m_exprs = agg_ctx.aggr_infos_.at(agg_col_idx).param_exprs_;
|
||||
bool has_null = false;
|
||||
for (int i = 0; !has_null && i < param_exprs.count(); i++) {
|
||||
has_null = param_exprs.at(i)->get_vector(agg_ctx.eval_ctx_)->is_null(batch_idx);
|
||||
}
|
||||
if (!has_null) {
|
||||
uint64_t hash_val = 0;
|
||||
ObDatum tmp_datum;
|
||||
const char *payload = nullptr;
|
||||
int32_t len = 0;
|
||||
for (int i = 0; OB_SUCC(ret) && i < param_exprs.count(); i++) {
|
||||
ObExpr *expr = param_exprs.at(i);
|
||||
ObExprHashFuncType hash_func = expr->basic_funcs_->murmur_hash_;
|
||||
expr->get_vector(agg_ctx.eval_ctx_)->get_payload(batch_idx, payload, len);
|
||||
tmp_datum.ptr_ = payload;
|
||||
tmp_datum.pack_ = len;
|
||||
if (OB_FAIL(hash_func(tmp_datum, hash_val, hash_val))) {
|
||||
SQL_LOG(WARN, "hash calculation failed", K(ret));
|
||||
}
|
||||
}
|
||||
if (OB_SUCC(ret)) {
|
||||
if (OB_FAIL(llc_add_value(hash_val, (char *)llc_bitmap_buf, LLC_NUM_BUCKETS))) {
|
||||
SQL_LOG(WARN, "llc add value failed", K(ret));
|
||||
} else {
|
||||
agg_ctx.locate_notnulls_bitmap(agg_col_idx, agg_cell).set(agg_col_idx);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
inline void *get_tmp_res(RuntimeContext &agg_ctx, int32_t agg_col_id, char *agg_cell) override
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
void *ret_ptr = nullptr;
|
||||
if (agg_func == T_FUN_APPROX_COUNT_DISTINCT) {
|
||||
int32_t cell_len = agg_ctx.get_cell_len(agg_col_id, agg_cell);
|
||||
char *llc_bitmap_buf = EXTRACT_MEM_ADDR((agg_cell + cell_len));
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
// allocate tmp res memory
|
||||
llc_bitmap_buf = (char *)agg_ctx.allocator_.alloc(LLC_NUM_BUCKETS);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(ERROR, "allocate memory failed", K(ret));
|
||||
} else {
|
||||
MEMSET(llc_bitmap_buf, 0, LLC_NUM_BUCKETS);
|
||||
STORE_MEM_ADDR(llc_bitmap_buf, (agg_cell + cell_len));
|
||||
ret_ptr = llc_bitmap_buf;
|
||||
}
|
||||
} else {
|
||||
ret_ptr = llc_bitmap_buf;
|
||||
}
|
||||
} else if (agg_func == T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS) {
|
||||
char *llc_bitmap_buf = EXTRACT_MEM_ADDR(agg_cell);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
llc_bitmap_buf = (char *)agg_ctx.allocator_.alloc(LLC_NUM_BUCKETS);
|
||||
if (OB_ISNULL(llc_bitmap_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(ERROR, "allocate memory failed", K(ret));
|
||||
} else {
|
||||
MEMSET(llc_bitmap_buf, 0, LLC_NUM_BUCKETS);
|
||||
STORE_MEM_ADDR(llc_bitmap_buf, agg_cell);
|
||||
// set result length of approx_count_distinct_synopsis
|
||||
*reinterpret_cast<int32_t *>(agg_cell + sizeof(char *)) = LLC_NUM_BUCKETS;
|
||||
}
|
||||
}
|
||||
if (OB_SUCC(ret)) {
|
||||
ret_ptr = llc_bitmap_buf;
|
||||
}
|
||||
} else {
|
||||
}
|
||||
return ret_ptr;
|
||||
}
|
||||
|
||||
inline int64_t get_batch_calc_info(RuntimeContext &agg_ctx, int32_t agg_col_id,
|
||||
char *agg_cell) override
|
||||
{
|
||||
OB_ASSERT(agg_ctx.aggr_infos_.at(agg_col_id).param_exprs_.count() == 1);
|
||||
const ObExpr *expr = agg_ctx.aggr_infos_.at(agg_col_id).param_exprs_.at(0);
|
||||
OB_ASSERT(expr != NULL);
|
||||
OB_ASSERT(expr->basic_funcs_ != NULL);
|
||||
ObExprHashFuncType hash_func = expr->basic_funcs_->murmur_hash_;
|
||||
return reinterpret_cast<int64_t>(hash_func);
|
||||
}
|
||||
TO_STRING_KV("aggregate", "approx_count_distinct", K(in_tc), K(out_tc), K(agg_func));
|
||||
private:
|
||||
int llc_add_value(const uint64_t value, char *llc_bitmap_buf, int64_t size)
|
||||
{
|
||||
// copy from `ObAggregateProcessor::llv_add_value`
|
||||
int ret = OB_SUCCESS;
|
||||
uint64_t bucket_index = value >> (64 - LLC_BUCKET_BITS);
|
||||
uint64_t pmax = 0;
|
||||
if (0 == value << LLC_BUCKET_BITS) {
|
||||
// do nothing
|
||||
} else {
|
||||
pmax = ObExprEstimateNdv::llc_leading_zeros(value << LLC_BUCKET_BITS, 64 - LLC_BUCKET_BITS) + 1;
|
||||
}
|
||||
ObString::obstr_size_t llc_num_buckets = size;
|
||||
OB_ASSERT(size == LLC_NUM_BUCKETS);
|
||||
OB_ASSERT(ObExprEstimateNdv::llc_is_num_buckets_valid(llc_num_buckets));
|
||||
OB_ASSERT(llc_num_buckets > bucket_index);
|
||||
if (pmax > static_cast<uint8_t>(llc_bitmap_buf[bucket_index])) {
|
||||
// 理论上pmax不会超过65.
|
||||
llc_bitmap_buf[bucket_index] = static_cast<uint8_t>(pmax);
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
};
|
||||
|
||||
} // end aggregate
|
||||
} // end share
|
||||
} // end oceanbase
|
||||
#endif // OCEANBASE_SHARE_AGGREGATE_APPROX_COUNT_DISTINCT_H
|
||||
@ -73,7 +73,22 @@ int quick_add_batch_rows_for_count(IAggregate *agg, RuntimeContext &agg_ctx,
|
||||
LOG_WARN("invalid null aggregate", K(ret));
|
||||
} else if (!is_single_row_agg) {
|
||||
auto &count_agg = *static_cast<CountAggregate<VEC_TC_INTEGER, VEC_TC_INTEGER> *>(agg);
|
||||
if (OB_LIKELY(row_sel.is_empty() && bound.get_all_rows_active())) {
|
||||
if (OB_UNLIKELY(agg_ctx.removal_info_.enable_removal_opt_)) {
|
||||
int64_t &count = *reinterpret_cast<int64_t *>(agg_cell);
|
||||
if (!agg_ctx.removal_info_.is_inverse_agg_) {
|
||||
if (row_sel.is_empty()) {
|
||||
count += bound.range_size() - skip.accumulate_bit_cnt(bound);
|
||||
} else {
|
||||
count += row_sel.size();
|
||||
}
|
||||
} else {
|
||||
if (row_sel.is_empty()) {
|
||||
count -= bound.range_size() - skip.accumulate_bit_cnt(bound);
|
||||
} else {
|
||||
count -= row_sel.size();
|
||||
}
|
||||
}
|
||||
} else if (OB_LIKELY(row_sel.is_empty() && bound.get_all_rows_active())) {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
ret =
|
||||
count_agg.add_row(agg_ctx, mock_cols, i, agg_col_id, agg_cell, nullptr, fake_calc_info);
|
||||
|
||||
@ -33,7 +33,6 @@ public:
|
||||
CountAggregate() {}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
// TODO: impl add_batch_rows, using null mpas to optimize counting logic
|
||||
int add_param_batch(RuntimeContext &agg_ctx, const ObBitVector &skip, ObBitVector &pvt_skip,
|
||||
const EvalBound &bound, const RowSelector &row_sel, const int32_t agg_col_id,
|
||||
const int32_t param_id, ColumnFmt ¶m_vec, char *aggr_cell)
|
||||
@ -55,10 +54,19 @@ public:
|
||||
if (param_id == agg_ctx.aggr_infos_.at(agg_col_id).param_exprs_.count() - 1) {
|
||||
// last param
|
||||
int64_t &count = *reinterpret_cast<int64_t *>(aggr_cell);
|
||||
if (row_sel.is_empty()) {
|
||||
for (int i = bound.start(); i < bound.end(); i++) { count += !pvt_skip.at(i); }
|
||||
if (OB_LIKELY(!agg_ctx.removal_info_.enable_removal_opt_)
|
||||
|| !agg_ctx.removal_info_.is_inverse_agg_) {
|
||||
if (row_sel.is_empty()) {
|
||||
for (int i = bound.start(); i < bound.end(); i++) { count += !pvt_skip.at(i); }
|
||||
} else {
|
||||
for (int i = 0; i < row_sel.size(); i++) { count += !pvt_skip.at(row_sel.index(i)); }
|
||||
}
|
||||
} else {
|
||||
for (int i = 0; i < row_sel.size(); i++) { count += !pvt_skip.at(row_sel.index(i)); }
|
||||
if (row_sel.is_empty()) {
|
||||
for (int i = bound.start(); i < bound.end(); i++) { count -= !pvt_skip.at(i); }
|
||||
} else {
|
||||
for (int i = 0; i < row_sel.size(); i++) { count -= !pvt_skip.at(row_sel.index(i)); }
|
||||
}
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
@ -132,6 +140,23 @@ public:
|
||||
}
|
||||
return OB_SUCCESS;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int add_or_sub_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
bool is_trans = !agg_ctx.removal_info_.is_inverse_agg_;
|
||||
if (!columns.is_null(row_num)) {
|
||||
int64_t &count = *reinterpret_cast<int64_t *>(agg_cell);
|
||||
if (is_trans) {
|
||||
count++;
|
||||
} else {
|
||||
count--;
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
TO_STRING_KV("aggregate", "count");
|
||||
};
|
||||
|
||||
|
||||
@ -13,9 +13,6 @@
|
||||
#define USING_LOG_PREFIX SQL_ENG
|
||||
#include "iaggregate.h"
|
||||
#include "share/datum/ob_datum_util.h"
|
||||
#include "share/aggregate/count.h"
|
||||
#include "share/aggregate/min_max.h"
|
||||
#include "share/aggregate/sum.h"
|
||||
#include "share/aggregate/first_row.h"
|
||||
|
||||
namespace oceanbase
|
||||
@ -40,9 +37,13 @@ extern int init_sum_aggregate(RuntimeContext &agg_ctx, const int64_t agg_col_id,
|
||||
ObIAllocator &allocator, IAggregate *&agg, int32 *tmp_res_size = NULL);
|
||||
extern int init_count_sum_aggregate(RuntimeContext &agg_ctx, const int64_t agg_col_id,
|
||||
ObIAllocator &allocator, IAggregate *&agg);
|
||||
#define INIT_AGGREGATE_CASE(OP_TYPE, func_name) \
|
||||
extern int init_approx_count_distinct_aggregate(RuntimeContext &agg_ctx, const int64_t agg_col_id,
|
||||
ObIAllocator &allocator, IAggregate *&agg);
|
||||
extern int init_sysbit_aggregate(RuntimeContext &agg_ctx, const int64_t agg_col_id,
|
||||
ObIAllocator &allocator, IAggregate *&agg);
|
||||
#define INIT_AGGREGATE_CASE(OP_TYPE, func_name, col_id) \
|
||||
case (OP_TYPE): { \
|
||||
ret = init_##func_name##_aggregate(agg_ctx, i, allocator, agg); \
|
||||
ret = init_##func_name##_aggregate(agg_ctx, col_id, allocator, aggregate); \
|
||||
} break
|
||||
int init_aggregates(RuntimeContext &agg_ctx, ObIAllocator &allocator,
|
||||
ObIArray<IAggregate *> &aggregates)
|
||||
@ -54,18 +55,27 @@ int init_aggregates(RuntimeContext &agg_ctx, ObIAllocator &allocator,
|
||||
}
|
||||
for (int i = 0; OB_SUCC(ret) && i < agg_ctx.aggr_infos_.count(); i++) {
|
||||
ObAggrInfo &aggr_info = agg_ctx.locate_aggr_info(i);
|
||||
IAggregate *agg = nullptr;
|
||||
IAggregate *aggregate = nullptr;
|
||||
if (aggr_info.is_implicit_first_aggr()) {
|
||||
if (OB_FAIL(init_first_row_aggregate(agg_ctx, i, allocator, agg))) {
|
||||
if (OB_FAIL(init_first_row_aggregate(agg_ctx, i, allocator, aggregate))) {
|
||||
SQL_LOG(WARN, "init first row aggregate failed", K(ret));
|
||||
}
|
||||
} else {
|
||||
switch (aggr_info.expr_->type_) {
|
||||
INIT_AGGREGATE_CASE(T_FUN_MIN, min);
|
||||
INIT_AGGREGATE_CASE(T_FUN_MAX, max);
|
||||
INIT_AGGREGATE_CASE(T_FUN_COUNT, count);
|
||||
INIT_AGGREGATE_CASE(T_FUN_SUM, sum);
|
||||
INIT_AGGREGATE_CASE(T_FUN_COUNT_SUM, count_sum);
|
||||
ObExprOperatorType fun_type =
|
||||
(aggr_info.expr_->type_ == T_WINDOW_FUNCTION ? aggr_info.real_aggr_type_ :
|
||||
aggr_info.expr_->type_);
|
||||
switch (fun_type) {
|
||||
INIT_AGGREGATE_CASE(T_FUN_MIN, min, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_MAX, max, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_COUNT, count, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_SUM, sum, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_COUNT_SUM, count_sum, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_APPROX_COUNT_DISTINCT, approx_count_distinct, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS, approx_count_distinct, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE, approx_count_distinct, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_SYS_BIT_OR, sysbit, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_SYS_BIT_AND, sysbit, i);
|
||||
INIT_AGGREGATE_CASE(T_FUN_SYS_BIT_XOR, sysbit, i);
|
||||
default: {
|
||||
ret = OB_NOT_SUPPORTED;
|
||||
SQL_LOG(WARN, "not supported aggregate function", K(ret), K(aggr_info.expr_->type_));
|
||||
@ -73,17 +83,18 @@ int init_aggregates(RuntimeContext &agg_ctx, ObIAllocator &allocator,
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "init aggregate failed", K(ret));
|
||||
} else if (OB_ISNULL(agg)) {
|
||||
} else if (OB_ISNULL(aggregate)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "unexpected null aggregate", K(ret));
|
||||
}
|
||||
}
|
||||
if (OB_SUCC(ret) && OB_FAIL(aggregates.push_back(agg))) {
|
||||
if (OB_SUCC(ret) && OB_FAIL(aggregates.push_back(aggregate))) {
|
||||
SQL_LOG(WARN, "push back element failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
return ret;
|
||||
}
|
||||
|
||||
#undef INIT_AGGREGATE_CASE
|
||||
|
||||
static int32_t agg_cell_tmp_res_size(RuntimeContext &agg_ctx, int64_t agg_col_id)
|
||||
@ -110,6 +121,8 @@ static int32_t agg_cell_tmp_res_size(RuntimeContext &agg_ctx, int64_t agg_col_id
|
||||
} else {
|
||||
ret_size = tmp_res_size;
|
||||
}
|
||||
} else if (info.get_expr_type() == T_FUN_APPROX_COUNT_DISTINCT) {
|
||||
ret_size = sizeof(char *);
|
||||
}
|
||||
return ret_size;
|
||||
}
|
||||
|
||||
@ -400,72 +400,99 @@ protected:
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
ObEvalCtx &ctx = agg_ctx.eval_ctx_;
|
||||
auto &columns = *static_cast<ColumnFmt *>(param_expr.get_vector(ctx));
|
||||
ColumnFmt &columns = *static_cast<ColumnFmt *>(param_expr.get_vector(ctx));
|
||||
bool all_not_null = !columns.has_null();
|
||||
Derived &derived = *static_cast<Derived *>(this);
|
||||
void *tmp_res = derived.get_tmp_res(agg_ctx, agg_col_id, agg_cell);
|
||||
int64_t calc_info = derived.get_batch_calc_info(agg_ctx, agg_col_id, agg_cell);
|
||||
if (OB_LIKELY(row_sel.is_empty() && bound.get_all_rows_active())) {
|
||||
if (all_not_null) {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (OB_FAIL(
|
||||
derived.add_row(agg_ctx, columns, i, agg_col_id, agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
if (OB_LIKELY(!agg_ctx.removal_info_.enable_removal_opt_)) {
|
||||
if (OB_LIKELY(row_sel.is_empty() && bound.get_all_rows_active())) {
|
||||
if (all_not_null) {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (OB_FAIL(
|
||||
derived.add_row(agg_ctx, columns, i, agg_col_id, agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
if (agg_cell != nullptr) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
}
|
||||
} // end for
|
||||
if (agg_cell != nullptr) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
} else {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (OB_FAIL(derived.add_nullable_row(agg_ctx, columns, i, agg_col_id, agg_cell, tmp_res,
|
||||
calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
}
|
||||
} else if (!row_sel.is_empty()) {
|
||||
if (all_not_null) {
|
||||
for (int i = 0; OB_SUCC(ret) && i < row_sel.size(); i++) {
|
||||
if (OB_FAIL(derived.add_row(agg_ctx, columns, row_sel.index(i), agg_col_id, agg_cell,
|
||||
tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
if (agg_cell != nullptr) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
}
|
||||
} else {
|
||||
for (int i = 0; OB_SUCC(ret) && i < row_sel.size(); i++) {
|
||||
if (OB_FAIL(derived.add_nullable_row(agg_ctx, columns, row_sel.index(i), agg_col_id,
|
||||
agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
}
|
||||
} else {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (OB_FAIL(derived.add_nullable_row(agg_ctx, columns, i, agg_col_id, agg_cell, tmp_res,
|
||||
if (all_not_null) {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (skip.at(i)) {
|
||||
} else if (OB_FAIL(derived.add_row(agg_ctx, columns, i, agg_col_id, agg_cell, tmp_res,
|
||||
calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
if (agg_cell != nullptr) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
}
|
||||
} // end for
|
||||
}
|
||||
} else if (!row_sel.is_empty()) {
|
||||
if (all_not_null) {
|
||||
for (int i = 0; OB_SUCC(ret) && i < row_sel.size(); i++) {
|
||||
if (OB_FAIL(derived.add_row(agg_ctx, columns, row_sel.index(i), agg_col_id, agg_cell,
|
||||
tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
if (agg_cell != nullptr) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
} else {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (skip.at(i)) {
|
||||
} else if (OB_FAIL(derived.add_nullable_row(agg_ctx, columns, i, agg_col_id, agg_cell,
|
||||
tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
}
|
||||
} else {
|
||||
for (int i = 0; OB_SUCC(ret) && i < row_sel.size(); i++) {
|
||||
if (OB_FAIL(derived.add_nullable_row(agg_ctx, columns, row_sel.index(i), agg_col_id,
|
||||
agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
}
|
||||
} else {
|
||||
if (all_not_null) {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (skip.at(i)) {
|
||||
} else if (OB_FAIL(derived.add_row(agg_ctx, columns, i, agg_col_id, agg_cell, tmp_res,
|
||||
calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
if (row_sel.is_empty()) {
|
||||
if (bound.get_all_rows_active()) {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
ret = removal_opt<Derived>::add_or_sub_row(derived, agg_ctx, columns, i, agg_col_id, agg_cell,
|
||||
tmp_res, calc_info);
|
||||
if (OB_FAIL(ret)) { SQL_LOG(WARN, "add or sub row failed", K(ret)); }
|
||||
}
|
||||
} else {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (skip.at(i)) {
|
||||
} else {
|
||||
ret = removal_opt<Derived>::add_or_sub_row(derived, agg_ctx, columns, i, agg_col_id, agg_cell,
|
||||
tmp_res, calc_info);
|
||||
if (OB_FAIL(ret)) { SQL_LOG(WARN, "add or sub row failed", K(ret)); }
|
||||
}
|
||||
}
|
||||
} // end for
|
||||
if (agg_cell != nullptr) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
}
|
||||
} else {
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) {
|
||||
if (skip.at(i)) {
|
||||
} else if (OB_FAIL(derived.add_nullable_row(agg_ctx, columns, i, agg_col_id, agg_cell,
|
||||
tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} // end for
|
||||
for (int i = 0; OB_SUCC(ret) && i < row_sel.size(); i++) {
|
||||
ret = removal_opt<Derived>::add_or_sub_row(derived, agg_ctx, columns, i, agg_col_id, agg_cell,
|
||||
tmp_res, calc_info);
|
||||
if (OB_FAIL(ret)) { SQL_LOG(WARN, "add or sub row failed", K(ret)); }
|
||||
}
|
||||
}
|
||||
}
|
||||
if (OB_SUCC(ret)) {
|
||||
@ -595,7 +622,7 @@ protected:
|
||||
} else {
|
||||
MEMSET(skip_buf, 0, skip_size);
|
||||
ObBitVector *tmp_skip = to_bit_vector(skip_buf);
|
||||
tmp_skip->deep_copy(skip, skip_size);
|
||||
tmp_skip->deep_copy(skip, bound.batch_size());
|
||||
Derived &derived = *static_cast<Derived *>(this);
|
||||
for (int param_id = 0; OB_SUCC(ret) && param_id < param_exprs.count(); param_id++) {
|
||||
ObIVector *param_vec = param_exprs.at(param_id)->get_vector(agg_ctx.eval_ctx_);
|
||||
@ -836,9 +863,8 @@ namespace helper
|
||||
int init_aggregates(RuntimeContext &agg_ctx, ObIAllocator &allocator,
|
||||
ObIArray<IAggregate *> &aggregates);
|
||||
|
||||
template <ObItemType op>
|
||||
int init_aggregate(sql::ObEvalCtx &ctx, RuntimeContext &agg_ctx, const int64_t agg_col_id,
|
||||
ObIAllocator &allocator, IAggregate *&agg);
|
||||
int init_aggregate(RuntimeContext &agg_ctx, ObIAllocator &allocator, const int64_t col_id,
|
||||
IAggregate *&aggregate);
|
||||
|
||||
inline constexpr bool is_var_len_agg_cell(VecValueTypeClass vec_tc)
|
||||
{
|
||||
|
||||
@ -33,20 +33,20 @@ inline static int init_min_max_agg(RuntimeContext &agg_ctx,
|
||||
if (OB_FAIL(ret)) { SQL_LOG(WARN, "init aggregate failed", K(ret)); } \
|
||||
} break
|
||||
|
||||
int ret = OB_SUCCESS;
|
||||
agg = nullptr;
|
||||
int ret = OB_SUCCESS;
|
||||
agg = nullptr;
|
||||
|
||||
ObAggrInfo &aggr_info = agg_ctx.locate_aggr_info(agg_col_id);
|
||||
if (OB_ISNULL(aggr_info.expr_)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid null expr", K(ret));
|
||||
} else {
|
||||
VecValueTypeClass res_vec =
|
||||
get_vec_value_tc(aggr_info.expr_->datum_meta_.type_, aggr_info.expr_->datum_meta_.scale_,
|
||||
aggr_info.expr_->datum_meta_.precision_);
|
||||
ObAggrInfo &aggr_info = agg_ctx.locate_aggr_info(agg_col_id);
|
||||
if (OB_ISNULL(aggr_info.expr_)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid null expr", K(ret));
|
||||
} else {
|
||||
VecValueTypeClass res_vec =
|
||||
get_vec_value_tc(aggr_info.expr_->datum_meta_.type_, aggr_info.expr_->datum_meta_.scale_,
|
||||
aggr_info.expr_->datum_meta_.precision_);
|
||||
|
||||
switch (res_vec) {
|
||||
LST_DO_CODE(INIT_AGGREGATE_CASE, AGG_VEC_TC_LIST);
|
||||
switch (res_vec) {
|
||||
LST_DO_CODE(INIT_AGGREGATE_CASE, AGG_VEC_TC_LIST);
|
||||
default: {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "unexpected result type of min/max aggregate", K(ret), K(res_vec));
|
||||
|
||||
@ -28,19 +28,30 @@ using namespace sql;
|
||||
struct CmpCalcInfo
|
||||
{
|
||||
CmpCalcInfo(ObObjMeta obj_meta, int16_t cell_len) :
|
||||
obj_meta_(obj_meta), agg_cell_len_(cell_len), calculated_(0) {}
|
||||
CmpCalcInfo() : obj_meta_(), agg_cell_len_(0), calculated_(0) {}
|
||||
obj_meta_(obj_meta), agg_cell_len_(cell_len), calc_flags_(0) {}
|
||||
CmpCalcInfo() : obj_meta_(), agg_cell_len_(0), calc_flags_(0) {}
|
||||
operator int64_t() const { return flags_; }
|
||||
inline void set_calculated()
|
||||
{
|
||||
calculated_ = static_cast<int16_t>(1);
|
||||
calculated_ = true;
|
||||
}
|
||||
inline void set_min_max_idx_changed()
|
||||
{
|
||||
min_max_idx_changed_ = true;
|
||||
}
|
||||
inline bool calculated() const { return calculated_ == 1; }
|
||||
union {
|
||||
struct {
|
||||
ObObjMeta obj_meta_;
|
||||
int16_t agg_cell_len_; // for fixed length type only
|
||||
int16_t calculated_;
|
||||
union {
|
||||
struct {
|
||||
uint16_t calculated_: 1;
|
||||
uint16_t min_max_idx_changed_: 1;
|
||||
uint16_t reserved_: 14;
|
||||
};
|
||||
uint16_t calc_flags_;
|
||||
};
|
||||
};
|
||||
int64_t flags_;
|
||||
};
|
||||
@ -157,10 +168,12 @@ public:
|
||||
SQL_LOG(WARN, "compare failed", K(ret));
|
||||
} else if ((is_min && cmp_ret > 0) || (!is_min && cmp_ret < 0)) {
|
||||
MEMCPY(agg_cell, row_data, row_len);
|
||||
cmp_info.set_min_max_idx_changed();
|
||||
}
|
||||
} else {
|
||||
MEMCPY(agg_cell, row_data, row_len);
|
||||
cmp_info.set_calculated();
|
||||
cmp_info.set_min_max_idx_changed();
|
||||
}
|
||||
} else {
|
||||
int32_t agg_cell_len = *reinterpret_cast<int32_t *>(agg_cell + sizeof(char *));
|
||||
@ -173,11 +186,13 @@ public:
|
||||
} else if ((is_min && cmp_ret > 0) || (!is_min && cmp_ret < 0)) {
|
||||
*reinterpret_cast<int64_t *>(agg_cell) = reinterpret_cast<int64_t>(row_data);
|
||||
*reinterpret_cast<int32_t *>(agg_cell + sizeof(char *)) = row_len;
|
||||
cmp_info.set_min_max_idx_changed();
|
||||
}
|
||||
} else {
|
||||
*reinterpret_cast<int64_t *>(agg_cell) = reinterpret_cast<int64_t>(row_data);
|
||||
*reinterpret_cast<int32_t *>(agg_cell + sizeof(char *)) = row_len;
|
||||
cmp_info.set_calculated();
|
||||
cmp_info.set_min_max_idx_changed();
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
@ -237,7 +252,10 @@ public:
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
// if result value has variable result length, e.g. string value,
|
||||
// we need copy value to tmp buffer in case value ptr stored was changed after next batch loop.
|
||||
if (not_nulls.at(agg_col_id) && helper::is_var_len_agg_cell(vec_tc)) {
|
||||
|
||||
if (agg_ctx.win_func_agg_) {
|
||||
// do nothing
|
||||
} else if (not_nulls.at(agg_col_id) && helper::is_var_len_agg_cell(vec_tc)) {
|
||||
if (OB_FAIL(set_tmp_var_agg_data(agg_ctx, agg_col_id, agg_cell))) {
|
||||
SQL_LOG(WARN, "set variable aggregate data failed", K(ret));
|
||||
}
|
||||
@ -268,6 +286,31 @@ public:
|
||||
}
|
||||
|
||||
TO_STRING_KV("aggregate", (is_min ? "min" : "max"), K(vec_tc));
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int add_or_sub_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
bool is_trans = !agg_ctx.removal_info_.is_inverse_agg_;
|
||||
if (!columns.is_null(row_num)) {
|
||||
CmpCalcInfo &cmp_info = reinterpret_cast<CmpCalcInfo &>(calc_info);
|
||||
cmp_info.min_max_idx_changed_ = 0;
|
||||
if (OB_FAIL(add_row(agg_ctx, columns, row_num, agg_col_id, agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
} else if (cmp_info.min_max_idx_changed_) {
|
||||
agg_ctx.removal_info_.max_min_index_ = row_num;
|
||||
agg_ctx.removal_info_.is_max_min_idx_changed_ = true;
|
||||
}
|
||||
agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell).set(0);
|
||||
} else if (is_trans) {
|
||||
agg_ctx.removal_info_.null_cnt_++;
|
||||
} else {
|
||||
agg_ctx.removal_info_.null_cnt_--;
|
||||
}
|
||||
// do nothing
|
||||
return ret;
|
||||
}
|
||||
private:
|
||||
int set_tmp_var_agg_data(RuntimeContext &agg_ctx, const int32_t agg_col_id, char *agg_cell)
|
||||
{
|
||||
|
||||
@ -323,23 +323,23 @@ int Processor::collect_group_results(const RowMeta &row_meta,
|
||||
}
|
||||
|
||||
int Processor::setup_rt_info(AggrRowPtr row,
|
||||
const int32_t row_size)
|
||||
RuntimeContext &agg_ctx)
|
||||
{
|
||||
static const int constexpr extra_arr_buf_size = 16;
|
||||
int ret = OB_SUCCESS;
|
||||
OB_ASSERT(row_size == agg_ctx_.row_meta().row_size_);
|
||||
int32_t row_size = agg_ctx.row_meta().row_size_;
|
||||
char *extra_array_buf = nullptr;
|
||||
MEMSET(row, 0, row_size);
|
||||
for (int col_id = 0; col_id < agg_ctx_.aggr_infos_.count(); col_id++) {
|
||||
ObDatumMeta &res_meta = agg_ctx_.aggr_infos_.at(col_id).expr_->datum_meta_;
|
||||
for (int col_id = 0; col_id < agg_ctx.aggr_infos_.count(); col_id++) {
|
||||
ObDatumMeta &res_meta = agg_ctx.aggr_infos_.at(col_id).expr_->datum_meta_;
|
||||
VecValueTypeClass res_tc = get_vec_value_tc(res_meta.type_, res_meta.scale_, res_meta.precision_);
|
||||
char *cell = nullptr;
|
||||
int32_t cell_len = 0;
|
||||
agg_ctx_.row_meta().locate_cell_payload(col_id, row, cell, cell_len);
|
||||
agg_ctx.row_meta().locate_cell_payload(col_id, row, cell, cell_len);
|
||||
// oracle mode use ObNumber as result type for count aggregation
|
||||
// we use int64_t as result type for count aggregation in aggregate row
|
||||
// and cast int64_t to ObNumber during `collect_group_result`
|
||||
if (res_tc == VEC_TC_NUMBER && agg_ctx_.aggr_infos_.at(col_id).get_expr_type() != T_FUN_COUNT) {
|
||||
if (res_tc == VEC_TC_NUMBER && agg_ctx.aggr_infos_.at(col_id).get_expr_type() != T_FUN_COUNT) {
|
||||
ObNumberDesc &d = *reinterpret_cast<ObNumberDesc *>(cell);
|
||||
// set zero number
|
||||
d.len_ = 0;
|
||||
@ -351,27 +351,19 @@ int Processor::setup_rt_info(AggrRowPtr row,
|
||||
*reinterpret_cast<double *>(cell) = double();
|
||||
}
|
||||
}
|
||||
int extra_size = agg_ctx_.row_meta().extra_cnt_ * sizeof(char *);
|
||||
if (agg_ctx_.row_meta().extra_cnt_ > 0) {
|
||||
if (cur_extra_rt_info_idx_ % extra_arr_buf_size == 0) {
|
||||
cur_extra_rt_info_idx_ = 0;
|
||||
extra_rt_info_buf_ = nullptr;
|
||||
extra_rt_info_buf_ = (char *)agg_ctx_.allocator_.alloc(extra_size * extra_arr_buf_size);
|
||||
if (OB_ISNULL(extra_rt_info_buf_)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
SQL_LOG(WARN, "allocate memory failed", K(ret));
|
||||
}
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
int extra_size = agg_ctx.row_meta().extra_cnt_ * sizeof(char *);
|
||||
if (agg_ctx.row_meta().extra_cnt_ > 0) {
|
||||
void *extra_array_buf = agg_ctx.allocator_.alloc(extra_size);
|
||||
if (OB_ISNULL(extra_array_buf)) {
|
||||
ret = OB_ALLOCATE_MEMORY_FAILED;
|
||||
LOG_WARN("allocate memory failed", K(ret));
|
||||
} else {
|
||||
extra_array_buf = extra_rt_info_buf_ + (cur_extra_rt_info_idx_ * extra_size);
|
||||
cur_extra_rt_info_idx_++;
|
||||
MEMSET(extra_array_buf, 0, extra_size);
|
||||
if (OB_FAIL(agg_ctx_.agg_extras_.push_back((AggregateExtras)extra_array_buf))) {
|
||||
if (OB_FAIL(agg_ctx.agg_extras_.push_back((AggregateExtras)extra_array_buf))) {
|
||||
SQL_LOG(WARN, "push back element failed", K(ret));
|
||||
} else {
|
||||
*reinterpret_cast<int32_t *>(row + agg_ctx_.row_meta().extra_idx_offset_) =
|
||||
static_cast<int32_t>(agg_ctx_.agg_extras_.count()) - 1;
|
||||
*reinterpret_cast<int32_t *>(row + agg_ctx.row_meta().extra_idx_offset_) =
|
||||
static_cast<int32_t>(agg_ctx.agg_extras_.count()) - 1;
|
||||
}
|
||||
}
|
||||
}
|
||||
@ -396,7 +388,7 @@ int Processor::single_row_agg_batch(AggrRowPtr *agg_rows, const int64_t batch_si
|
||||
sql::EvalBound bound(1, true);
|
||||
for (int i = 0; OB_SUCC(ret) && i < batch_size; i++) {
|
||||
if (skip.at(i)) {
|
||||
} else if (OB_FAIL(setup_rt_info(agg_rows[i], get_aggregate_row_size()))) {
|
||||
} else if (OB_FAIL(setup_rt_info(agg_rows[i], agg_ctx_))) {
|
||||
SQL_LOG(WARN, "setup runtime info failed", K(ret));
|
||||
} else {
|
||||
AggrRowPtr row = agg_rows[i];
|
||||
@ -591,7 +583,7 @@ int Processor::init_scalar_aggregate_row(ObCompactRow *&row, RowMeta &row_meta,
|
||||
row = new(row_buf)ObCompactRow();
|
||||
row->init(row_meta);
|
||||
AggrRowPtr agg_row = (char *)row->get_extra_payload(row_meta);
|
||||
if (OB_FAIL(setup_rt_info(agg_row, agg_ctx_.row_meta().row_size_))) {
|
||||
if (OB_FAIL(setup_rt_info(agg_row, agg_ctx_))) {
|
||||
LOG_WARN("setup rt info failed", K(ret));
|
||||
}
|
||||
}
|
||||
@ -632,7 +624,8 @@ int Processor::add_one_aggregate_row(AggrRowPtr data, const int32_t row_size,
|
||||
bool push_agg_row /*true*/)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
if (OB_FAIL(setup_rt_info(data, row_size))) {
|
||||
UNUSED(row_size);
|
||||
if (OB_FAIL(setup_rt_info(data, agg_ctx_))) {
|
||||
SQL_LOG(WARN, "setup runtime info failed", K(ret));
|
||||
} else if (push_agg_row && OB_FAIL(agg_ctx_.agg_rows_.push_back(data))) {
|
||||
SQL_LOG(WARN, "push back element failed", K(ret));
|
||||
|
||||
@ -33,7 +33,7 @@ public:
|
||||
support_fast_single_row_agg_(false), has_distinct_(false), has_order_by_(false), dir_id_(-1),
|
||||
op_eval_infos_(nullptr),
|
||||
allocator_(label, OB_MALLOC_NORMAL_BLOCK_SIZE, tenant_id, ObCtxIds::WORK_AREA),
|
||||
agg_ctx_(eval_ctx, tenant_id, aggr_infos, label, allocator_),
|
||||
agg_ctx_(eval_ctx, tenant_id, aggr_infos, label),
|
||||
aggregates_(allocator_, aggr_infos.count()),
|
||||
fast_single_row_aggregates_(allocator_, aggr_infos.count()), extra_rt_info_buf_(nullptr),
|
||||
cur_extra_rt_info_idx_(0), add_one_row_fns_(allocator_, aggr_infos.count()), row_selector_(nullptr)
|
||||
@ -212,9 +212,14 @@ public:
|
||||
int init_scalar_aggregate_row(ObCompactRow *&row, RowMeta &row_meta, ObIAllocator &allocator);
|
||||
|
||||
int init_fast_single_row_aggs();
|
||||
private:
|
||||
int setup_rt_info(AggrRowPtr data, const int32_t row_size);
|
||||
|
||||
RuntimeContext *get_rt_ctx() { return &agg_ctx_; }
|
||||
|
||||
static int setup_rt_info(AggrRowPtr data, RuntimeContext &agg_ctx);
|
||||
|
||||
ObIArray<IAggregate *> &get_aggregates() { return aggregates_; }
|
||||
|
||||
private:
|
||||
int setup_bypass_rt_infos(const int64_t batch_size);
|
||||
|
||||
int llc_init_empty(ObExpr &expr, ObEvalCtx &eval_ctx) const;
|
||||
@ -249,4 +254,4 @@ private:
|
||||
} // end aggregate
|
||||
} // end share
|
||||
} // end oceanbase
|
||||
#endif // OCEANBASE_SHARE_AGGREGATE_PROCESSOR_H_
|
||||
#endif // OCEANBASE_SHARE_AGGREGATE_PROCESSOR_H_
|
||||
|
||||
@ -19,6 +19,8 @@ namespace share
|
||||
{
|
||||
namespace aggregate
|
||||
{
|
||||
// TODO: add approx_distinct_count
|
||||
// TODO: add sysbit_ops
|
||||
#define INIT_GENERAL_CASE(func_type, vec_tc) \
|
||||
case (vec_tc): { \
|
||||
ret = \
|
||||
|
||||
@ -36,7 +36,7 @@ public:
|
||||
const int32_t agg_col_id, const char *agg_cell,
|
||||
const int32_t agg_cell_len)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
int ret = OB_SUCCESS;
|
||||
const NotNullBitVector ¬nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
int64_t output_idx = agg_ctx.eval_ctx_.get_batch_idx();
|
||||
ResultFmt *res_vec = static_cast<ResultFmt *>(agg_expr.get_vector(agg_ctx.eval_ctx_));
|
||||
@ -194,7 +194,7 @@ public:
|
||||
return ret;
|
||||
}
|
||||
void set_first_row(bool is_first_row) { is_first_row_ = is_first_row; }
|
||||
TO_STRING_KV("aggregate", "single_row", K(in_tc), K(out_tc), K(is_first_row_));
|
||||
TO_STRING_KV("aggregate", "single_row", K(in_tc), K(out_tc), K(is_first_row_), K(agg_func));
|
||||
private:
|
||||
bool is_first_row_;
|
||||
};
|
||||
|
||||
@ -87,6 +87,24 @@ public:
|
||||
return ret;
|
||||
}
|
||||
|
||||
template<typename ColumnFmt>
|
||||
int sub_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
const char* param_payload = nullptr;
|
||||
int32_t param_len = 0;
|
||||
columns.get_payload(row_num, param_payload, param_len);
|
||||
const ParamType *lparam = reinterpret_cast<const ParamType *>(param_payload);
|
||||
const ResultType &rparam = *reinterpret_cast<const ResultType *>(agg_cell);
|
||||
SQL_LOG(DEBUG, "sum sub row", K(agg_col_id));
|
||||
ret = sub_values(rparam, *lparam, agg_cell, sizeof(ResultType));
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "sub value failed", K(ret));
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int add_nullable_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
@ -104,6 +122,31 @@ public:
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int add_or_sub_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
bool is_trans = !agg_ctx.removal_info_.is_inverse_agg_;
|
||||
if (!columns.is_null(row_num)) {
|
||||
if (is_trans) {
|
||||
if (OB_FAIL(add_row(agg_ctx, columns, row_num, agg_col_id, agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} else if (OB_FAIL(
|
||||
sub_row(agg_ctx, columns, row_num, agg_col_id, agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "sub row failed", K(ret));
|
||||
}
|
||||
} else {
|
||||
if (is_trans) {
|
||||
agg_ctx.removal_info_.null_cnt_++;
|
||||
} else {
|
||||
agg_ctx.removal_info_.null_cnt_--;
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int collect_group_result(RuntimeContext &agg_ctx, const sql::ObExpr &agg_expr,
|
||||
const int32_t agg_col_id, const char *agg_cell,
|
||||
@ -116,7 +159,7 @@ public:
|
||||
ColumnFmt *res_vec = static_cast<ColumnFmt *>(agg_expr.get_vector(agg_ctx.eval_ctx_));
|
||||
if (OB_LIKELY(not_nulls.at(agg_col_id))) {
|
||||
CellWriter<ResultType>::set(agg_cell, agg_cell_len, res_vec, output_idx, nullptr);
|
||||
} else if (agg_expr.type_ == T_FUN_COUNT_SUM) {
|
||||
} else if (agg_ctx.locate_aggr_info(agg_col_id).get_expr_type() == T_FUN_COUNT_SUM) {
|
||||
if (VEC_TC_INTEGER == out_tc) {
|
||||
int64_t res = 0;
|
||||
res_vec->set_payload(output_idx, &res, sizeof(int64_t));
|
||||
@ -243,21 +286,49 @@ public:
|
||||
int ret = OB_SUCCESS;
|
||||
TmpStore &tmp_res = *reinterpret_cast<TmpStore *>(get_tmp_res(agg_ctx, agg_col_id, agg_cell));
|
||||
if (OB_LIKELY(tmp_res != 0)) {
|
||||
ResultType &res = *reinterpret_cast<ResultType *>(agg_cell);
|
||||
const int32_t agg_cell_len = agg_ctx.row_meta().get_cell_len(agg_col_id, agg_cell);
|
||||
ObScale scale = agg_ctx.get_first_param_scale(agg_col_id);
|
||||
if (out_tc == VEC_TC_NUMBER) {
|
||||
if (OB_FAIL(add_value_to_nmb(tmp_res, scale, agg_cell))) {
|
||||
SQL_LOG(WARN, "add value to nmb failed", K(ret));
|
||||
if (OB_LIKELY(!agg_ctx.removal_info_.enable_removal_opt_)
|
||||
|| !agg_ctx.removal_info_.is_inverse_agg_) {
|
||||
ResultType &res = *reinterpret_cast<ResultType *>(agg_cell);
|
||||
const int32_t agg_cell_len = agg_ctx.row_meta().get_cell_len(agg_col_id, agg_cell);
|
||||
ObScale scale = agg_ctx.get_first_param_scale(agg_col_id);
|
||||
if (out_tc == VEC_TC_NUMBER) {
|
||||
if (OB_FAIL(add_value_to_nmb(tmp_res, scale, agg_cell))) {
|
||||
SQL_LOG(WARN, "add value to nmb failed", K(ret));
|
||||
}
|
||||
} else {
|
||||
ResultType *res_val = reinterpret_cast<ResultType *>(agg_cell);
|
||||
ret = add_values(tmp_res, *res_val, agg_cell, agg_ctx.get_cell_len(agg_col_id, agg_cell));
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "do op failed", K(ret));
|
||||
} else {
|
||||
tmp_res = 0;
|
||||
}
|
||||
} else {
|
||||
ResultType *res_val = reinterpret_cast<ResultType *>(agg_cell);
|
||||
ret = add_values(tmp_res, *res_val, agg_cell, agg_ctx.get_cell_len(agg_col_id, agg_cell));
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "do op failed", K(ret));
|
||||
} else {
|
||||
tmp_res = 0;
|
||||
if (out_tc == VEC_TC_NUMBER) {
|
||||
ObScale scale = agg_ctx.get_first_param_scale(agg_col_id);
|
||||
number::ObNumber param_nmb, res_nmb;
|
||||
number::ObNumber cur_nmb(*reinterpret_cast<number::ObCompactNumber *>(agg_cell));
|
||||
number::ObCompactNumber *res_cnum = reinterpret_cast<number::ObCompactNumber *>(agg_cell);
|
||||
ObNumStackAllocator<3> tmp_alloc;
|
||||
if (OB_FAIL(to_nmb(tmp_res, scale, tmp_alloc, param_nmb))) {
|
||||
SQL_LOG(WARN, "to number failed", K(ret));
|
||||
} else if (OB_FAIL(cur_nmb.sub(param_nmb, res_nmb, tmp_alloc))) {
|
||||
SQL_LOG(WARN, "number::sub failed", K(ret));
|
||||
} else {
|
||||
res_cnum->desc_ = res_nmb.d_;
|
||||
MEMCPY(&(res_cnum->digits_[0]), res_nmb.get_digits(), sizeof(uint32_t) * res_nmb.d_.len_);
|
||||
tmp_res = 0;
|
||||
}
|
||||
} else {
|
||||
constexpr unsigned res_bits =
|
||||
(out_tc == VEC_TC_NUMBER ? 128 : sizeof(ResultType) * CHAR_BIT);
|
||||
using res_int_type = wide::ObWideInteger<res_bits>;
|
||||
res_int_type &res_val = *reinterpret_cast<res_int_type *>(agg_cell);
|
||||
// overflow is impossible
|
||||
sub_values(res_val, tmp_res, agg_cell, sizeof(res_int_type));
|
||||
tmp_res = 0;
|
||||
}
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
@ -271,14 +342,16 @@ public:
|
||||
if (!is_null) {
|
||||
const ParamType *row_param = reinterpret_cast<const ParamType *>(data);
|
||||
TmpStore *tmp_res = reinterpret_cast<TmpStore *>(get_tmp_res(agg_ctx, agg_col_idx, agg_cell));
|
||||
TmpStore copied_tmp_res = *tmp_res;
|
||||
SQL_LOG(DEBUG, "sum with tmp::add_one_row ", K(*row_param), K(*tmp_res), K(batch_idx));
|
||||
ret = add_overflow(*row_param, *tmp_res, reinterpret_cast<char *>(tmp_res), sizeof(TmpStore));
|
||||
ret = add_overflow(*row_param, copied_tmp_res, reinterpret_cast<char *>(tmp_res),
|
||||
sizeof(TmpStore));
|
||||
if (OB_FAIL(ret)) {
|
||||
if (OB_LIKELY(OB_OPERATE_OVERFLOW == ret)) {
|
||||
ret = OB_SUCCESS;
|
||||
if (out_tc == VEC_TC_NUMBER) {
|
||||
ObScale scale = agg_ctx.get_first_param_scale(agg_col_idx);
|
||||
if (OB_FAIL(add_value_to_nmb(*tmp_res, scale, agg_cell))) {
|
||||
if (OB_FAIL(add_value_to_nmb(copied_tmp_res, scale, agg_cell))) {
|
||||
SQL_LOG(WARN, "add value to nmb failed", K(ret));
|
||||
} else if (OB_FAIL(add_value_to_nmb(*row_param, scale, agg_cell))) {
|
||||
SQL_LOG(WARN, "add value to nmb failed", K(ret));
|
||||
@ -315,16 +388,19 @@ public:
|
||||
int32_t param_len = 0;
|
||||
columns.get_payload(row_num, param_payload, param_len);
|
||||
const ParamType *row_param = reinterpret_cast<const ParamType *>(param_payload);
|
||||
// add_overflow may overwrite `tmp_res`, deep copy is needed
|
||||
TmpStore &tmp_res = *reinterpret_cast<TmpStore *>(tmp_res_ptr);
|
||||
TmpStore copied_tmp_res = tmp_res;
|
||||
|
||||
SQL_LOG(DEBUG, "sum add row", K(row_param), K(row_num), K(tmp_res));
|
||||
ret = add_overflow(*row_param, tmp_res, reinterpret_cast<char *>(&tmp_res), sizeof(TmpStore));
|
||||
ret = add_overflow(*row_param, copied_tmp_res, reinterpret_cast<char *>(&tmp_res),
|
||||
sizeof(TmpStore));
|
||||
if (OB_FAIL(ret)) {
|
||||
if (OB_LIKELY(ret == OB_OPERATE_OVERFLOW)) {
|
||||
ret = OB_SUCCESS;
|
||||
if (out_tc == VEC_TC_NUMBER) {
|
||||
ObScale in_scale = agg_ctx.get_first_param_scale(agg_col_id);
|
||||
if (OB_FAIL(add_value_to_nmb(tmp_res, in_scale, agg_cell))) {
|
||||
if (OB_FAIL(add_value_to_nmb(copied_tmp_res, in_scale, agg_cell))) {
|
||||
SQL_LOG(WARN, "add_value_to_nmb failed", K(ret));
|
||||
} else if (OB_FAIL(add_value_to_nmb(*row_param, in_scale, agg_cell))) {
|
||||
SQL_LOG(WARN, "add_value_to_nmb failed", K(ret));
|
||||
@ -333,7 +409,7 @@ public:
|
||||
}
|
||||
} else {
|
||||
ResultType &res_val = *reinterpret_cast<ResultType *>(agg_cell);
|
||||
if (OB_FAIL(add_values(tmp_res, res_val, agg_cell, sizeof(ResultType)))) {
|
||||
if (OB_FAIL(add_values(copied_tmp_res, res_val, agg_cell, sizeof(ResultType)))) {
|
||||
SQL_LOG(WARN, "add values failed", K(ret));
|
||||
} else if (OB_FAIL(add_values(*row_param, res_val, agg_cell, sizeof(ResultType)))) {
|
||||
SQL_LOG(WARN, "add values failed", K(ret));
|
||||
@ -367,6 +443,87 @@ public:
|
||||
return ret;
|
||||
}
|
||||
|
||||
template<typename ColumnFmt>
|
||||
int sub_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
// for substraction with tmp result, we added all substractions in tmp results
|
||||
// and do substraction when overflow or tmp result collecting happended.
|
||||
const char *payload = columns.get_payload(row_num);
|
||||
const ParamType ¶m = *reinterpret_cast<const ParamType *>(payload);
|
||||
TmpStore &tmp_val = *reinterpret_cast<TmpStore *>(tmp_res);
|
||||
TmpStore copied_tmp_val = tmp_val;
|
||||
ret = add_overflow(param, copied_tmp_val, (char *)tmp_res, sizeof(TmpStore));
|
||||
if (OB_FAIL(ret)) {
|
||||
if (OB_LIKELY(ret == OB_OPERATE_OVERFLOW)) {
|
||||
ret = OB_SUCCESS;
|
||||
if (out_tc == VEC_TC_NUMBER) {
|
||||
ObNumStackAllocator<4> tmp_alloc;
|
||||
number::ObNumber sub1, sub2, tmp_sum, res_nmb;
|
||||
number::ObNumber cur_nmb(*reinterpret_cast<number::ObCompactNumber *>(agg_cell));
|
||||
ObScale scale = agg_ctx.get_first_param_scale(agg_col_id);
|
||||
if (OB_FAIL(to_nmb(tmp_val, scale, tmp_alloc, sub1))) {
|
||||
SQL_LOG(WARN, "to number failed", K(ret));
|
||||
} else if (OB_FAIL(to_nmb(param, scale, tmp_alloc, sub2))) {
|
||||
SQL_LOG(WARN, "to number failed", K(ret));
|
||||
} else if (OB_FAIL(sub1.add(sub2, tmp_sum, tmp_alloc))) {
|
||||
SQL_LOG(WARN, "number::add failed", K(ret));
|
||||
} else if (OB_FAIL(cur_nmb.sub(tmp_sum, res_nmb, tmp_alloc))) {
|
||||
SQL_LOG(WARN, "number::sub failed", K(ret));
|
||||
} else {
|
||||
number::ObCompactNumber *res_cnum = reinterpret_cast<number::ObCompactNumber *>(agg_cell);
|
||||
res_cnum->desc_ = res_nmb.d_;
|
||||
MEMCPY(&(res_cnum->digits_[0]), res_nmb.get_digits(), res_nmb.d_.len_ * sizeof(uint32_t));
|
||||
tmp_val = 0;
|
||||
}
|
||||
} else {
|
||||
constexpr unsigned res_bits =
|
||||
(out_tc == VEC_TC_NUMBER ? 128 : sizeof(ResultType) * CHAR_BIT);
|
||||
using res_int_type = wide::ObWideInteger<res_bits>;
|
||||
res_int_type &res_val = *reinterpret_cast<res_int_type *>(agg_cell);
|
||||
if (OB_FAIL(sub_values(res_val, tmp_val, agg_cell, sizeof(ResultType)))) {
|
||||
SQL_LOG(WARN, "sub values failed", K(ret));
|
||||
} else if (OB_FAIL(sub_values(res_val, param, agg_cell, sizeof(ResultType)))) {
|
||||
SQL_LOG(WARN, "sub values failed", K(ret));
|
||||
} else {
|
||||
tmp_val = 0;
|
||||
}
|
||||
}
|
||||
} else {
|
||||
SQL_LOG(WARN, "add overflow", K(ret));
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int add_or_sub_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
bool is_trans = !agg_ctx.removal_info_.is_inverse_agg_;
|
||||
if (!columns.is_null(row_num)) {
|
||||
if (is_trans) {
|
||||
ret = add_row(agg_ctx, columns, row_num, agg_col_id, agg_cell, tmp_res, calc_info);
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "add row failed", K(ret));
|
||||
}
|
||||
} else {
|
||||
if (OB_FAIL(sub_row(agg_ctx, columns, row_num, agg_col_id, agg_cell, tmp_res, calc_info))) {
|
||||
SQL_LOG(WARN, "sub row failed", K(ret));
|
||||
}
|
||||
}
|
||||
} else {
|
||||
if (is_trans) {
|
||||
agg_ctx.removal_info_.null_cnt_++;
|
||||
} else {
|
||||
agg_ctx.removal_info_.null_cnt_--;
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int collect_group_result(RuntimeContext &agg_ctx, const sql::ObExpr &agg_expr,
|
||||
const int32_t agg_col_id, const char *agg_cell, const int32_t agg_len)
|
||||
@ -395,7 +552,7 @@ public:
|
||||
CellWriter<ResultType>::set(agg_cell, agg_len, res_vec, output_idx, nullptr);
|
||||
}
|
||||
}
|
||||
} else if (agg_expr.type_ == T_FUN_COUNT_SUM) {
|
||||
} else if (agg_ctx.locate_aggr_info(agg_col_id).get_expr_type() == T_FUN_COUNT_SUM) {
|
||||
if (lib::is_oracle_mode() || out_tc == VEC_TC_NUMBER) {
|
||||
number::ObNumber res_nmb;
|
||||
res_nmb.set_zero();
|
||||
@ -483,6 +640,27 @@ private:
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
template<typename T>
|
||||
int to_nmb(const T &v, ObScale in_scale, ObIAllocator &alloc, number::ObNumber &res_nmb)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
if (sizeof(T) > sizeof(int64_t)) {
|
||||
if (OB_FAIL(wide::to_number(v, in_scale, alloc, res_nmb))) {
|
||||
SQL_LOG(WARN, "wide::to_number failed", K(ret));
|
||||
}
|
||||
} else if (sizeof(T) <= sizeof(int32_t) || std::is_same<T, int64_t>::value) {
|
||||
int64_t tmp_v = v;
|
||||
if (OB_FAIL(wide::to_number(tmp_v, in_scale, alloc, res_nmb))) {
|
||||
SQL_LOG(WARN, "wide::to_number failed", K(ret));
|
||||
}
|
||||
} else if (std::is_same<T, uint64_t>::value) {
|
||||
if (OB_FAIL(wide::to_number(static_cast<uint64_t>(v), in_scale, alloc, res_nmb))) {
|
||||
SQL_LOG(WARN, "wide::to_number failed", K(ret));
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
template<typename T>
|
||||
int add_value_to_nmb(const T &v, const ObScale in_scale, char *agg_cell)
|
||||
{
|
||||
@ -490,23 +668,8 @@ private:
|
||||
ObNumStackAllocator<2> tmp_alloc;
|
||||
number::ObNumber res_nmb, param_nmb;
|
||||
number::ObNumber tmp(*reinterpret_cast<number::ObCompactNumber *>(agg_cell));
|
||||
if (sizeof(T) > sizeof(int64_t)) {
|
||||
if (OB_FAIL(wide::to_number(v, in_scale, tmp_alloc, param_nmb))) {
|
||||
SQL_LOG(WARN, "to_number failed", K(ret));
|
||||
}
|
||||
} else if (sizeof(T) <= sizeof(int32_t)) {
|
||||
int64_t tmp_v = v;
|
||||
if (OB_FAIL(wide::to_number(tmp_v, in_scale, tmp_alloc, param_nmb))) {
|
||||
SQL_LOG(WARN, "wide::to_number failed", K(ret));
|
||||
}
|
||||
} else if (std::is_same<T, uint64_t>::value
|
||||
&& OB_FAIL(
|
||||
wide::to_number(static_cast<uint64_t>(v), in_scale, tmp_alloc, param_nmb))) {
|
||||
SQL_LOG(WARN, "wide::to_number failed", K(ret));
|
||||
} else if (OB_FAIL(wide::to_number(static_cast<int64_t>(v), in_scale, tmp_alloc, param_nmb))) {
|
||||
SQL_LOG(WARN, "wide::to_number failed", K(ret));
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
if (OB_FAIL(to_nmb(v, in_scale, tmp_alloc, param_nmb))) {
|
||||
SQL_LOG(WARN, "to_nmb failed", K(ret));
|
||||
} else if (OB_FAIL(tmp.add_v3(param_nmb, res_nmb, tmp_alloc))) {
|
||||
SQL_LOG(WARN, "ObNumber::add_v3 failed", K(ret));
|
||||
} else {
|
||||
|
||||
@ -35,6 +35,25 @@
|
||||
} \
|
||||
}
|
||||
|
||||
#define ADD_OR_SUB_NMB(fmt) \
|
||||
do { \
|
||||
fmt *columns = static_cast<fmt *>(input_vec); \
|
||||
if (row_sel.is_empty()) { \
|
||||
for (int i = bound.start(); OB_SUCC(ret) && i < bound.end(); i++) { \
|
||||
if (skip.at(i)) { \
|
||||
} else { \
|
||||
ret = add_or_sub_row<fmt>(agg_ctx, *columns, i, agg_col_id, agg_cell, nullptr, \
|
||||
mock_calc_info); \
|
||||
} \
|
||||
} \
|
||||
} else { \
|
||||
for (int i = 0; OB_SUCC(ret) && i < row_sel.size(); i++) { \
|
||||
ret = add_or_sub_row<fmt>(agg_ctx, *columns, row_sel.index(i), agg_col_id, agg_cell, \
|
||||
nullptr, mock_calc_info); \
|
||||
} \
|
||||
} \
|
||||
} while (false)
|
||||
|
||||
namespace oceanbase
|
||||
{
|
||||
namespace share
|
||||
@ -87,8 +106,8 @@ public:
|
||||
return ret;
|
||||
}
|
||||
inline int add_batch_rows(RuntimeContext &agg_ctx, const int32_t agg_col_id,
|
||||
const sql::ObBitVector &skip, const sql::EvalBound &bound, char *agg_cell,
|
||||
const RowSelector row_sel = RowSelector{}) override
|
||||
const sql::ObBitVector &skip, const sql::EvalBound &bound,
|
||||
char *agg_cell, const RowSelector row_sel = RowSelector{}) override
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
#ifndef NDEBUG
|
||||
@ -100,45 +119,70 @@ public:
|
||||
OB_ASSERT(param_expr != NULL);
|
||||
ObIVector *input_vec = param_expr->get_vector(agg_ctx.eval_ctx_);
|
||||
VectorFormat in_fmt = param_expr->get_format(agg_ctx.eval_ctx_);
|
||||
uint32_t sum_digits[number::ObNumber::OB_CALC_BUFFER_SIZE] = {0};
|
||||
char buf_alloc1[number::ObNumber::MAX_CALC_BYTE_LEN] = {0};
|
||||
char buf_alloc2[number::ObNumber::MAX_CALC_BYTE_LEN] = {0};
|
||||
ObDataBuffer allocator1(buf_alloc1, number::ObNumber::MAX_CALC_BYTE_LEN);
|
||||
ObDataBuffer allocator2(buf_alloc2, number::ObNumber::MAX_CALC_BYTE_LEN);
|
||||
number::ObNumber result(*reinterpret_cast<number::ObCompactNumber *>(agg_cell));
|
||||
bool all_skip = true;
|
||||
switch (in_fmt) {
|
||||
case common::VEC_UNIFORM: {
|
||||
ACCUMULATE_NMB(ObUniformFormat<false>);
|
||||
break;
|
||||
if (OB_LIKELY(!agg_ctx.removal_info_.enable_removal_opt_)) {
|
||||
uint32_t sum_digits[number::ObNumber::OB_CALC_BUFFER_SIZE] = {0};
|
||||
char buf_alloc1[number::ObNumber::MAX_CALC_BYTE_LEN] = {0};
|
||||
char buf_alloc2[number::ObNumber::MAX_CALC_BYTE_LEN] = {0};
|
||||
ObDataBuffer allocator1(buf_alloc1, number::ObNumber::MAX_CALC_BYTE_LEN);
|
||||
ObDataBuffer allocator2(buf_alloc2, number::ObNumber::MAX_CALC_BYTE_LEN);
|
||||
number::ObNumber result(*reinterpret_cast<number::ObCompactNumber *>(agg_cell));
|
||||
bool all_skip = true;
|
||||
switch (in_fmt) {
|
||||
case common::VEC_UNIFORM: {
|
||||
ACCUMULATE_NMB(ObUniformFormat<false>);
|
||||
break;
|
||||
}
|
||||
case common::VEC_UNIFORM_CONST: {
|
||||
ACCUMULATE_NMB(ObUniformFormat<true>);
|
||||
break;
|
||||
}
|
||||
case common::VEC_DISCRETE: {
|
||||
ACCUMULATE_NMB(ObDiscreteFormat);
|
||||
break;
|
||||
}
|
||||
case common::VEC_CONTINUOUS: {
|
||||
ACCUMULATE_NMB(ObContinuousFormat);
|
||||
break;
|
||||
}
|
||||
default: {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "unexpected format for sum aggregate", K(ret), K(*this), K(in_fmt));
|
||||
}
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "accumulate number failed", K(ret));
|
||||
} else if (OB_LIKELY(!all_skip)) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
number::ObCompactNumber *cnum = reinterpret_cast<number::ObCompactNumber *>(agg_cell);
|
||||
cnum->desc_ = result.d_;
|
||||
MEMCPY(&(cnum->digits_[0]), result.get_digits(), result.d_.len_ * sizeof(uint32_t));
|
||||
}
|
||||
SQL_LOG(DEBUG, "number result", K(result), K(all_skip), K(ret));
|
||||
} else {
|
||||
int64_t mock_calc_info = 0;
|
||||
switch(in_fmt) {
|
||||
case common::VEC_UNIFORM: {
|
||||
ADD_OR_SUB_NMB(ObUniformFormat<false>);
|
||||
break;
|
||||
}
|
||||
case common::VEC_DISCRETE: {
|
||||
ADD_OR_SUB_NMB(ObDiscreteFormat);
|
||||
break;
|
||||
}
|
||||
case common::VEC_UNIFORM_CONST: {
|
||||
ADD_OR_SUB_NMB(ObUniformFormat<true>);
|
||||
break;
|
||||
}
|
||||
default: {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "invalid input format", K(ret), K(in_fmt));
|
||||
}
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "add or sub rows failed", K(ret));
|
||||
}
|
||||
}
|
||||
case common::VEC_UNIFORM_CONST: {
|
||||
ACCUMULATE_NMB(ObUniformFormat<true>);
|
||||
break;
|
||||
}
|
||||
case common::VEC_DISCRETE: {
|
||||
ACCUMULATE_NMB(ObDiscreteFormat);
|
||||
break;
|
||||
}
|
||||
case common::VEC_CONTINUOUS: {
|
||||
ACCUMULATE_NMB(ObContinuousFormat);
|
||||
break;
|
||||
}
|
||||
default: {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
SQL_LOG(WARN, "unexpected format for sum aggregate", K(ret), K(*this), K(in_fmt));
|
||||
}
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
SQL_LOG(WARN, "accumulate number failed", K(ret));
|
||||
} else if (OB_LIKELY(!all_skip)) {
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
not_nulls.set(agg_col_id);
|
||||
number::ObCompactNumber *cnum = reinterpret_cast<number::ObCompactNumber *>(agg_cell);
|
||||
cnum->desc_ = result.d_;
|
||||
MEMCPY(&(cnum->digits_[0]), result.get_digits(), result.d_.len_ * sizeof(uint32_t));
|
||||
}
|
||||
SQL_LOG(DEBUG, "number result", K(result), K(all_skip), K(ret));
|
||||
return ret;
|
||||
}
|
||||
|
||||
@ -182,6 +226,40 @@ public:
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int add_or_sub_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
UNUSEDx(tmp_res, calc_info);
|
||||
bool is_trans = !agg_ctx.removal_info_.is_inverse_agg_;
|
||||
if (!columns.is_null(row_num)) {
|
||||
const number::ObCompactNumber *param_cnum =
|
||||
reinterpret_cast<const number::ObCompactNumber *>(columns.get_payload(row_num));
|
||||
number::ObCompactNumber *res_cnum = reinterpret_cast<number::ObCompactNumber *>(agg_cell);
|
||||
number::ObNumber param1(*param_cnum);
|
||||
number::ObNumber param2(*reinterpret_cast<number::ObCompactNumber *>(agg_cell));
|
||||
number::ObNumber res_nmb;
|
||||
ObNumStackOnceAlloc tmp_alloc;
|
||||
if (is_trans) {
|
||||
if (OB_FAIL(param2.add_v3(param1, res_nmb, tmp_alloc))) {
|
||||
SQL_LOG(WARN, "add number failed", K(ret));
|
||||
}
|
||||
} else if (OB_FAIL(param2.sub_v3(param1, res_nmb, tmp_alloc))){
|
||||
SQL_LOG(WARN, "sub number failed", K(ret));
|
||||
}
|
||||
if (OB_SUCC(ret)) {
|
||||
res_cnum->desc_ = res_nmb.d_;
|
||||
MEMCPY(&(res_cnum->digits_[0]), res_nmb.get_digits(), res_nmb.d_.len_ * sizeof(uint32_t));
|
||||
}
|
||||
} else if (is_trans){
|
||||
agg_ctx.removal_info_.null_cnt_++;
|
||||
} else {
|
||||
agg_ctx.removal_info_.null_cnt_--;
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
TO_STRING_KV("aggregate", "sum_nmb", K_(is_ora_count_sum));
|
||||
private:
|
||||
template<typename ColumnFmt, bool has_null>
|
||||
@ -275,4 +353,5 @@ private:
|
||||
} // end share
|
||||
} // end oceanbase
|
||||
#undef ACCUMULATE_NMB
|
||||
#undef ADD_OR_SUB_NMB
|
||||
#endif // OCEANBASE_SHARE_AGGREGATE_SUM_NMB_H_
|
||||
71
src/share/aggregate/sys_bit.cpp
Normal file
71
src/share/aggregate/sys_bit.cpp
Normal file
@ -0,0 +1,71 @@
|
||||
/**
|
||||
* Copyright (c) 2021 OceanBase
|
||||
* OceanBase CE is licensed under Mulan PubL v2.
|
||||
* You can use this software according to the terms and conditions of the Mulan PubL v2.
|
||||
* You may obtain a copy of Mulan PubL v2 at:
|
||||
* http://license.coscl.org.cn/MulanPubL-2.0
|
||||
* THIS SOFTWARE IS PROVIDED ON AN "AS IS" BASIS, WITHOUT WARRANTIES OF ANY KIND,
|
||||
* EITHER EXPRESS OR IMPLIED, INCLUDING BUT NOT LIMITED TO NON-INFRINGEMENT,
|
||||
* MERCHANTABILITY OR FIT FOR A PARTICULAR PURPOSE.
|
||||
* See the Mulan PubL v2 for more details.
|
||||
*/
|
||||
#define USING_LOG_PREFIX SQL_ENG
|
||||
#include "sys_bit.h"
|
||||
|
||||
namespace oceanbase
|
||||
{
|
||||
namespace share
|
||||
{
|
||||
namespace aggregate
|
||||
{
|
||||
namespace helper
|
||||
{
|
||||
int init_sysbit_aggregate(RuntimeContext &agg_ctx, const int64_t agg_col_id,
|
||||
ObIAllocator &allocator, IAggregate *&agg)
|
||||
{
|
||||
#define INIT_SYS_BIT_AGG(op_type) \
|
||||
do { \
|
||||
if (in_tc == VEC_TC_INTEGER) { \
|
||||
ret = init_agg_func<SysBitAggregate<op_type, VEC_TC_INTEGER, VEC_TC_UINTEGER>>( \
|
||||
agg_ctx, agg_col_id, has_distinct, allocator, agg); \
|
||||
} else { \
|
||||
ret = init_agg_func<SysBitAggregate<op_type, VEC_TC_UINTEGER, VEC_TC_UINTEGER>>( \
|
||||
agg_ctx, agg_col_id, has_distinct, allocator, agg); \
|
||||
} \
|
||||
} while (false)
|
||||
|
||||
int ret = OB_SUCCESS;
|
||||
ObAggrInfo &aggr_info = agg_ctx.locate_aggr_info(agg_col_id);
|
||||
bool has_distinct = aggr_info.has_distinct_;
|
||||
if (OB_UNLIKELY(aggr_info.param_exprs_.count() != 1
|
||||
|| (aggr_info.param_exprs_.at(0)->get_vec_value_tc() != VEC_TC_UINTEGER
|
||||
&& aggr_info.param_exprs_.at(0)->get_vec_value_tc() != VEC_TC_INTEGER)
|
||||
&& aggr_info.param_exprs_.at(0)->get_vec_value_tc() != VEC_TC_BIT)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
LOG_WARN("unexpected param exprs", K(ret), K(aggr_info));
|
||||
} else if (OB_UNLIKELY(aggr_info.expr_->get_vec_value_tc() != VEC_TC_UINTEGER)) {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
LOG_WARN("unexpected result expr", K(ret), K(aggr_info));
|
||||
} else {
|
||||
VecValueTypeClass in_tc = aggr_info.param_exprs_.at(0)->get_vec_value_tc();
|
||||
ObExprOperatorType fn_type = aggr_info.get_expr_type();
|
||||
if (fn_type == T_FUN_SYS_BIT_OR) {
|
||||
INIT_SYS_BIT_AGG(T_FUN_SYS_BIT_OR);
|
||||
} else if (fn_type == T_FUN_SYS_BIT_AND) {
|
||||
INIT_SYS_BIT_AGG(T_FUN_SYS_BIT_AND);
|
||||
} else if (fn_type == T_FUN_SYS_BIT_XOR) {
|
||||
INIT_SYS_BIT_AGG(T_FUN_SYS_BIT_XOR);
|
||||
} else {
|
||||
ret = OB_ERR_UNEXPECTED;
|
||||
LOG_WARN("unexpected sysbit operator", K(ret), K(fn_type));
|
||||
}
|
||||
if (OB_FAIL(ret)) {
|
||||
LOG_WARN("init sysbit functions failed", K(ret));
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
}
|
||||
} // end aggregate
|
||||
} // end share
|
||||
} // end oceanbase
|
||||
116
src/share/aggregate/sys_bit.h
Normal file
116
src/share/aggregate/sys_bit.h
Normal file
@ -0,0 +1,116 @@
|
||||
/**
|
||||
* Copyright (c) 2021 OceanBase
|
||||
* OceanBase CE is licensed under Mulan PubL v2.
|
||||
* You can use this software according to the terms and conditions of the Mulan PubL v2.
|
||||
* You may obtain a copy of Mulan PubL v2 at:
|
||||
* http://license.coscl.org.cn/MulanPubL-2.0
|
||||
* THIS SOFTWARE IS PROVIDED ON AN "AS IS" BASIS, WITHOUT WARRANTIES OF ANY KIND,
|
||||
* EITHER EXPRESS OR IMPLIED, INCLUDING BUT NOT LIMITED TO NON-INFRINGEMENT,
|
||||
* MERCHANTABILITY OR FIT FOR A PARTICULAR PURPOSE.
|
||||
* See the Mulan PubL v2 for more details.
|
||||
*/
|
||||
#ifndef OCEANBASE_SHARE_SYSBIT_AGGREGATE_H_
|
||||
#define OCEANBASE_SHARE_SYSBIT_AGGREGATE_H_
|
||||
|
||||
#include "share/aggregate/iaggregate.h"
|
||||
|
||||
namespace oceanbase
|
||||
{
|
||||
namespace share
|
||||
{
|
||||
namespace aggregate
|
||||
{
|
||||
template <ObExprOperatorType agg_func, VecValueTypeClass in_tc, VecValueTypeClass out_tc>
|
||||
struct SysBitAggregate final
|
||||
: public BatchAggregateWrapper<SysBitAggregate<agg_func, in_tc, out_tc>>
|
||||
{
|
||||
public:
|
||||
static const constexpr VecValueTypeClass IN_TC = in_tc;
|
||||
static const constexpr VecValueTypeClass OUT_TC = out_tc;
|
||||
public:
|
||||
SysBitAggregate() {}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
inline int add_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
setup_inital_value(agg_ctx, agg_cell, agg_col_id);
|
||||
uint64_t &res_uint = *reinterpret_cast<uint64_t *>(agg_cell);
|
||||
uint64_t cur_uint = *reinterpret_cast<const uint64_t *>(columns.get_payload(row_num));
|
||||
if (agg_func == T_FUN_SYS_BIT_AND) {
|
||||
res_uint &= cur_uint;
|
||||
} else if (agg_func == T_FUN_SYS_BIT_OR) {
|
||||
res_uint |= cur_uint;
|
||||
} else if (agg_func == T_FUN_SYS_BIT_XOR) {
|
||||
res_uint ^= cur_uint;
|
||||
} else {
|
||||
// impossible
|
||||
ob_assert(false);
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
inline int add_nullable_row(RuntimeContext &agg_ctx, ColumnFmt &columns, const int32_t row_num,
|
||||
const int32_t agg_col_id, char *agg_cell, void *tmp_res, int64_t &calc_info)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
setup_inital_value(agg_ctx, agg_cell, agg_col_id);
|
||||
if (OB_LIKELY(!columns.is_null(row_num))) {
|
||||
ret = add_row(agg_ctx, columns, row_num, agg_col_id, agg_cell, tmp_res, calc_info);
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
template <typename ColumnFmt>
|
||||
int collect_group_result(RuntimeContext &agg_ctx, const sql::ObExpr &agg_expr,
|
||||
const int32_t agg_col_id, const char *agg_cell,
|
||||
const int32_t agg_cell_len)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
int64_t output_idx = agg_ctx.eval_ctx_.get_batch_idx();
|
||||
ColumnFmt *res_vec = static_cast<ColumnFmt *>(agg_expr.get_vector(agg_ctx.eval_ctx_));
|
||||
res_vec->set_payload(output_idx, agg_cell, sizeof(uint64_t));
|
||||
return ret;
|
||||
}
|
||||
|
||||
int add_one_row(RuntimeContext &agg_ctx, int64_t batch_idx, int64_t batch_size,
|
||||
const bool is_null, const char *data, const int32_t data_len, int32_t agg_col_idx,
|
||||
char *agg_cell) override
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
setup_inital_value(agg_ctx, agg_cell, agg_col_idx);
|
||||
if (!is_null) {
|
||||
uint64_t cur_uint = *reinterpret_cast<const uint64_t *>(data);
|
||||
uint64_t &res_uint = *reinterpret_cast<uint64_t *>(agg_cell);
|
||||
if (agg_func == T_FUN_SYS_BIT_AND) {
|
||||
res_uint &= cur_uint;
|
||||
} else if (agg_func == T_FUN_SYS_BIT_OR) {
|
||||
res_uint |= cur_uint;
|
||||
} else if (agg_func == T_FUN_SYS_BIT_XOR) {
|
||||
res_uint ^= cur_uint;
|
||||
} else {
|
||||
// impossible
|
||||
ob_assert(false);
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
TO_STRING_KV("aggregate", "sysbit_ops", K(in_tc), K(out_tc), K(agg_func));
|
||||
private:
|
||||
void setup_inital_value(RuntimeContext &agg_ctx, char *agg_cell, const int32_t agg_col_id)
|
||||
{
|
||||
NotNullBitVector ¬_nulls = agg_ctx.locate_notnulls_bitmap(agg_col_id, agg_cell);
|
||||
if (OB_UNLIKELY(!not_nulls.at(agg_col_id))) {
|
||||
uint64_t initial_val = (agg_func == T_FUN_SYS_BIT_AND ? UINT_MAX_VAL[ObUInt64Type] : 0);
|
||||
*reinterpret_cast<uint64_t *>(agg_cell) = initial_val;
|
||||
not_nulls.set(agg_col_id);
|
||||
}
|
||||
}
|
||||
};
|
||||
} // namespace aggregate
|
||||
} // namespace share
|
||||
} // namespace oceanbase
|
||||
|
||||
#endif // OCEANBASE_SHARE_SYSBIT_AGGREGATE_H_
|
||||
@ -21,6 +21,12 @@
|
||||
#include "sql/engine/ob_bit_vector.h"
|
||||
#include "share/vector/ob_vector_define.h"
|
||||
|
||||
#define EXTRACT_MEM_ADDR(ptr) (reinterpret_cast<char *>(*reinterpret_cast<int64_t *>((ptr))))
|
||||
#define STORE_MEM_ADDR(addr, dst) \
|
||||
do { \
|
||||
*reinterpret_cast<int64_t *>((dst)) = reinterpret_cast<int64_t>((addr)); \
|
||||
} while (false)
|
||||
|
||||
namespace oceanbase
|
||||
{
|
||||
namespace sql
|
||||
@ -115,6 +121,37 @@ struct add_param_batch<T, false>
|
||||
}
|
||||
};
|
||||
|
||||
template <typename T, typename = std::void_t<>>
|
||||
struct defined_removal_func: std::false_type {};
|
||||
|
||||
template <typename T>
|
||||
struct defined_removal_func<
|
||||
T, std::void_t<decltype(&T::template add_or_sub_row<ObVectorBase>)>>
|
||||
: std::true_type
|
||||
{};
|
||||
|
||||
template<typename T, bool defined = defined_removal_func<T>::value>
|
||||
struct removal_opt
|
||||
{
|
||||
template<typename... Args>
|
||||
inline static int add_or_sub_row(T &v, Args&&... args)
|
||||
{
|
||||
return v.add_or_sub_row(std::forward<Args>(args)...);
|
||||
}
|
||||
};
|
||||
|
||||
template<typename T>
|
||||
struct removal_opt<T, false>
|
||||
{
|
||||
template<typename... Args>
|
||||
inline static int add_or_sub_row(T &v, Args&&... args)
|
||||
{
|
||||
int ret = OB_NOT_SUPPORTED;
|
||||
SQL_LOG(WARN, "removal opt not suppported", K(ret));
|
||||
return ret;
|
||||
}
|
||||
};
|
||||
|
||||
// ================
|
||||
// read/writer helper
|
||||
|
||||
@ -409,6 +446,15 @@ inline int add_values(const int32_t &l, const number::ObCompactNumber &r, char *
|
||||
return ret;
|
||||
}
|
||||
|
||||
template<typename L, typename R>
|
||||
inline int sub_values(const L &l, const R &r, char *res_buf, const int32_t res_len)
|
||||
{
|
||||
int ret = OB_SUCCESS;
|
||||
L &res = *reinterpret_cast<L *>(res_buf) ;
|
||||
res = l - r;
|
||||
return ret;
|
||||
}
|
||||
|
||||
template<typename Input, typename Output>
|
||||
struct Caster
|
||||
{
|
||||
@ -555,7 +601,13 @@ inline bool supported_aggregate_function(const ObItemType agg_op)
|
||||
case T_FUN_MIN:
|
||||
case T_FUN_MAX:
|
||||
case T_FUN_COUNT_SUM:
|
||||
case T_FUN_SUM: {
|
||||
case T_FUN_SUM:
|
||||
case T_FUN_APPROX_COUNT_DISTINCT:
|
||||
case T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS:
|
||||
case T_FUN_APPROX_COUNT_DISTINCT_SYNOPSIS_MERGE:
|
||||
case T_FUN_SYS_BIT_OR:
|
||||
case T_FUN_SYS_BIT_AND:
|
||||
case T_FUN_SYS_BIT_XOR: {
|
||||
return true;
|
||||
}
|
||||
default:
|
||||
@ -563,6 +615,23 @@ inline bool supported_aggregate_function(const ObItemType agg_op)
|
||||
}
|
||||
}
|
||||
|
||||
inline bool agg_res_not_null(const ObItemType agg_op)
|
||||
{
|
||||
// TODO: add other functions
|
||||
bool ret = false;
|
||||
switch (agg_op) {
|
||||
case T_FUN_COUNT:
|
||||
case T_FUN_COUNT_SUM: {
|
||||
ret = true;
|
||||
break;
|
||||
}
|
||||
default: {
|
||||
break;
|
||||
}
|
||||
}
|
||||
return ret;
|
||||
}
|
||||
|
||||
#define AGG_FIXED_TC_LIST \
|
||||
VEC_TC_INTEGER, \
|
||||
VEC_TC_UINTEGER, \
|
||||
@ -598,6 +667,7 @@ inline bool supported_aggregate_function(const ObItemType agg_op)
|
||||
VEC_TC_TIME, \
|
||||
VEC_TC_YEAR, \
|
||||
VEC_TC_STRING, \
|
||||
VEC_TC_EXTEND, \
|
||||
VEC_TC_BIT, \
|
||||
VEC_TC_ENUM_SET, \
|
||||
VEC_TC_ENUM_SET_INNER, \
|
||||
|
||||
Reference in New Issue
Block a user