Skip to content

Commit

Permalink
sql/schemachanger: support ADD COLUMN SERIAL for DSC unique_rowid only
Browse files Browse the repository at this point in the history
Previously we could only add SERIAL columns with the legacy schema changer
with these code changes the ALTER TABLE ADD COLUMN statement
can now add SERIAL type columns via the declarative schema changer for
the default rowid serial_normalization mode.

Informs: #126900
Release note: none
  • Loading branch information
Dedej-Bergin committed Sep 4, 2024
1 parent 39f3b6f commit 5a790be
Show file tree
Hide file tree
Showing 6 changed files with 135 additions and 38 deletions.
1 change: 1 addition & 0 deletions pkg/sql/catalog/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -11,6 +11,7 @@ go_library(
"metadata.go",
"post_deserialization_changes.go",
"schema.go",
"serial_helper.go",
"system_table.go",
"table_col_map.go",
"table_col_set.go",
Expand Down
54 changes: 54 additions & 0 deletions pkg/sql/catalog/serial_helper.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,54 @@
// Copyright 2024 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 catalog

import (
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgcode"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgerror"
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree"
"github.com/cockroachdb/cockroach/pkg/sql/types"
)

func UseRowID(d tree.ColumnTableDef) *tree.ColumnTableDef {
d.DefaultExpr.Expr = &tree.FuncExpr{Func: tree.WrapFunction("unique_rowid")}
d.Type = types.Int
// Column is non-nullable in all cases. PostgreSQL requires this.
d.Nullable.Nullability = tree.NotNull

return &d
}

func AssertValidSerialColumnDef(d *tree.ColumnTableDef, tableName *tree.TableName) error {
if d.HasDefaultExpr() {
// SERIAL implies a new default expression, we can't have one to
// start with. This is the error produced by pg in such case.
return pgerror.Newf(pgcode.Syntax,
"multiple default values specified for column %q of table %q",
tree.ErrString(&d.Name), tree.ErrString(tableName))
}

if d.Nullable.Nullability == tree.Null {
// SERIAL implies a non-NULL column, we can't accept a nullability
// spec. This is the error produced by pg in such case.
return pgerror.Newf(pgcode.Syntax,
"conflicting NULL/NOT NULL declarations for column %q of table %q",
tree.ErrString(&d.Name), tree.ErrString(tableName))
}

if d.Computed.Expr != nil {
// SERIAL cannot be a computed column.
return pgerror.Newf(pgcode.Syntax,
"SERIAL column %q of table %q cannot be computed",
tree.ErrString(&d.Name), tree.ErrString(tableName))
}

return nil
}
34 changes: 34 additions & 0 deletions pkg/sql/logictest/testdata/logic_test/alter_table
Original file line number Diff line number Diff line change
Expand Up @@ -3967,3 +3967,37 @@ statement error pgcode 42601 variable sub-expressions are not allowed in EXPRESS
ALTER TABLE t_124546 ADD CONSTRAINT ident UNIQUE ( ( EXISTS ( TABLE error FOR READ ONLY ) ) DESC ) STORING ( ident , ident );

subtest end

subtest alter_table_add_column_serial

statement ok
create table roach (id int);
insert into roach DEFAULT VALUES;
insert into roach DEFAULT VALUES;
SET serial_normalization = rowid

statement ok
alter table roach add column serial_id SERIAL;

query TTBTTTB colnames,rowsort
show columns from roach;
----
column_name data_type is_nullable column_default generation_expression indices is_hidden
id INT8 true NULL · {roach_pkey} false
rowid INT8 false unique_rowid() · {roach_pkey} true
serial_id INT8 false unique_rowid() · {roach_pkey} false

subtest end

subtest unimplemented_for_non_rowid_in_DSC

statement ok
SET serial_normalization = sql_sequence

statement ok
SET use_declarative_schema_changer = unsafe_always

statement error pq: \*tree.ColumnTableDef not implemented in the new schema changer: contains serial data type in unsupported mode
alter table roach add column serial_id2 SERIAL

subtest end
Original file line number Diff line number Diff line change
Expand Up @@ -15,6 +15,8 @@ import (
"sort"
"strings"

"github.com/cockroachdb/cockroach/pkg/docs"
"github.com/cockroachdb/cockroach/pkg/server/telemetry"
"github.com/cockroachdb/cockroach/pkg/sql/catalog"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/catenumpb"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/catpb"
Expand All @@ -25,12 +27,14 @@ import (
"github.com/cockroachdb/cockroach/pkg/sql/catalog/typedesc"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgcode"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgerror"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgnotice"
"github.com/cockroachdb/cockroach/pkg/sql/privilege"
"github.com/cockroachdb/cockroach/pkg/sql/schemachanger/scdecomp"
"github.com/cockroachdb/cockroach/pkg/sql/schemachanger/scerrors"
"github.com/cockroachdb/cockroach/pkg/sql/schemachanger/scpb"
"github.com/cockroachdb/cockroach/pkg/sql/sem/catid"
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree"
"github.com/cockroachdb/cockroach/pkg/sql/sessiondatapb"
"github.com/cockroachdb/cockroach/pkg/sql/sqlerrors"
"github.com/cockroachdb/cockroach/pkg/sql/sqltelemetry"
"github.com/cockroachdb/cockroach/pkg/sql/types"
Expand Down Expand Up @@ -71,7 +75,11 @@ func alterTableAddColumn(
}
}
if d.IsSerial {
panic(scerrors.NotImplementedErrorf(d, "contains serial data type"))
if b.SessionData().SerialNormalizationMode != sessiondatapb.SerialUsesRowID {
panic(scerrors.NotImplementedErrorf(d, "contains serial data type in unsupported mode"))
}
d = alterTableAddColumnSerial(b, d, tn)
d.IsSerial = false
}
if d.GeneratedIdentity.IsGeneratedAsIdentity {
panic(scerrors.NotImplementedErrorf(d, "contains generated identity type"))
Expand Down Expand Up @@ -259,6 +267,40 @@ func alterTableAddColumn(
}
}

func alterTableAddColumnSerial(
b BuildCtx, d *tree.ColumnTableDef, tn *tree.TableName,
) *tree.ColumnTableDef {
if err := catalog.AssertValidSerialColumnDef(d, tn); err != nil {
panic(err)
}

defType, err := tree.ResolveType(b, d.Type, b.SemaCtx().GetTypeResolver())
if err != nil {
panic(err)
}

telemetry.Inc(sqltelemetry.SerialColumnNormalizationCounter(
defType.Name(), b.SessionData().SerialNormalizationMode.String()))

if defType.Width() < types.Int.Width() {
b.EvalCtx().ClientNoticeSender.BufferClientNotice(
b,
errors.WithHintf(
pgnotice.Newf(
"upgrading the column %s to %s to utilize the session serial_normalization setting",
d.Name.String(),
types.Int.SQLString(),
),
"change the serial_normalization to sql_sequence or sql_sequence_cached if you wish "+
"to use a smaller sized serial column at the cost of performance. See %s",
docs.URL("serial.html"),
),
)
}

return catalog.UseRowID(*d)
}

func columnNamesToIDs(b BuildCtx, tbl *scpb.Table) map[string]descpb.ColumnID {
tableElts := b.QueryByID(tbl.TableID)
namesToIDs := make(map[string]descpb.ColumnID)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -12,10 +12,6 @@ CREATE TABLE defaultdb.foo (
);
----

unimplemented
ALTER TABLE defaultdb.foo ADD COLUMN j SERIAL
----

unimplemented
ALTER TABLE defaultdb.foo ALTER COLUMN i DROP NOT NULL
----
Expand Down
36 changes: 3 additions & 33 deletions pkg/sql/serial.go
Original file line number Diff line number Diff line change
Expand Up @@ -19,8 +19,6 @@ import (
"github.com/cockroachdb/cockroach/pkg/settings"
"github.com/cockroachdb/cockroach/pkg/sql/catalog"
"github.com/cockroachdb/cockroach/pkg/sql/catalog/resolver"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgcode"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgerror"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgnotice"
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree"
"github.com/cockroachdb/cockroach/pkg/sql/sessiondatapb"
Expand Down Expand Up @@ -132,14 +130,13 @@ func (p *planner) generateSerialInColumnDef(
error,
) {

if err := assertValidSerialColumnDef(d, tableName); err != nil {
if err := catalog.AssertValidSerialColumnDef(d, tableName); err != nil {
return nil, nil, nil, nil, err
}

newSpec := *d

// Make the column non-nullable in all cases. PostgreSQL requires
// this.
// Column is non-nullable in all cases. PostgreSQL requires this.
newSpec.Nullable.Nullability = tree.NotNull

// Clear the IsSerial bit now that it's been remapped.
Expand Down Expand Up @@ -356,7 +353,7 @@ func SimplifySerialInColumnDefWithRowID(
return nil
}

if err := assertValidSerialColumnDef(d, tableName); err != nil {
if err := catalog.AssertValidSerialColumnDef(d, tableName); err != nil {
return err
}

Expand All @@ -374,30 +371,3 @@ func SimplifySerialInColumnDefWithRowID(

return nil
}

func assertValidSerialColumnDef(d *tree.ColumnTableDef, tableName *tree.TableName) error {
if d.HasDefaultExpr() {
// SERIAL implies a new default expression, we can't have one to
// start with. This is the error produced by pg in such case.
return pgerror.Newf(pgcode.Syntax,
"multiple default values specified for column %q of table %q",
tree.ErrString(&d.Name), tree.ErrString(tableName))
}

if d.Nullable.Nullability == tree.Null {
// SERIAL implies a non-NULL column, we can't accept a nullability
// spec. This is the error produced by pg in such case.
return pgerror.Newf(pgcode.Syntax,
"conflicting NULL/NOT NULL declarations for column %q of table %q",
tree.ErrString(&d.Name), tree.ErrString(tableName))
}

if d.Computed.Expr != nil {
// SERIAL cannot be a computed column.
return pgerror.Newf(pgcode.Syntax,
"SERIAL column %q of table %q cannot be computed",
tree.ErrString(&d.Name), tree.ErrString(tableName))
}

return nil
}

0 comments on commit 5a790be

Please sign in to comment.