Skip to content

Commit

Permalink
[BACKPORT 2.18.5][#20359] YSQL: Limit query layer retries for Read Co…
Browse files Browse the repository at this point in the history
…mmitted isolation

Summary:
Original commit: cf11481 / D31327

As of now, we have two TServer gflags `ysql_max_read_restart_attempts`
and `ysql_max_write_restart_attempts` to control the number of query layer
retries for the first statement of a transaction in Repeatable Read and
Serializable isolation levels. In Read Committed isolation, the query
layer has indefinite retries for any statement in the transaction.

This diff deprecates the two gflags and unifies them into a single GUC
variable called `yb_max_query_layer_retries` which has a default of 60.
If any database cluster on an older version uses either of those gflags,
they will not be honored. This diff also applies the new GUC configurable
limit to per-statement retries in Read Committed isolation.

Moreover, the default for `retry_backoff_multiplier` is changed to 1.2
and the default for `retry_min_backoff` is changed to 10ms.
Jira: DB-9346

Test Plan: ./yb_build.sh release --java-test org.yb.pgsql.TestPgTransparentRestarts

Reviewers: tvesely, bkolagani, dmitry, smishra

Reviewed By: bkolagani, smishra

Subscribers: aaruj, dmitry, yql

Tags: #jenkins-ready

Differential Revision: https://phorge.dev.yugabyte.com/D31407
  • Loading branch information
pkj415 committed Jan 3, 2024
1 parent f0bc7c5 commit 18fca47
Show file tree
Hide file tree
Showing 21 changed files with 72 additions and 59 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -125,7 +125,7 @@ protected Map<String, String> getTServerFlags() {

flagMap.put("ysql_beta_features", "true");
flagMap.put("ysql_sleep_before_retry_on_txn_conflict", "false");
flagMap.put("ysql_max_write_restart_attempts", "2");
flagMap.put("ysql_pg_conf_csv", maxQueryLayerRetriesConf(2));
flagMap.put("ysql_enable_packed_row", "false");

return flagMap;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -458,4 +458,7 @@ public static void tearDownAfterClass() throws Exception {
LOG.info("BaseMiniClusterTest.tearDownAfterClass completed");
}

protected String maxQueryLayerRetriesConf(int max_retries) {
return "yb_max_query_layer_retries=" + max_retries;
}
}
2 changes: 0 additions & 2 deletions java/yb-pgsql/src/test/java/org/yb/pgsql/BasePgSQLTest.java
Original file line number Diff line number Diff line change
Expand Up @@ -254,8 +254,6 @@ protected Map<String, String> getTServerFlags() {
}

flagMap.put("ysql_beta_features", "true");
flagMap.put("ysql_sleep_before_retry_on_txn_conflict", "false");
flagMap.put("ysql_max_write_restart_attempts", "2");
flagMap.put("ysql_enable_reindex", "true");

return flagMap;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -435,8 +435,8 @@ public void testNoWait() throws Exception {
assertTrue("Should not reach here since the statement is supposed to fail", false);
} catch (SQLException e) {
// If txn2 had a lower priority than txn1, instead of attempting retries for
// ysql_max_write_restart_attempts, it would fail immediately due to the NOWAIT clause
// with the appropriate message.
// yb_max_query_layer_retries, it would fail immediately due to the NOWAIT clause with the
// appropriate message.
assertTrue(StringUtils.containsIgnoreCase(e.getMessage(),
"ERROR: could not obtain lock on row in relation \"test\""));

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -25,12 +25,20 @@
import java.sql.ResultSet;
import java.sql.Statement;
import java.util.HashSet;
import java.util.Map;
import java.util.Set;

@RunWith(value=YBTestRunner.class)
public class TestPgForeignKey extends BasePgSQLTest {
private static final Logger LOG = LoggerFactory.getLogger(TestPgForeignKey.class);

@Override
protected Map<String, String> getTServerFlags() {
Map<String, String> flagMap = super.getTServerFlags();
flagMap.put("ysql_pg_conf_csv", maxQueryLayerRetriesConf(2));
return flagMap;
}

@Override
public int getTestMethodTimeoutSec() {
return 1800;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -41,6 +41,7 @@
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.Map;
import java.util.Random;

import static org.yb.AssertionWrappers.*;
Expand Down Expand Up @@ -69,6 +70,13 @@ private void checkTransactionFairness(
assertLessThan("Second Win Too Low", totalIterations / 4, numSecondWinners);
}

@Override
protected Map<String, String> getTServerFlags() {
Map<String, String> flags = super.getTServerFlags();
flags.put("ysql_pg_conf_csv", maxQueryLayerRetriesConf(2));
return flags;
}

@Override
protected void customizeMiniClusterBuilder(MiniYBClusterBuilder builder) {
super.customizeMiniClusterBuilder(builder);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -124,8 +124,6 @@ public class TestPgTransparentRestarts extends BasePgSQLTest {
protected Map<String, String> getTServerFlags() {
Map<String, String> flags = super.getTServerFlags();
flags.put("ysql_output_buffer_size", String.valueOf(PG_OUTPUT_BUFFER_SIZE_BYTES));
flags.put("ysql_sleep_before_retry_on_txn_conflict", "true");
flags.put("ysql_max_write_restart_attempts", "5");
flags.put("yb_enable_read_committed_isolation", "true");
return flags;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,9 +26,7 @@ public class TestPgWithoutWaitQueuesIsolationRegress extends BasePgSQLTest {
protected Map<String, String> getTServerFlags() {
Map<String, String> flagMap = super.getTServerFlags();
flagMap.put("enable_wait_queues", "false");
flagMap.put("ysql_sleep_before_retry_on_txn_conflict", "true");
flagMap.put("yb_enable_read_committed_isolation", "true");
flagMap.put("ysql_max_write_restart_attempts", "20");
return flagMap;
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -43,8 +43,7 @@ public class TestPgWriteRestart extends BasePgSQLTest {
@Override
protected Map<String, String> getTServerFlags() {
Map<String, String> flags = super.getTServerFlags();
flags.put("ysql_sleep_before_retry_on_txn_conflict", "true");
flags.put("ysql_max_write_restart_attempts", "20");
flags.put("ysql_pg_conf_csv", maxQueryLayerRetriesConf(20));
return flags;
}

Expand Down
1 change: 1 addition & 0 deletions src/postgres/src/backend/storage/lmgr/proc.c
Original file line number Diff line number Diff line change
Expand Up @@ -66,6 +66,7 @@ bool log_lock_waits = false;
int RetryMaxBackoffMsecs;
int RetryMinBackoffMsecs;
double RetryBackoffMultiplier;
int yb_max_query_layer_retries;

/* Pointer to this process's PGPROC and PGXACT structs, if any */
PGPROC *MyProc = NULL;
Expand Down
38 changes: 10 additions & 28 deletions src/postgres/src/backend/tcop/postgres.c
Original file line number Diff line number Diff line change
Expand Up @@ -4107,46 +4107,28 @@ yb_is_restart_possible(const ErrorData* edata,
elog(DEBUG1, "Error details: edata->message=%s edata->filename=%s edata->lineno=%d",
edata->message, edata->filename, edata->lineno);
bool is_read_restart_error = YBCIsRestartReadError(edata->yb_txn_errcode);
bool is_conflict_error = YBCIsTxnConflictError(edata->yb_txn_errcode);
if (!is_read_restart_error && !is_conflict_error)
{
if (yb_debug_log_internal_restarts)
elog(LOG, "Restart isn't possible, code %d isn't a read restart/conflict error",
edata->yb_txn_errcode);
return false;
}
bool is_conflict_error = YBCIsTxnConflictError(edata->yb_txn_errcode);

/*
* In case of READ COMMITTED, retries for kConflict are performed indefinitely until statement
* timeout is hit.
*/
if (!IsYBReadCommitted() &&
(is_conflict_error && attempt >= *YBCGetGFlags()->ysql_max_write_restart_attempts))
if (!is_read_restart_error && !is_conflict_error)
{
if (yb_debug_log_internal_restarts)
elog(LOG, "Restart isn't possible, we're out of write restart attempts (%d)",
attempt);
*retries_exhausted = true;
elog(
LOG, "Restart isn't possible, code %d isn't a read restart/conflict error",
edata->yb_txn_errcode);
return false;
}

/*
* Retries for kReadRestart are performed indefinitely in case the true READ COMMITTED isolation
* level implementation is used.
*/
if (!IsYBReadCommitted() &&
(is_read_restart_error && attempt >= *YBCGetGFlags()->ysql_max_read_restart_attempts))
if (attempt >= yb_max_query_layer_retries)
{
if (yb_debug_log_internal_restarts)
elog(LOG, "Restart isn't possible, we're out of read restart attempts (%d)",
attempt);
elog(LOG, "Query layer is out of retries, retry limit is %d", yb_max_query_layer_retries);
*retries_exhausted = true;
return false;
}

// We can perform kReadRestart retries in READ COMMITTED isolation level even if data has been
// sent as part of the txn, but not as part of the current query. This is because we just have to
// retry the query and not the whole transaction.
// We can perform retries in READ COMMITTED isolation level even if data has been sent as part of
// the txn, but not as part of the current query. This is because we just have to retry the query
// and not the whole transaction in RC.
if ((!IsYBReadCommitted() && YBIsDataSent()) ||
(IsYBReadCommitted() && YBIsDataSentForCurrQuery()))
{
Expand Down
22 changes: 20 additions & 2 deletions src/postgres/src/backend/utils/misc/guc.c
Original file line number Diff line number Diff line change
Expand Up @@ -2676,7 +2676,7 @@ static struct config_int ConfigureNamesInt[] =
GUC_UNIT_MS
},
&RetryMinBackoffMsecs,
100, 0, INT_MAX,
10, 0, INT_MAX,
check_min_backoff, NULL, NULL
},

Expand Down Expand Up @@ -3579,6 +3579,24 @@ static struct config_int ConfigureNamesInt[] =
NULL, NULL, NULL
},

{
{"yb_max_query_layer_retries", PGC_USERSET, CLIENT_CONN_STATEMENT,
gettext_noop("Max number of internal query layer retries of a statement"),
gettext_noop("Max number of query layer retries of a statement for the following errors: "
"serialization error (40001), \"Restart read required\" (40001), "
"deadlock detected (40P01). In Repeatable Read and Serializable isolation levels, the "
"query layer only retries errors faced in the first statement of a transation. In "
"READ COMMITTED isolation, the query layer has the ability to do retries for any "
"statement in a transaction. Retries are not possible if some response data has "
"already been sent to the client while the query is still executing. This happens if "
"the output buffer, the size of which is configurable using the TServer gflag "
"ysql_output_buffer_size, has filled atleast once and is flushed."),
},
&yb_max_query_layer_retries,
60, 0, INT_MAX,
NULL, NULL, NULL
},

/* End-of-list marker */
{
{NULL, 0, 0, NULL, NULL}, NULL, 0, 0, 0, NULL, NULL, NULL
Expand Down Expand Up @@ -3824,7 +3842,7 @@ static struct config_real ConfigureNamesReal[] =
GUC_UNIT_MS
},
&RetryBackoffMultiplier,
2.0, 1.0, 1e10,
1.2, 1.0, 1e10,
check_backoff_multiplier, NULL, NULL
},

Expand Down
1 change: 1 addition & 0 deletions src/postgres/src/include/storage/proc.h
Original file line number Diff line number Diff line change
Expand Up @@ -335,6 +335,7 @@ extern bool log_lock_waits;
extern int RetryMaxBackoffMsecs;
extern int RetryMinBackoffMsecs;
extern double RetryBackoffMultiplier;
extern int yb_max_query_layer_retries;

/* Metrics */
extern int *yb_too_many_conn;
Expand Down
10 changes: 4 additions & 6 deletions src/yb/yql/pggate/pggate_flags.cc
Original file line number Diff line number Diff line change
Expand Up @@ -55,9 +55,6 @@ DEFINE_UNKNOWN_uint64(ysql_session_max_batch_size, 3072,
DEFINE_UNKNOWN_bool(ysql_non_txn_copy, false,
"Execute COPY inserts non-transactionally.");

DEFINE_UNKNOWN_int32(ysql_max_read_restart_attempts, 20,
"How many read restarts can we try transparently before giving up");

DEFINE_test_flag(bool, ysql_disable_transparent_cache_refresh_retry, false,
"Never transparently retry commands that fail with cache version mismatch error");

Expand Down Expand Up @@ -111,12 +108,13 @@ DEFINE_UNKNOWN_int32(ysql_select_parallelism, -1,
"Number of read requests to issue in parallel to tablets of a table "
"for SELECT.");

DEFINE_UNKNOWN_int32(ysql_max_write_restart_attempts, 20,
"Max number of restart attempts made for writes on transaction conflicts.");

DEFINE_UNKNOWN_bool(ysql_sleep_before_retry_on_txn_conflict, true,
"Whether to sleep before retrying the write on transaction conflicts.");

DEPRECATE_FLAG(int32, ysql_max_read_restart_attempts, "12_2023");

DEPRECATE_FLAG(int32, ysql_max_write_restart_attempts, "12_2023");

// Flag for disabling runContext to Postgres's portal. Currently, each portal has two contexts.
// - PortalContext whose lifetime lasts for as long as the Portal object.
// - TmpContext whose lifetime lasts until one associated row of SELECT result set is sent out.
Expand Down
2 changes: 0 additions & 2 deletions src/yb/yql/pggate/pggate_flags.h
Original file line number Diff line number Diff line change
Expand Up @@ -25,7 +25,6 @@ DECLARE_uint64(ysql_prefetch_limit);
DECLARE_double(ysql_backward_prefetch_scale_factor);
DECLARE_uint64(ysql_session_max_batch_size);
DECLARE_bool(ysql_non_txn_copy);
DECLARE_int32(ysql_max_read_restart_attempts);
DECLARE_bool(TEST_ysql_disable_transparent_cache_refresh_retry);
DECLARE_int64(TEST_inject_delay_between_prepare_ybctid_execute_batch_ybctid_ms);
DECLARE_bool(TEST_index_read_multiple_partitions);
Expand All @@ -41,7 +40,6 @@ DECLARE_bool(ysql_beta_feature_tablegroup);
DECLARE_bool(ysql_colocate_database_by_default);
DECLARE_bool(ysql_beta_feature_tablespace_alteration);
DECLARE_bool(ysql_serializable_isolation_for_ddl_txn);
DECLARE_int32(ysql_max_write_restart_attempts);
DECLARE_bool(ysql_sleep_before_retry_on_txn_conflict);
DECLARE_bool(ysql_disable_portal_run_context);
DECLARE_bool(TEST_yb_lwlock_crash_after_acquire_pg_stat_statements_reset);
Expand Down
2 changes: 0 additions & 2 deletions src/yb/yql/pggate/ybc_pg_typedefs.h
Original file line number Diff line number Diff line change
Expand Up @@ -358,8 +358,6 @@ typedef struct PgGFlagsAccessor {
const bool* ysql_disable_index_backfill;
const bool* ysql_disable_server_file_access;
const bool* ysql_enable_reindex;
const int32_t* ysql_max_read_restart_attempts;
const int32_t* ysql_max_write_restart_attempts;
const int32_t* ysql_num_databases_reserved_in_db_catalog_version_mode;
const int32_t* ysql_output_buffer_size;
const int32_t* ysql_sequence_cache_minval;
Expand Down
2 changes: 0 additions & 2 deletions src/yb/yql/pggate/ybc_pggate.cc
Original file line number Diff line number Diff line change
Expand Up @@ -1416,8 +1416,6 @@ const YBCPgGFlagsAccessor* YBCGetGFlags() {
.ysql_disable_index_backfill = &FLAGS_ysql_disable_index_backfill,
.ysql_disable_server_file_access = &FLAGS_ysql_disable_server_file_access,
.ysql_enable_reindex = &FLAGS_ysql_enable_reindex,
.ysql_max_read_restart_attempts = &FLAGS_ysql_max_read_restart_attempts,
.ysql_max_write_restart_attempts = &FLAGS_ysql_max_write_restart_attempts,
.ysql_num_databases_reserved_in_db_catalog_version_mode =
&FLAGS_ysql_num_databases_reserved_in_db_catalog_version_mode,
.ysql_output_buffer_size = &FLAGS_ysql_output_buffer_size,
Expand Down
4 changes: 4 additions & 0 deletions src/yb/yql/pgwrapper/libpq_utils.cc
Original file line number Diff line number Diff line change
Expand Up @@ -699,6 +699,10 @@ bool IsRetryable(const Status& status) {
return HasSubstring(status.message(), kExpectedErrors);
}

std::string MaxQueryLayerRetriesConf(uint16_t max_retries) {
return Format("yb_max_query_layer_retries=$0", max_retries);
}

PGConnBuilder::PGConnBuilder(const PGConnSettings& settings)
: conn_str_(BuildConnectionString(settings)),
conn_str_for_log_(BuildConnectionString(settings, true /* mask_password */)),
Expand Down
1 change: 1 addition & 0 deletions src/yb/yql/pgwrapper/libpq_utils.h
Original file line number Diff line number Diff line change
Expand Up @@ -239,6 +239,7 @@ class PGConnBuilder {

bool HasTransactionError(const Status& status);
bool IsRetryable(const Status& status);
std::string MaxQueryLayerRetriesConf(uint16_t max_retries);

Result<PGConn> Execute(Result<PGConn> connection, const std::string& query);

Expand Down
6 changes: 4 additions & 2 deletions src/yb/yql/pgwrapper/pg_tablet_split-test.cc
Original file line number Diff line number Diff line change
Expand Up @@ -58,14 +58,16 @@
#include "yb/util/tsan_util.h"

#include "yb/yql/pggate/ybc_pg_typedefs.h"
#include "yb/yql/pgwrapper/libpq_utils.h"
#include "yb/yql/pgwrapper/pg_tablet_split_test_base.h"


DECLARE_int32(cleanup_split_tablets_interval_sec);
DECLARE_bool(enable_automatic_tablet_splitting);
DECLARE_bool(enable_wait_queues);
DECLARE_int32(ysql_client_read_write_timeout_ms);
DECLARE_int32(ysql_max_write_restart_attempts);
DECLARE_bool(ysql_enable_packed_row);
DECLARE_string(ysql_pg_conf_csv);

DECLARE_bool(TEST_skip_deleting_split_tablets);
DECLARE_int32(TEST_fetch_next_delay_ms);
Expand Down Expand Up @@ -1203,7 +1205,7 @@ class PgPartitioningWaitQueuesOffTest : public PgPartitioningTest {
// request times out.
ANNOTATE_UNPROTECTED_WRITE(FLAGS_enable_wait_queues) = false;
// Fail txn early in case of conflict to reduce test runtime.
ANNOTATE_UNPROTECTED_WRITE(FLAGS_ysql_max_write_restart_attempts) = 0;
ANNOTATE_UNPROTECTED_WRITE(FLAGS_ysql_pg_conf_csv) = MaxQueryLayerRetriesConf(0);
PgPartitioningTest::SetUp();
}
};
Expand Down
8 changes: 4 additions & 4 deletions src/yb/yql/pgwrapper/pg_wait_on_conflict-test.cc
Original file line number Diff line number Diff line change
Expand Up @@ -51,11 +51,11 @@ DECLARE_uint64(force_single_shard_waiter_retry_ms);
DECLARE_uint64(rpc_connection_timeout_ms);
DECLARE_uint64(transactions_status_poll_interval_ms);
DECLARE_int32(TEST_sleep_amidst_iterating_blockers_ms);
DECLARE_int32(ysql_max_write_restart_attempts);
DECLARE_uint64(refresh_waiter_timeout_ms);
DECLARE_bool(ysql_enable_packed_row);
DECLARE_bool(ysql_enable_pack_full_row_update);
DECLARE_bool(TEST_drop_participant_signal);
DECLARE_string(ysql_pg_conf_csv);

using namespace std::literals;

Expand All @@ -73,7 +73,7 @@ class PgWaitQueuesTest : public PgMiniTestBase {
FLAGS_enable_deadlock_detection = true;
FLAGS_TEST_select_all_status_tablets = true;
FLAGS_force_single_shard_waiter_retry_ms = 10000;
FLAGS_ysql_max_write_restart_attempts = 0;
FLAGS_ysql_pg_conf_csv = MaxQueryLayerRetriesConf(0);
PgMiniTestBase::SetUp();
}

Expand Down Expand Up @@ -758,7 +758,7 @@ class PgTabletSplittingWaitQueuesTest : public PgTabletSplitTestBase,
FLAGS_enable_wait_queues = true;
FLAGS_enable_deadlock_detection = true;
FLAGS_enable_automatic_tablet_splitting = false;
FLAGS_ysql_max_write_restart_attempts = 0;
FLAGS_ysql_pg_conf_csv = MaxQueryLayerRetriesConf(0);
PgTabletSplitTestBase::SetUp();
}

Expand Down Expand Up @@ -850,7 +850,7 @@ class PgWaitQueueContentionStressTest : public PgMiniTestBase {
FLAGS_enable_wait_queues = true;
FLAGS_wait_queue_poll_interval_ms = 2;
FLAGS_transactions_status_poll_interval_ms = 5;
FLAGS_ysql_max_write_restart_attempts = 0;
FLAGS_ysql_pg_conf_csv = MaxQueryLayerRetriesConf(0);
PgMiniTestBase::SetUp();
}

Expand Down

0 comments on commit 18fca47

Please sign in to comment.