提交 6a1c0792 编写于 作者: K KyrielightWei 提交者: ob-robot

check dup tablet after acquire max_commit_ts

上级 eb33a615
......@@ -697,7 +697,7 @@ int ObSqlTransControl::stmt_setup_snapshot_(ObSQLSessionInfo *session,
stmt_expire_ts,
snapshot))) {
} else {
local_single_ls_plan = has_same_lsid(das_ctx, snapshot.core_.version_, first_ls_id);
local_single_ls_plan = has_same_lsid(das_ctx, snapshot, first_ls_id);
}
}
if (OB_SUCC(ret) && !local_single_ls_plan) {
......@@ -798,12 +798,13 @@ int ObSqlTransControl::get_first_lsid(const ObDASCtx &das_ctx, share::ObLSID &fi
}
bool ObSqlTransControl::has_same_lsid(const ObDASCtx &das_ctx,
const share::SCN &snapshot_version,
const transaction::ObTxReadSnapshot &snapshot,
share::ObLSID &first_lsid)
{
int ret = OB_SUCCESS;
bool bret = true;
ObLSHandle ls_handle;
const share::SCN snapshot_version = snapshot.core_.version_;
const DASTableLocList &table_locs = das_ctx.get_table_loc_list();
FOREACH_X(table_node, table_locs, bret) {
ObDASTableLoc *table_loc = *table_node;
......@@ -847,6 +848,17 @@ bool ObSqlTransControl::has_same_lsid(const ObDASCtx &das_ctx,
}
}
}
if (bret && common::ObRole::FOLLOWER == snapshot.snapshot_ls_role_) {
ObLS *ls = NULL;
if (OB_ISNULL(ls = ls_handle.get_ls())) {
bret = false;
LOG_WARN("invalid ls", K(bret), K(first_lsid), K(snapshot));
} else if (!(ls->get_dup_table_ls_handler()->is_dup_tablet(tablet_id))) {
bret = false;
LOG_WARN("There is a normal tablet, retry to acquire snapshot with gts", K(bret), K(first_lsid),
K(snapshot), K(tablet_loc));
}
}
}
}
return bret;
......
......@@ -257,7 +257,7 @@ private:
static uint32_t get_real_session_id(ObSQLSessionInfo &session);
static int get_first_lsid(const ObDASCtx &das_ctx, share::ObLSID &first_lsid);
static bool has_same_lsid(const ObDASCtx &das_ctx,
const share::SCN &snapshot_version,
const transaction::ObTxReadSnapshot &snapshot,
share::ObLSID &first_lsid);
public:
/*
......
......@@ -573,6 +573,16 @@ int ObDupTableLSLeaseMgr::leader_revoke()
return ret;
}
bool ObDupTableLSLeaseMgr::is_follower_lease_valid()
{
bool is_follower_lease = false;
SpinRLockGuard guard(lease_lock_);
is_follower_lease = follower_lease_info_.lease_expired_ts_ > ObTimeUtility::current_time();
return is_follower_lease;
}
bool ObDupTableLSLeaseMgr::check_follower_lease_serving(const bool is_election_leader,
const share::SCN &max_replayed_scn)
{
......
......@@ -89,6 +89,8 @@ public:
int get_lease_valid_array(LeaseAddrArray &lease_array);
bool is_follower_lease_valid();
bool check_follower_lease_serving(const bool election_is_leader,
const share::SCN &max_replayed_scn);
......
......@@ -602,12 +602,13 @@ public:
bool &readable,
const share::SCN &snapshot,
DupTableInterfaceStat interface_stat);
int search_dup_tablet_for_read(const common::ObTabletID &tablet_id, bool &is_dup_table);
// For part_ctx, check_dup_table will be invoked after submit_log in LS which has dup_table
// tablets. It will bring performance effect for normal part_ctx without dup_table tablets.
int find_dup_tablet_in_set(const common::ObTabletID &tablet_id,
bool &is_dup_table,
const share::SCN &from_scn,
const share::SCN &to_scn);
int search_dup_tablet_in_redo_log(const common::ObTabletID &tablet_id,
bool &is_dup_table,
const share::SCN &from_scn,
const share::SCN &to_scn);
int gc_dup_tablets(const int64_t gc_ts, const int64_t max_task_interval);
// new gc methods
int scan_readable_set_for_gc();
......
......@@ -1058,8 +1058,8 @@ int ObDupTableLSHandler::check_dup_tablet_in_redo(const ObTabletID &tablet_id,
is_dup_tablet = false;
} else if (!has_dup_tablet()) {
is_dup_tablet = false;
} else if (OB_FAIL(tablets_mgr_ptr_->find_dup_tablet_in_set(tablet_id, is_dup_tablet,
base_snapshot, redo_scn))) {
} else if (OB_FAIL(tablets_mgr_ptr_->search_dup_tablet_in_redo_log(tablet_id, is_dup_tablet,
base_snapshot, redo_scn))) {
DUP_TABLE_LOG(WARN, "check dup tablet failed", K(ret), K(tablet_id), K(base_snapshot),
K(redo_scn));
}
......@@ -1120,6 +1120,28 @@ int ObDupTableLSHandler::check_dup_tablet_readable(const ObTabletID &tablet_id,
return ret;
}
bool ObDupTableLSHandler::is_dup_table_lease_valid()
{
bool is_dup_lease_ls = false;
const bool is_election_leader = false;
if (has_dup_tablet()) {
if (OB_ISNULL(lease_mgr_ptr_)) {
is_dup_lease_ls = false;
} else if (ls_state_helper_.is_leader()) {
is_dup_lease_ls = true;
DUP_TABLE_LOG(INFO, "the lease is always valid for a dup ls leader", K(is_dup_lease_ls),
KPC(this));
} else {
is_dup_lease_ls = lease_mgr_ptr_->is_follower_lease_valid();
}
} else {
is_dup_lease_ls = false;
}
return is_dup_lease_ls;
}
int64_t ObDupTableLSHandler::get_dup_tablet_count()
{
int64_t dup_tablet_cnt = 0;
......@@ -1144,6 +1166,24 @@ bool ObDupTableLSHandler::has_dup_tablet()
return has_dup;
}
bool ObDupTableLSHandler::is_dup_tablet(const common::ObTabletID &tablet_id)
{
bool is_dup_tablet = false;
int ret = OB_SUCCESS;
if (!tablet_id.is_valid()) {
ret = OB_INVALID_ARGUMENT;
DUP_TABLE_LOG(WARN, "invalid argument", K(ret), K(tablet_id));
} else if (OB_ISNULL(tablets_mgr_ptr_)) {
is_dup_tablet = false;
} else if (OB_FAIL(tablets_mgr_ptr_->search_dup_tablet_for_read(tablet_id, is_dup_tablet))) {
DUP_TABLE_LOG(WARN, "check dup tablet failed", K(ret), K(tablet_id), K(is_dup_tablet));
is_dup_tablet = false;
}
return is_dup_tablet;
}
// if return false, there are no tablets and tablet set need log
bool ObDupTableLSHandler::check_tablet_set_exist()
{
......
......@@ -177,10 +177,13 @@ public:
const share::SCN &max_replayed_scn,
bool &readable);
bool is_dup_table_lease_valid();
public:
int64_t get_dup_tablet_count();
bool check_tablet_set_exist();
bool has_dup_tablet();
bool is_dup_tablet(const common::ObTabletID &tablet_id);
int gc_dup_tablets(const int64_t gc_ts, const int64_t max_task_interval);
int get_local_ts_info(DupTableTsInfo &ts_info);
int get_cache_ts_info(const common::ObAddr &addr, DupTableTsInfo &ts_info);
......
......@@ -31,6 +31,7 @@
#include "wrs/ob_weak_read_util.h" // ObWeakReadUtil
#include "storage/memtable/ob_memtable_context.h"
#include "storage/memtable/ob_memtable.h"
#include "storage/tx_storage/ob_ls_service.h"
#include "common/storage/ob_sequence.h"
#include "observer/ob_srv_network_frame.h"
#include "share/rc/ob_tenant_module_init_ctx.h"
......@@ -613,6 +614,38 @@ int ObTransService::get_max_decided_scn(const share::ObLSID &ls_id, share::SCN &
return ret;
}
int ObTransService::check_dup_table_lease_valid(const ObLSID ls_id,
bool &is_dup_ls,
bool &is_lease_valid)
{
int ret = OB_SUCCESS;
ObLSHandle ls_handle;
is_dup_ls = false;
is_lease_valid = false;
if (!ls_id.is_valid()) {
ret = OB_INVALID_ARGUMENT;
TRANS_LOG(WARN, "invalid argument", K(ret), K(ls_id));
} else if (!dup_table_loop_worker_.is_useful_dup_ls(ls_id)) {
is_dup_ls = false;
ret = OB_SUCCESS;
} else if (OB_FAIL(MTL(ObLSService *)->get_ls(ls_id, ls_handle, ObLSGetMod::TRANS_MOD))) {
is_dup_ls = false;
TRANS_LOG(WARN, "get ls failed", K(ret), K(ls_id), K(ls_handle));
} else if (!ls_handle.is_valid()) {
is_dup_ls = false;
ret = OB_ERR_UNEXPECTED;
TRANS_LOG(WARN, "invalid ls handle", K(ret), K(ls_id), K(ls_handle));
} else if (ls_handle.get_ls()->get_dup_table_ls_handler()->is_dup_table_lease_valid()) {
is_dup_ls = true;
is_lease_valid = true;
ret = OB_SUCCESS;
}
return ret;
}
int ObTransService::handle_redo_sync_task_(ObDupTableRedoSyncTask *task, bool &need_release_task)
{
UNUSED(task);
......
......@@ -174,6 +174,7 @@ public:
int calculate_trans_cost(const ObTransID &tid, uint64_t &cost);
int get_ls_min_uncommit_prepare_version(const share::ObLSID &ls_id, share::SCN &min_prepare_version);
int get_min_undecided_log_ts(const share::ObLSID &ls_id, share::SCN &log_ts);
int check_dup_table_lease_valid(const ObLSID ls_id, bool &is_dup_ls, bool &is_lease_valid);
//get the memory used condition of transaction module
int iterate_trans_memory_stat(ObTransMemStatIterator &mem_stat_iter);
int dump_elr_statistic();
......
......@@ -1453,40 +1453,6 @@ int ObTransService::check_replica_readable_(const ObTxReadSnapshot &snapshot,
}
}
if (OB_SUCC(ret) && !dup_table_readable && ObTxReadSnapshot::SRC::LS == src
&& ObRole::FOLLOWER == snapshot.snapshot_ls_role_) {
int tmp_ret = OB_SUCCESS;
if (readable) {
if (OB_TMP_FAIL(ls.get_tx_svr()->get_tx_ls_log_adapter()->get_role(leader, epoch))) {
TRANS_LOG(WARN, "get replica status fail", K(tmp_ret), K(ls_id));
} else if (leader) {
tmp_ret = OB_SUCCESS;
} else if (OB_TMP_FAIL(ls.get_max_decided_scn(max_replayed_scn))) {
TRANS_LOG(WARN, "get max decided scn failed", K(ret), K(tmp_ret), K(ls_id));
// rewrite ret code when get max decided scn failed
tmp_ret = OB_NOT_MASTER;
} else if (OB_TMP_FAIL(ls.get_tx_svr()->get_tx_ls_log_adapter()->check_dup_tablet_readable(
tablet_id, snapshot.core_.version_, leader, max_replayed_scn,
dup_table_readable))) {
TRANS_LOG(WARN, "check dup tablet readable error", K(ret), K(tmp_ret));
} else if (dup_table_readable) {
TRANS_LOG(INFO, "the dup tablet is readable with the max_commit_ts from a follower", K(ret),
K(tmp_ret), K(tablet_id), K(snapshot), K(leader), K(max_replayed_scn),
K(dup_table_readable), K(ls_id), K(expire_ts));
tmp_ret = OB_SUCCESS;
}
}
if (tmp_ret == OB_SUCCESS && dup_table_readable) {
// do nothing
} else {
ret = OB_NOT_MASTER;
TRANS_LOG(WARN, "we can not use a max_commit_ts from the follower", K(ret), K(tablet_id),
K(snapshot), K(leader), K(max_replayed_scn), K(dup_table_readable), K(ls_id),
K(expire_ts), K(MTL_IS_PRIMARY_TENANT()));
}
}
TRANS_LOG(TRACE, "check replica readable", K(ret), K(snapshot), K(ls_id));
return ret;
}
......@@ -1641,9 +1607,9 @@ int ObTransService::acquire_local_snapshot_(const share::ObLSID &ls_id,
ls_tx_ctx_mgr->get_ls_log_adapter()->get_committing_dup_trx_cnt();
if (!MTL_IS_PRIMARY_TENANT()) {
ret = OB_NOT_MASTER;
TRANS_LOG(WARN, "the max_commmit_ts can not be used as a snapshot in standby tenant ", K(ret),
TRANS_LOG(DEBUG, "the max_commmit_ts can not be used as a snapshot in standby tenant ", K(ret),
K(ls_id), K(snapshot), K(MTL_IS_PRIMARY_TENANT()), K(committing_dup_trx_cnt));
} else if (!MTL(ObTransService*)->get_dup_table_loop_worker().is_useful_dup_ls(ls_id)) {
} else if (!ls_tx_ctx_mgr->get_ls_log_adapter()->is_dup_table_lease_valid()) {
ret = OB_NOT_MASTER;
} else if (committing_dup_trx_cnt > 0) {
ret = OB_NOT_MASTER;
......@@ -1676,7 +1642,7 @@ int ObTransService::acquire_local_snapshot_(const share::ObLSID &ls_id,
// | count == 0
// v
// +----------------+ false +----------------------------------------------------------------+
// | Not Master | <----------- | check_replica_readable : read a dup table tablet on a follower |
// | Not Master | <----------- | check all tablet loc: all tablet is dup table tablet |
// +----------------+ +----------------------------------------------------------------+
// |
// | true
......
......@@ -253,6 +253,19 @@ int ObLSTxLogAdapter::check_redo_sync_completed(const ObTransID &tx_id,
return ret;
}
bool ObLSTxLogAdapter::is_dup_table_lease_valid()
{
bool is_follower_lease = false;
if (OB_ISNULL(dup_table_ls_handler_)) {
DUP_TABLE_LOG_RET(WARN, OB_ERR_UNEXPECTED, "invalid dup table ls handler");
} else {
is_follower_lease = dup_table_ls_handler_->is_dup_table_lease_valid();
}
return is_follower_lease;
}
bool ObLSTxLogAdapter::has_dup_tablet()
{
bool has_dup = false;
......
......@@ -86,6 +86,7 @@ public:
const share::SCN &redo_completed_scn,
bool &redo_sync_finish,
share::SCN &total_max_read_version);
virtual bool is_dup_table_lease_valid() { return false; }
virtual bool has_dup_tablet() { return false; }
virtual int64_t get_committing_dup_trx_cnt();
virtual int add_commiting_dup_trx(const ObTransID &tx_id);
......@@ -127,6 +128,7 @@ public:
const share::SCN &redo_completed_scn,
bool &redo_sync_finish,
share::SCN &total_max_read_version);
bool is_dup_table_lease_valid();
bool has_dup_tablet();
int64_t get_committing_dup_trx_cnt();
int add_commiting_dup_trx(const ObTransID &tx_id);
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册