Skip to content

Commit

Permalink
sql: added support for create index inside new schema changer
Browse files Browse the repository at this point in the history
Fixes: #67264

Previously, create index was not supported inside the
new shcema changer. This was inadequate because, to
help support transaction schema changes in the future
we need all schema changes inside the new schema changer.
To address this, this patch adds CREATE INDEX inside
the new schema changer.

Release note: None
  • Loading branch information
fqazi committed Jul 30, 2021
1 parent 1b60983 commit 014c13e
Show file tree
Hide file tree
Showing 32 changed files with 5,711 additions and 2,106 deletions.
13 changes: 13 additions & 0 deletions pkg/ccl/logictestccl/testdata/logic_test/new_schema_changer
Original file line number Diff line number Diff line change
@@ -0,0 +1,13 @@
statement ok
SET experimental_use_new_schema_changer = 'on'

subtest create_index

statement ok
CREATE TABLE defaultdb.t1 (id INT PRIMARY KEY, name varchar(256), money int)

statement ok
CREATE INDEX id1 on defaultdb.t1(id, name) storing (money) PARTITION BY LIST (id) (
PARTITION p1 VALUES IN (1)
);

1 change: 1 addition & 0 deletions pkg/ccl/partitionccl/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@ go_library(
"//pkg/sql/pgwire/pgcode",
"//pkg/sql/pgwire/pgerror",
"//pkg/sql/rowenc",
"//pkg/sql/schemachanger/scbuild",
"//pkg/sql/sem/tree",
"//pkg/sql/types",
"//pkg/util/encoding",
Expand Down
2 changes: 2 additions & 0 deletions pkg/ccl/partitionccl/partition.go
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgcode"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgerror"
"github.com/cockroachdb/cockroach/pkg/sql/rowenc"
"github.com/cockroachdb/cockroach/pkg/sql/schemachanger/scbuild"
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree"
"github.com/cockroachdb/cockroach/pkg/sql/types"
"github.com/cockroachdb/cockroach/pkg/util/encoding"
Expand Down Expand Up @@ -611,4 +612,5 @@ func selectPartitionExprsByName(

func init() {
sql.CreatePartitioningCCL = createPartitioning
scbuild.CreatePartitioningCCL = createPartitioning
}
8 changes: 3 additions & 5 deletions pkg/sql/catalog/tabledesc/structured.go
Original file line number Diff line number Diff line change
Expand Up @@ -139,7 +139,7 @@ func (desc *wrapper) KeysPerRow(indexID descpb.IndexID) (int, error) {
return len(desc.Families), nil
}

// buildIndexName returns an index name that is not equal to any
// BuildIndexName returns an index name that is not equal to any
// of tableDesc's indexes, roughly following Postgres's conventions for naming
// anonymous indexes. For example:
//
Expand All @@ -152,9 +152,7 @@ func (desc *wrapper) KeysPerRow(indexID descpb.IndexID) (int, error) {
// CREATE INDEX ON t ((a + b), c, lower(d))
// => t_expr_c_expr1_idx
//
func buildIndexName(tableDesc *Mutable, index catalog.Index) (string, error) {
idx := index.IndexDesc()

func BuildIndexName(tableDesc *Mutable, idx *descpb.IndexDescriptor) (string, error) {
// An index name has a segment for the table name, each key column, and a
// final word (either "idx" or "key").
segments := make([]string, 0, len(idx.KeyColumnNames)+2)
Expand Down Expand Up @@ -675,7 +673,7 @@ func (desc *Mutable) allocateIndexIDs(columnNames map[string]descpb.ColumnID) er
// Assign names to unnamed indexes.
err := catalog.ForEachDeletableNonPrimaryIndex(desc, func(idx catalog.Index) error {
if len(idx.GetName()) == 0 {
name, err := buildIndexName(desc, idx)
name, err := BuildIndexName(desc, idx.IndexDesc())
if err != nil {
return err
}
Expand Down
2 changes: 1 addition & 1 deletion pkg/sql/conn_executor.go
Original file line number Diff line number Diff line change
Expand Up @@ -2744,7 +2744,7 @@ func (ex *connExecutor) runPreCommitStages(ctx context.Context) error {
executor := scexec.NewExecutor(
ex.planner.txn, &ex.extraTxnState.descCollection, ex.server.cfg.Codec,
nil /* backfiller */, nil /* jobTracker */, ex.server.cfg.NewSchemaChangerTestingKnobs,
ex.server.cfg.JobRegistry, ex.planner.execCfg.InternalExecutor,
ex.server.cfg.JobRegistry, ex.planner.execCfg.InternalExecutor,ex.server.cfg.Settings, ex.planner.EvalContext(),
)
after, err := runNewSchemaChanger(
ctx,
Expand Down
177 changes: 177 additions & 0 deletions pkg/sql/logictest/testdata/logic_test/new_schema_changer
Original file line number Diff line number Diff line change
Expand Up @@ -666,6 +666,183 @@ CREATE TABLE public.trewrite (
FAMILY fam_0_k_ts (k, ts, c)
)

subtest create-index
statement ok
CREATE TABLE tIndex (
a INT PRIMARY KEY,
b INT,
FAMILY (a),
FAMILY (b)
)

statement ok
INSERT INTO tIndex VALUES (1,1)

user root

statement ok
CREATE INDEX foo ON tIndex (b)

statement error relation \"foo\" already exists
CREATE INDEX foo ON tIndex (a)

statement error column "c" does not exist
CREATE INDEX bar ON tIndex (c)

statement error index \"bar\" contains duplicate column \"b\"
CREATE INDEX bar ON tIndex (b, b);

statement error pgcode 0A000 only simple columns are supported as index elements
CREATE INDEX bar ON tIndex ((a+b))

statement error pgcode 0A000 only simple columns are supported as index elements
CREATE INDEX bar ON tIndex (abs(b))

statement error pgcode 0A000 only simple columns are supported as index elements
CREATE UNIQUE INDEX bar ON tIndex (abs(b))

statement error pgcode 0A000 only simple columns are supported as index elements
CREATE INVERTED INDEX bar ON tIndex ((ARRAY[a,b]))

statement error pgcode 0A000 only simple columns are supported as index elements
CREATE TABLE tIndx2 (a INT PRIMARY KEY, b INT, INDEX ((a+b)))

statement error pgcode 0A000 only simple columns are supported as index elements
CREATE TABLE tIndx2 (a INT PRIMARY KEY, b INT, INVERTED INDEX ((ARRAY[a,b])))

query TTBITTBB colnames
SHOW INDEXES FROM tIndex
----
table_name index_name non_unique seq_in_index column_name direction storing implicit
tindex foo true 1 b ASC false false
tindex foo true 2 a ASC false true
tindex primary false 1 a ASC false false
tindex primary false 2 b N/A true false

statement ok
INSERT INTO tIndex VALUES (2,1)

# FIXME: Disabled until rollback is supported
#statement error pgcode 23505 violates unique constraint "bar"
#CREATE UNIQUE INDEX bar ON tIndex (b)

query TTBITTBB colnames
SHOW INDEXES FROM tIndex
----
table_name index_name non_unique seq_in_index column_name direction storing implicit
tindex foo true 1 b ASC false false
tindex foo true 2 a ASC false true
tindex primary false 1 a ASC false false
tindex primary false 2 b N/A true false

# test for DESC index

statement ok
DROP TABLE tIndex

statement ok
CREATE TABLE tIndx (
a INT PRIMARY KEY,
b INT,
c INT
)

statement ok
INSERT INTO tIndx VALUES (1,1,1), (2,2,2)

statement ok
CREATE INDEX b_desc ON tIndx (b DESC)

statement ok
CREATE INDEX b_asc ON tIndx (b ASC, c DESC)

query TTBITTBB colnames
SHOW INDEXES FROM tIndx
----
table_name index_name non_unique seq_in_index column_name direction storing implicit
tindx b_asc true 1 b ASC false false
tindx b_asc true 2 c DESC false false
tindx b_asc true 3 a ASC false true
tindx b_desc true 1 b DESC false false
tindx b_desc true 2 a ASC false true
tindx primary false 1 a ASC false false
tindx primary false 2 b N/A true false
tindx primary false 3 c N/A true false

statement error pgcode 42P01 relation "foo" does not exist
CREATE INDEX fail ON foo (b DESC)

statement ok
CREATE VIEW vIndx AS SELECT a,b FROM tIndx

statement error pgcode 42809 "vindx" is not a table or materialized view
CREATE INDEX failview ON vIndx (b DESC)

statement ok
CREATE TABLE privs (a INT PRIMARY KEY, b INT)

user testuser

statement error user testuser does not have CREATE privilege on relation privs
CREATE INDEX foo ON privs (b)

user root

query TTBITTBB colnames
SHOW INDEXES FROM privs
----
table_name index_name non_unique seq_in_index column_name direction storing implicit
privs primary false 1 a ASC false false
privs primary false 2 b N/A true false

statement ok
GRANT CREATE ON privs TO testuser

user testuser

statement ok
CREATE INDEX foo ON privs (b)

query TTBITTBB colnames
SHOW INDEXES FROM privs
----
table_name index_name non_unique seq_in_index column_name direction storing implicit
privs foo true 1 b ASC false false
privs foo true 2 a ASC false true
privs primary false 1 a ASC false false
privs primary false 2 b N/A true false


user root

statement ok
SET experimental_enable_hash_sharded_indexes = true;
CREATE TABLE telemetry (
x INT PRIMARY KEY,
y INT,
z JSONB
)


# Test that creating an index on a column which is currently being dropped
# causes an error.
subtest create_index_on_dropping_column

statement ok
CREATE TABLE create_idx_drop_column (c0 INT PRIMARY KEY, c1 INT);

statement ok
begin; ALTER TABLE create_idx_drop_column DROP COLUMN c1;

statement error column "c1" does not exist
CREATE INDEX idx_create_idx_drop_column ON create_idx_drop_column (c1);

statement ok
ROLLBACK;

statement ok
DROP TABLE create_idx_drop_column;

# Works around view back references to tables not being cleaned up.
# see issue #66677
statement ok
Expand Down
2 changes: 1 addition & 1 deletion pkg/sql/schema_change_plan_node.go
Original file line number Diff line number Diff line change
Expand Up @@ -122,7 +122,7 @@ func (s *schemaChangePlanNode) startExec(params runParams) error {
scs := p.extendedEvalCtx.SchemaChangerState
executor := scexec.NewExecutor(p.txn, p.Descriptors(), p.EvalContext().Codec,
nil /* backfiller */, nil /* jobTracker */, p.ExecCfg().NewSchemaChangerTestingKnobs,
params.extendedEvalCtx.ExecCfg.JobRegistry, params.p.execCfg.InternalExecutor)
params.extendedEvalCtx.ExecCfg.JobRegistry, params.p.execCfg.InternalExecutor, p.ExecCfg().Settings, p.EvalContext())
after, err := runNewSchemaChanger(
params.ctx, scplan.StatementPhase, s.plannedState, executor, scs.stmts,
)
Expand Down
3 changes: 3 additions & 0 deletions pkg/sql/schemachanger/scbuild/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -5,6 +5,7 @@ go_library(
srcs = [
"builder.go",
"database.go",
"index.go",
"relation_common.go",
"schema.go",
"sequence.go",
Expand All @@ -15,6 +16,7 @@ go_library(
importpath = "github.com/cockroachdb/cockroach/pkg/sql/schemachanger/scbuild",
visibility = ["//visibility:public"],
deps = [
"//pkg/settings/cluster",
"//pkg/sql/catalog",
"//pkg/sql/catalog/dbdesc",
"//pkg/sql/catalog/descpb",
Expand All @@ -31,6 +33,7 @@ go_library(
"//pkg/sql/sem/tree",
"//pkg/sql/sqlerrors",
"//pkg/sql/sqltelemetry",
"//pkg/sql/types",
"//pkg/util/errorutil/unimplemented",
"//pkg/util/protoutil",
"//pkg/util/sequence",
Expand Down
2 changes: 2 additions & 0 deletions pkg/sql/schemachanger/scbuild/builder.go
Original file line number Diff line number Diff line change
Expand Up @@ -160,6 +160,8 @@ func (b *buildContext) build(ctx context.Context, n tree.Statement) (output scpb
b.dropDatabase(ctx, n)
case *tree.AlterTable:
b.alterTable(ctx, n)
case *tree.CreateIndex:
b.createIndex(ctx, n)
default:
return nil, &notImplementedError{n: n}
}
Expand Down
Loading

0 comments on commit 014c13e

Please sign in to comment.