Skip to content

Commit

Permalink
sql, parser: add EXPERIMENTAL_RELOCATE NON_VOTERS
Browse files Browse the repository at this point in the history
This commit adds a new variant of the `EXPERIMENTAL_RELOCATE` statement
which enables relocation of non-voting replicas via SQL.

Release note (sql change): `EXPERIMENTAL_RELOCATE` now has a new
variant: `EXPERIMENTAL_RELOCATE NON_VOTERS` which allows users to
relocate non-voting replicas via SQL.
  • Loading branch information
aayushshah15 committed Mar 29, 2021
1 parent af80850 commit c49b853
Show file tree
Hide file tree
Showing 15 changed files with 100 additions and 27 deletions.
1 change: 1 addition & 0 deletions docs/generated/sql/bnf/stmt_block.bnf
Original file line number Diff line number Diff line change
Expand Up @@ -968,6 +968,7 @@ unreserved_keyword ::=
| 'NOCONTROLJOB'
| 'NOLOGIN'
| 'NOMODIFYCLUSTERSETTING'
| 'NON_VOTERS'
| 'NOVIEWACTIVITY'
| 'NOWAIT'
| 'NULLS'
Expand Down
6 changes: 4 additions & 2 deletions pkg/ccl/kvccl/kvfollowerreadsccl/followerreads_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -540,7 +540,8 @@ func TestFollowerReadsWithStaleDescriptor(t *testing.T) {
n1 := sqlutils.MakeSQLRunner(tc.Conns[0])
n1.Exec(t, `CREATE DATABASE t`)
n1.Exec(t, `CREATE TABLE test (k INT PRIMARY KEY)`)
n1.Exec(t, `ALTER TABLE test EXPERIMENTAL_RELOCATE VOTERS VALUES (ARRAY[1,2], 1)`)
n1.Exec(t, `ALTER TABLE test EXPERIMENTAL_RELOCATE VOTERS VALUES (ARRAY[1], 1)`)
n1.Exec(t, `ALTER TABLE test EXPERIMENTAL_RELOCATE NON_VOTERS VALUES (ARRAY[2], 1)`)
// Speed up closing of timestamps, as we'll in order to be able to use
// follower_read_timestamp().
// Every 0.2s we'll close the timestamp from 0.4s ago. We'll attempt follower reads
Expand Down Expand Up @@ -570,11 +571,12 @@ func TestFollowerReadsWithStaleDescriptor(t *testing.T) {
require.Equal(t, roachpb.StoreID(1), entry.Lease().Replica.StoreID)
require.Equal(t, []roachpb.ReplicaDescriptor{
{NodeID: 1, StoreID: 1, ReplicaID: 1},
{NodeID: 2, StoreID: 2, ReplicaID: 2},
{NodeID: 2, StoreID: 2, ReplicaID: 2, Type: roachpb.ReplicaTypeNonVoter()},
}, entry.Desc().Replicas().Descriptors())

// Relocate the follower. n2 will no longer have a replica.
n1.Exec(t, `ALTER TABLE test EXPERIMENTAL_RELOCATE VOTERS VALUES (ARRAY[1,3], 1)`)
n1.Exec(t, `ALTER TABLE test EXPERIMENTAL_RELOCATE NON_VOTERS VALUES (ARRAY[], 1)`)

// Execute the query again and assert the cache is updated. This query will
// not be executed as a follower read since it attempts to use n2 which
Expand Down
2 changes: 1 addition & 1 deletion pkg/sql/distsql_spec_exec_factory.go
Original file line number Diff line number Diff line change
Expand Up @@ -952,7 +952,7 @@ func (e *distSQLSpecExecFactory) ConstructAlterTableUnsplitAll(index cat.Index)
}

func (e *distSQLSpecExecFactory) ConstructAlterTableRelocate(
index cat.Index, input exec.Node, relocateLease bool,
index cat.Index, input exec.Node, relocateLease bool, relocateNonVoters bool,
) (exec.Node, error) {
return nil, unimplemented.NewWithIssue(47473, "experimental opt-driven distsql planning: alter table relocate")
}
Expand Down
1 change: 1 addition & 0 deletions pkg/sql/opt/exec/execbuilder/statement.go
Original file line number Diff line number Diff line change
Expand Up @@ -201,6 +201,7 @@ func (b *Builder) buildAlterTableRelocate(relocate *memo.AlterTableRelocateExpr)
table.Index(relocate.Index),
input.root,
relocate.RelocateLease,
relocate.RelocateNonVoters,
)
if err != nil {
return execPlan{}, err
Expand Down
1 change: 1 addition & 0 deletions pkg/sql/opt/exec/factory.opt
Original file line number Diff line number Diff line change
Expand Up @@ -609,6 +609,7 @@ define AlterTableRelocate {
Index cat.Index
input exec.Node
relocateLease bool
relocateNonVoters bool
}

# Buffer passes through the input rows but also saves them in a buffer, which
Expand Down
4 changes: 4 additions & 0 deletions pkg/sql/opt/memo/expr_format.go
Original file line number Diff line number Diff line change
Expand Up @@ -1427,6 +1427,10 @@ func FormatPrivate(f *ExprFmtCtx, private interface{}, physProps *physical.Requi
FormatPrivate(f, &t.AlterTableSplitPrivate, nil)
if t.RelocateLease {
f.Buffer.WriteString(" [lease]")
} else if t.RelocateNonVoters {
f.Buffer.WriteString(" [non-voters]")
} else {
f.Buffer.WriteString(" [voters]")
}

case *ControlJobsPrivate:
Expand Down
1 change: 1 addition & 0 deletions pkg/sql/opt/ops/statement.opt
Original file line number Diff line number Diff line change
Expand Up @@ -192,6 +192,7 @@ define AlterTableRelocate {
[Private]
define AlterTableRelocatePrivate {
RelocateLease bool
RelocateNonVoters bool
_ AlterTableSplitPrivate
}

Expand Down
11 changes: 9 additions & 2 deletions pkg/sql/opt/optbuilder/alter_table.go
Original file line number Diff line number Diff line change
Expand Up @@ -61,7 +61,13 @@ func (b *Builder) buildAlterTableSplit(split *tree.Split, inScope *scope) (outSc
b.semaCtx.Properties.Require(emptyScope.context.String(), tree.RejectSpecial)

texpr := emptyScope.resolveType(split.ExpireExpr, types.String)
expiration = b.buildScalar(texpr, emptyScope, nil /* outScope */, nil /* outCol */, nil /* colRefs */)
expiration = b.buildScalar(
texpr,
emptyScope,
nil, /* outScope */
nil, /* outCol */
nil, /* colRefs */
)
} else {
expiration = b.factory.ConstructNull(types.String)
}
Expand Down Expand Up @@ -175,7 +181,8 @@ func (b *Builder) buildAlterTableRelocate(
outScope.expr = b.factory.ConstructAlterTableRelocate(
inputScope.expr.(memo.RelExpr),
&memo.AlterTableRelocatePrivate{
RelocateLease: relocate.RelocateLease,
RelocateLease: relocate.RelocateLease,
RelocateNonVoters: relocate.RelocateNonVoters,
AlterTableSplitPrivate: memo.AlterTableSplitPrivate{
Table: b.factory.Metadata().AddTable(table, &tn),
Index: index.Ordinal(),
Expand Down
24 changes: 22 additions & 2 deletions pkg/sql/opt/optbuilder/testdata/alter_table
Original file line number Diff line number Diff line change
Expand Up @@ -129,7 +129,17 @@ alter-table-unsplit abc@bc
build
ALTER TABLE abc EXPERIMENTAL_RELOCATE VALUES (ARRAY[1,2,3], 1), (ARRAY[4], 2)
----
alter-table-relocate abc
alter-table-relocate abc [voters]
├── columns: key:1 pretty:2
└── values
├── columns: column1:3 column2:4!null
├── (ARRAY[1,2,3], 1)
└── (ARRAY[4], 2)

build
ALTER TABLE abc EXPERIMENTAL_RELOCATE NON_VOTERS VALUES (ARRAY[1,2,3], 1), (ARRAY[4], 2)
----
alter-table-relocate abc [non-voters]
├── columns: key:1 pretty:2
└── values
├── columns: column1:3 column2:4!null
Expand All @@ -149,7 +159,17 @@ error (42601): too many columns in EXPERIMENTAL_RELOCATE LEASE data
build
ALTER INDEX abc@bc EXPERIMENTAL_RELOCATE VALUES (ARRAY[5], 1, 'foo'), (ARRAY[6,7,8], 2, 'bar')
----
alter-table-relocate abc@bc
alter-table-relocate abc@bc [voters]
├── columns: key:1 pretty:2
└── values
├── columns: column1:3 column2:4!null column3:5!null
├── (ARRAY[5], 1, 'foo')
└── (ARRAY[6,7,8], 2, 'bar')

build
ALTER INDEX abc@bc EXPERIMENTAL_RELOCATE NON_VOTERS VALUES (ARRAY[5], 1, 'foo'), (ARRAY[6,7,8], 2, 'bar')
----
alter-table-relocate abc@bc [non-voters]
├── columns: key:1 pretty:2
└── values
├── columns: column1:3 column2:4!null column3:5!null
Expand Down
11 changes: 6 additions & 5 deletions pkg/sql/opt_exec_factory.go
Original file line number Diff line number Diff line change
Expand Up @@ -1825,17 +1825,18 @@ func (ef *execFactory) ConstructAlterTableUnsplitAll(index cat.Index) (exec.Node

// ConstructAlterTableRelocate is part of the exec.Factory interface.
func (ef *execFactory) ConstructAlterTableRelocate(
index cat.Index, input exec.Node, relocateLease bool,
index cat.Index, input exec.Node, relocateLease bool, relocateNonVoters bool,
) (exec.Node, error) {
if !ef.planner.ExecCfg().Codec.ForSystemTenant() {
return nil, errorutil.UnsupportedWithMultiTenancy(54250)
}

return &relocateNode{
relocateLease: relocateLease,
tableDesc: index.Table().(*optTable).desc,
index: index.(*optIndex).desc,
rows: input.(planNode),
relocateLease: relocateLease,
relocateNonVoters: relocateNonVoters,
tableDesc: index.Table().(*optTable).desc,
index: index.(*optIndex).desc,
rows: input.(planNode),
}, nil
}

Expand Down
6 changes: 6 additions & 0 deletions pkg/sql/parser/parse_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -1524,6 +1524,12 @@ func TestParse(t *testing.T) {
{`ALTER TABLE d.a EXPERIMENTAL_RELOCATE VOTERS VALUES (ARRAY[1, 2, 3], 'b', 2)`},
{`ALTER INDEX d.i EXPERIMENTAL_RELOCATE VOTERS VALUES (ARRAY[1], 2)`},

{`ALTER TABLE a EXPERIMENTAL_RELOCATE NON_VOTERS VALUES (ARRAY[1], 1)`},
{`EXPLAIN ALTER TABLE a EXPERIMENTAL_RELOCATE NON_VOTERS TABLE b`},
{`ALTER TABLE a EXPERIMENTAL_RELOCATE NON_VOTERS SELECT * FROM t`},
{`ALTER TABLE d.a EXPERIMENTAL_RELOCATE NON_VOTERS VALUES (ARRAY[1, 2, 3], 'b', 2)`},
{`ALTER INDEX d.i EXPERIMENTAL_RELOCATE NON_VOTERS VALUES (ARRAY[1], 2)`},

{`ALTER TABLE a EXPERIMENTAL_RELOCATE LEASE VALUES (1, 1)`},
{`ALTER TABLE a EXPERIMENTAL_RELOCATE LEASE SELECT * FROM t`},
{`ALTER TABLE d.a EXPERIMENTAL_RELOCATE LEASE VALUES (1, 'b', 2)`},
Expand Down
18 changes: 17 additions & 1 deletion pkg/sql/parser/sql.y
Original file line number Diff line number Diff line change
Expand Up @@ -666,7 +666,7 @@ func (u *sqlSymUnion) objectNamePrefixList() tree.ObjectNamePrefixList {

%token <str> NAN NAME NAMES NATURAL NEVER NEXT NO NOCANCELQUERY NOCONTROLCHANGEFEED NOCONTROLJOB
%token <str> NOCREATEDB NOCREATELOGIN NOCREATEROLE NOLOGIN NOMODIFYCLUSTERSETTING NO_INDEX_JOIN
%token <str> NONE NORMAL NOT NOTHING NOTNULL NOVIEWACTIVITY NOWAIT NULL NULLIF NULLS NUMERIC
%token <str> NONE NON_VOTERS NORMAL NOT NOTHING NOTNULL NOVIEWACTIVITY NOWAIT NULL NULLIF NULLS NUMERIC

%token <str> OF OFF OFFSET OID OIDS OIDVECTOR ON ONLY OPT OPTION OPTIONS OR
%token <str> ORDER ORDINALITY OTHERS OUT OUTER OVER OVERLAPS OVERLAY OWNED OWNER OPERATOR
Expand Down Expand Up @@ -1698,13 +1698,28 @@ alter_relocate_stmt:
Rows: $6.slct(),
}
}
| ALTER TABLE table_name relocate_kw NON_VOTERS select_stmt
{
/* SKIP DOC */
name := $3.unresolvedObjectName().ToTableName()
$$.val = &tree.Relocate{
TableOrIndex: tree.TableIndexName{Table: name},
Rows: $6.slct(),
RelocateNonVoters: true,
}
}

alter_relocate_index_stmt:
ALTER INDEX table_index_name relocate_kw voters_kw select_stmt
{
/* SKIP DOC */
$$.val = &tree.Relocate{TableOrIndex: $3.tableIndexName(), Rows: $6.slct()}
}
| ALTER INDEX table_index_name relocate_kw NON_VOTERS select_stmt
{
/* SKIP DOC */
$$.val = &tree.Relocate{TableOrIndex: $3.tableIndexName(), Rows: $6.slct(), RelocateNonVoters: true}
}

alter_relocate_lease_stmt:
ALTER TABLE table_name relocate_kw LEASE select_stmt
Expand Down Expand Up @@ -12457,6 +12472,7 @@ unreserved_keyword:
| NOCONTROLJOB
| NOLOGIN
| NOMODIFYCLUSTERSETTING
| NON_VOTERS
| NOVIEWACTIVITY
| NOWAIT
| NULLS
Expand Down
25 changes: 16 additions & 9 deletions pkg/sql/relocate.go
Original file line number Diff line number Diff line change
Expand Up @@ -29,10 +29,11 @@ import (
type relocateNode struct {
optColumnsSlot

relocateLease bool
tableDesc catalog.TableDescriptor
index *descpb.IndexDescriptor
rows planNode
relocateLease bool
relocateNonVoters bool
tableDesc catalog.TableDescriptor
index *descpb.IndexDescriptor
rows planNode

run relocateRun
}
Expand All @@ -52,9 +53,6 @@ func (n *relocateNode) startExec(runParams) error {
return nil
}

// TODO(aayush): Extend EXPERIMENTAL_RELOCATE syntax to support relocating
// non-voting replicas.

func (n *relocateNode) Next(params runParams) (bool, error) {
// Each Next call relocates one range (corresponding to one row from n.rows).
// TODO(radu): perform multiple relocations in parallel.
Expand Down Expand Up @@ -82,7 +80,8 @@ func (n *relocateNode) Next(params runParams) (bool, error) {
)
}
relocation := data[0].(*tree.DArray)
if len(relocation.Array) == 0 {
if !n.relocateNonVoters && len(relocation.Array) == 0 {
// We cannot remove all voters.
return false, errors.Errorf("empty relocation array for EXPERIMENTAL_RELOCATE")
}

Expand Down Expand Up @@ -129,13 +128,21 @@ func (n *relocateNode) Next(params runParams) (bool, error) {
}
n.run.lastRangeStartKey = rangeDesc.StartKey.AsRawKey()

existingVoters := rangeDesc.Replicas().Voters().ReplicationTargets()
existingNonVoters := rangeDesc.Replicas().NonVoters().ReplicationTargets()
if n.relocateLease {
if err := params.p.ExecCfg().DB.AdminTransferLease(params.ctx, rowKey, leaseStoreID); err != nil {
return false, err
}
} else if n.relocateNonVoters {
if err := params.p.ExecCfg().DB.AdminRelocateRange(
params.ctx, rowKey, existingVoters, relocationTargets,
); err != nil {
return false, err
}
} else {
if err := params.p.ExecCfg().DB.AdminRelocateRange(
params.ctx, rowKey, relocationTargets, rangeDesc.Replicas().NonVoters().ReplicationTargets(),
params.ctx, rowKey, relocationTargets, existingNonVoters,
); err != nil {
return false, err
}
Expand Down
12 changes: 8 additions & 4 deletions pkg/sql/sem/tree/split.go
Original file line number Diff line number Diff line change
Expand Up @@ -73,8 +73,9 @@ type Relocate struct {
// Each row contains an array with store ids and values for the columns in the
// PK or index (or a prefix of the columns).
// See docs/RFCS/sql_split_syntax.md.
Rows *Select
RelocateLease bool
Rows *Select
RelocateLease bool
RelocateNonVoters bool
}

// Format implements the NodeFormatter interface.
Expand All @@ -86,10 +87,13 @@ func (node *Relocate) Format(ctx *FmtCtx) {
ctx.WriteString("TABLE ")
}
ctx.FormatNode(&node.TableOrIndex)
ctx.WriteString(" EXPERIMENTAL_RELOCATE ")
if node.RelocateLease {
ctx.WriteString(" EXPERIMENTAL_RELOCATE LEASE ")
ctx.WriteString("LEASE ")
} else if node.RelocateNonVoters {
ctx.WriteString("NON_VOTERS ")
} else {
ctx.WriteString(" EXPERIMENTAL_RELOCATE VOTERS ")
ctx.WriteString("VOTERS ")
}
ctx.FormatNode(node.Rows)
}
Expand Down
4 changes: 3 additions & 1 deletion pkg/sql/sem/tree/stmt.go
Original file line number Diff line number Diff line change
Expand Up @@ -673,8 +673,10 @@ func (*Relocate) StatementType() StatementType { return Rows }
func (n *Relocate) StatementTag() string {
if n.RelocateLease {
return "EXPERIMENTAL_RELOCATE LEASE"
} else if n.RelocateNonVoters {
return "EXPERIMENTAL_RELOCATE NON_VOTERS"
}
return "EXPERIMENTAL_RELOCATE"
return "EXPERIMENTAL_RELOCATE VOTERS"
}

// StatementType implements the Statement interface.
Expand Down

0 comments on commit c49b853

Please sign in to comment.