Skip to content

Commit

Permalink
upgrade: use high priority txn's to update the cluster version
Browse files Browse the repository at this point in the history
Previously, it was possible for the leasing subsystem to starve out
attempts to set the cluster version during upgrades, since the leasing
subsystem uses high priority txn for renewals. To address this, this
patch makes the logic to set the cluster version high priority so
it can't be pushed out by lease renewals.

Fixes: cockroachdb#113908

Release note (bug fix): Addressed a bug that could cause cluster version
finalization to get starved out by descriptor lease renewals on larger
clusters.
  • Loading branch information
fqazi committed Nov 13, 2023
1 parent b42bca4 commit 6973e9c
Show file tree
Hide file tree
Showing 3 changed files with 163 additions and 1 deletion.
43 changes: 42 additions & 1 deletion pkg/sql/set_cluster_setting.go
Original file line number Diff line number Diff line change
Expand Up @@ -617,6 +617,37 @@ func setVersionSetting(
return err
}
return db.Txn(ctx, func(ctx context.Context, txn isql.Txn) error {
// On complex clusters with a large number of descriptors (> 500) and
// multi-region nodes (> 9), normal priority transactions reading/updating
// the version row can be starved. This is due to the lease manager reading
// the version row at high priority, when refreshing leases (#95227), with
// a complex cluster this traffic will continuous.
// Run the version bump inside the upgrade as a high priority txn to avoid
// being starved out by the lease manager.
// Run the version bump inside the upgrade as high priority, since
// lease manager ends up reading the version row (with high priority)
// inside the settings table when refreshing leases. On complex clusters
// (multi-region with high latency) or with a large number of descriptors
// ( >500) it's possible for normal transactions to be starved by continuous
// lease traffic.
// This is safe from deadlocks / starvation because we expected this
// transaction only do the following:
// 1) We expect this transaction to only read and write to the
// version key in the system.settings table. It will also do high
// priority reads from the system.namespace and system.descriptor tables
// because the WithSkipDescriptorCache below (this will push out schema
// changes during the upgrade).
// 2) Reads from the system.sql_instances table to confirm all SQL servers
// have been upgraded in multi-tenant environments.
// 3) Other transactions will use a normal priority and get pushed out by
// this one, if they involve schema changes on the system database
// descriptor (highly unlikely).
// 4) There is a potential danger to conflict with lease renewal of the
// settings table and the upgrade, but to avoid this, we intentionally
// force the internal executor to avoid leased descriptors.
if err := txn.KV().SetUserPriority(roachpb.MaxUserPriority); err != nil {
return err
}
// Confirm if the version has actually changed on us.
datums, err := txn.QueryRowEx(
ctx, "retrieve-prev-setting", txn.KV(),
Expand Down Expand Up @@ -664,7 +695,17 @@ func setVersionSetting(
return err
}
return err
})
},
// We will intentionally skip the leasing layer to update the
// system.settings table to avoid a deadlock with the leasing layer.
// The leasing layer will read the version key in system.settings with
// a high priority before any renewal. The update code above will need a
// lease on system.settings and potentially system.sqlinstances. If any of
// leases required expire, we could deadlock with the renewal being
// blocked on the version key. To get around this, we will force the
// transaction to directly make round trips to read descriptors instead
// of using the leasing cache.
isql.WithSkipDescriptorCache())
}

// If we're here, we know we aren't in a transaction because we don't
Expand Down
2 changes: 2 additions & 0 deletions pkg/upgrade/upgrades/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -117,6 +117,7 @@ go_test(
"system_exec_insights_test.go",
"system_rbr_indexes_test.go",
"upgrades_test.go",
"version_starvation_test.go",
],
data = glob(["testdata/**"]),
embed = [":upgrades"],
Expand Down Expand Up @@ -170,6 +171,7 @@ go_test(
"//pkg/util/leaktest",
"//pkg/util/log",
"//pkg/util/protoutil",
"//pkg/util/retry",
"//pkg/util/syncutil",
"//pkg/util/timeutil",
"@com_github_cockroachdb_cockroach_go_v2//crdb",
Expand Down
119 changes: 119 additions & 0 deletions pkg/upgrade/upgrades/version_starvation_test.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,119 @@
// Copyright 2023 The Cockroach Authors.
//
// Use of this software is governed by the Business Source License
// included in the file licenses/BSL.txt.
//
// As of the Change Date specified in that file, in accordance with
// the Business Source License, use of this software will be governed
// by the Apache License, Version 2.0, included in the file
// licenses/APL.txt.

package upgrades_test

import (
"context"
"testing"

"github.com/cockroachdb/cockroach/pkg/base"
"github.com/cockroachdb/cockroach/pkg/clusterversion"
"github.com/cockroachdb/cockroach/pkg/server"
clustersettings "github.com/cockroachdb/cockroach/pkg/settings/cluster"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/lease"
"github.com/cockroachdb/cockroach/pkg/testutils/testcluster"
"github.com/cockroachdb/cockroach/pkg/upgrade/upgradebase"
"github.com/cockroachdb/cockroach/pkg/upgrade/upgrades"
"github.com/cockroachdb/cockroach/pkg/util/leaktest"
"github.com/cockroachdb/cockroach/pkg/util/retry"
"github.com/cockroachdb/errors"
"github.com/stretchr/testify/require"
)

// TestLeasingClusterVersionStarvation validates that setting
// the cluster version is done with a high priority txn and cannot
// be pushed out. Previously, this would be normal priority and
// get pushed by the leasing code, leading to starvation
// when leases were acquired with sufficiently high frequency
// Note: This test just confirms its not normal priority by checking
// if it can push other txns.
func TestLeasingClusterVersionStarvation(t *testing.T) {
defer leaktest.AfterTest(t)()
ctx := context.Background()

routineChan := make(chan error)
waitToStartBump := make(chan struct{})
resumeBump := make(chan struct{})
clusterArgs := base.TestClusterArgs{
ServerArgs: base.TestServerArgs{
Knobs: base.TestingKnobs{
UpgradeManager: &upgradebase.TestingKnobs{
InterlockPausePoint: upgradebase.AfterVersionBumpRPC,
InterlockReachedPausePointChannel: &waitToStartBump,
InterlockResumeChannel: &resumeBump,
},
Server: &server.TestingKnobs{
DisableAutomaticVersionUpgrade: make(chan struct{}),
BinaryVersionOverride: clusterversion.ByKey(
clusterversion.V23_2),
},
},
},
}

// Disable lease renewals intentionally, so that we validate
// no deadlock risk exists with the settings table.
st := clustersettings.MakeTestingClusterSettingsWithVersions(
clusterversion.TestingBinaryVersion,
clusterversion.TestingBinaryMinSupportedVersion,
false)

clusterArgs.ServerArgs.Settings = st

tc := testcluster.StartTestCluster(t, 1, clusterArgs)
lease.LeaseDuration.Override(ctx, &st.SV, 0)
lease.LeaseRenewalDuration.Override(ctx, &st.SV, 0)

defer tc.Stopper().Stop(ctx)
db := tc.ServerConn(0)
defer db.Close()

proceedWithCommit := make(chan struct{})
// Start a background transaction that will have an intent
// on the version key inside the settings table, with a
// normal priority (which should get pushed by the upgrade).
go func() {
<-waitToStartBump
tx, err := db.Begin()
if err != nil {
routineChan <- err
return
}
_, err = tx.Exec("SELECT name from system.settings where name='version' FOR UPDATE")
if err != nil {
routineChan <- err
return
}
resumeBump <- struct{}{}
for retry := retry.Start(retry.Options{}); retry.Next(); {
_, err = tx.Exec("SELECT name from system.settings where name='version' FOR UPDATE")
if err != nil {
rollbackErr := tx.Rollback()
routineChan <- errors.WithSecondaryError(err, rollbackErr)
return
}
}
}()

upgrades.Upgrade(
t,
db,
clusterversion.V24_1,
nil,
false,
)

// Our txn should have been pushed by the upgrade,
// which has a higher txn priority.
close(proceedWithCommit)
require.ErrorContainsf(t, <-routineChan, "pq: restart transaction: TransactionRetryWithProtoRefreshError:",
"upgrade was not able to push transaction")
}

0 comments on commit 6973e9c

Please sign in to comment.