// Licensed to the Apache Software Foundation (ASF) under one
// or more contributor license agreements.  See the NOTICE file
// distributed with this work for additional information
// regarding copyright ownership.  The ASF licenses this file
// to you under the Apache License, Version 2.0 (the
// "License"); you may not use this file except in compliance
// with the License.  You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.

package scalar

import (
	
	
	

	
	
	
	
	
)

type ListScalar interface {
	Scalar
	GetList() arrow.Array
	Release()
	Retain()
}

type List struct {
	scalar
	Value arrow.Array
}

func ( *List) () {
	if .Value != nil {
		.Value.Release()
	}
}

func ( *List) () {
	if .Value != nil {
		.Value.Retain()
	}
}

func ( *List) () interface{}   { return .Value }
func ( *List) () arrow.Array { return .Value }
func ( *List) ( Scalar) bool {
	return array.Equal(.Value, .(ListScalar).GetList())
}
func ( *List) () ( error) {
	if  = .scalar.Validate();  != nil {
		return
	}
	if  = validateOptional(&.scalar, .Value, "value");  != nil {
		return
	}

	if !.Valid {
		return
	}

	 := .Type.(arrow.ListLikeType).Elem()
	 := .Type

	if !arrow.TypeEqual(.Value.DataType(), ) {
		 = fmt.Errorf("%s scalar should have a value of type %s, got %s",
			, , .Value.DataType())
	}
	return
}

func ( *List) () error { return .Validate() }
func ( *List) ( arrow.DataType) (Scalar, error) {
	if !.Valid {
		return MakeNullScalar(), nil
	}

	if arrow.TypeEqual(.Type, ) {
		return , nil
	}

	if .ID() == arrow.STRING {
		var  bytes.Buffer
		fmt.Fprint(&, .Value)
		 := memory.NewBufferBytes(.Bytes())
		defer .Release()
		return NewStringScalarFromBuffer(), nil
	}

	return nil, fmt.Errorf("cannot convert non-nil list scalar to type %s", )
}

func ( *List) () string {
	if !.Valid {
		return "null"
	}
	,  := .CastTo(arrow.BinaryTypes.String)
	if  != nil {
		return "..."
	}
	return string(.(*String).Value.Bytes())
}

func ( arrow.Array) *List {
	return &List{scalar{arrow.ListOf(.DataType()), true}, array.MakeFromData(.Data())}
}

func ( arrow.ArrayData) *List {
	return &List{scalar{arrow.ListOf(.DataType()), true}, array.MakeFromData()}
}

type LargeList struct {
	*List
}

func ( arrow.Array) *LargeList {
	return &LargeList{&List{scalar{arrow.LargeListOf(.DataType()), true}, array.MakeFromData(.Data())}}
}

func ( arrow.ArrayData) *LargeList {
	return &LargeList{&List{scalar{arrow.LargeListOf(.DataType()), true}, array.MakeFromData()}}
}

func makeMapType( *arrow.StructType) *arrow.MapType {
	debug.Assert(.NumFields() == 2, "must pass struct with only 2 fields for MapScalar")
	return arrow.MapOf(.Field(0).Type, .Field(1).Type)
}

type Map struct {
	*List
}

func ( arrow.Array) *Map {
	return &Map{&List{scalar{makeMapType(.DataType().(*arrow.StructType)), true}, array.MakeFromData(.Data())}}
}

type FixedSizeList struct {
	*List
}

func ( *FixedSizeList) () ( error) {
	if  = .List.Validate();  != nil {
		return
	}

	if .Valid {
		 := .Type.(*arrow.FixedSizeListType)
		if .Value.Len() != int(.Len()) {
			return fmt.Errorf("%s scalar should have a child value of length %d, got %d",
				.Type, .Len(), .Value.Len())
		}
	}
	return
}

func ( *FixedSizeList) () error { return .Validate() }

func ( arrow.Array) *FixedSizeList {
	return NewFixedSizeListScalarWithType(, arrow.FixedSizeListOf(int32(.Len()), .DataType()))
}

func ( arrow.Array,  arrow.DataType) *FixedSizeList {
	debug.Assert(.Len() == int(.(*arrow.FixedSizeListType).Len()), "length of value for fixed size list scalar must match type")
	return &FixedSizeList{&List{scalar{, true}, array.MakeFromData(.Data())}}
}

type Vector []Scalar

type Struct struct {
	scalar
	Value Vector
}

func ( *Struct) () {
	for ,  := range .Value {
		if ,  := .(Releasable);  {
			.Release()
		}
	}
}

func ( *Struct) ( string) (Scalar, error) {
	,  := .Type.(*arrow.StructType).FieldIdx()
	if ! {
		return nil, fmt.Errorf("no field named %s found in struct scalar %s", , .Type)
	}

	return .Value[], nil
}

func ( *Struct) () interface{} { return .Value }

func ( *Struct) () string {
	if !.Valid {
		return "null"
	}
	,  := .CastTo(arrow.BinaryTypes.String)
	if  != nil {
		return "..."
	}
	return string(.(*String).Value.Bytes())
}

func ( *Struct) ( arrow.DataType) (Scalar, error) {
	if !.Valid {
		return MakeNullScalar(), nil
	}

	if .ID() != arrow.STRING {
		return nil, fmt.Errorf("cannot cast non-null struct scalar to type %s", )
	}

	var  bytes.Buffer
	 := .Type.(*arrow.StructType)
	.WriteByte('{')
	for ,  := range .Value {
		if  > 0 {
			.WriteString(", ")
		}
		.WriteString(fmt.Sprintf("%s:%s = %s", .Field().Name, .Field().Type, .String()))
	}
	.WriteByte('}')
	 := memory.NewBufferBytes(.Bytes())
	defer .Release()
	return NewStringScalarFromBuffer(), nil
}

func ( *Struct) ( Scalar) bool {
	 := .(*Struct)
	if len(.Value) != len(.Value) {
		return false
	}

	for  := range .Value {
		if !Equals(.Value[], .Value[]) {
			return false
		}
	}
	return true
}

func ( *Struct) () ( error) {
	if  = .scalar.Validate();  != nil {
		return
	}

	if !.Valid {
		for ,  := range .Value {
			if .IsValid() {
				 = fmt.Errorf("%s scalar is marked null but has child values", .Type)
				return
			}
		}
		return
	}

	 := .Type.(*arrow.StructType)
	 := .NumFields()
	if len(.Value) !=  {
		return fmt.Errorf("non-null %s scalar should have %d child values, got %d", .Type, , len(.Value))
	}

	for ,  := range .Fields() {
		if .Value[] == nil {
			return fmt.Errorf("non-null %s scalar has missing child value at index %d", .Type, )
		}

		 = .Value[].Validate()
		if  != nil {
			return fmt.Errorf("%s scalar fails validation for child at index %d: %w", .Type, , )
		}

		if !arrow.TypeEqual(.Value[].DataType(), .Type) {
			return fmt.Errorf("%s scalar should have a child value of type %s at index %d, got %s", .Type, .Type, , .Value[].DataType())
		}
	}
	return
}

func ( *Struct) () ( error) {
	if  = .scalar.ValidateFull();  != nil {
		return
	}

	if !.Valid {
		for ,  := range .Value {
			if .IsValid() {
				 = fmt.Errorf("%s scalar is marked null but has child values", .Type)
				return
			}
		}
		return
	}

	 := .Type.(*arrow.StructType)
	 := .NumFields()
	if len(.Value) !=  {
		return fmt.Errorf("non-null %s scalar should have %d child values, got %d", .Type, , len(.Value))
	}

	for ,  := range .Fields() {
		if .Value[] == nil {
			return fmt.Errorf("non-null %s scalar has missing child value at index %d", .Type, )
		}

		 = .Value[].ValidateFull()
		if  != nil {
			return fmt.Errorf("%s scalar fails validation for child at index %d: %w", .Type, , )
		}

		if !arrow.TypeEqual(.Value[].DataType(), .Type) {
			return fmt.Errorf("%s scalar should have a child value of type %s at index %d, got %s", .Type, .Type, , .Value[].DataType())
		}
	}
	return
}

func ( []Scalar,  arrow.DataType) *Struct {
	return &Struct{scalar{, true}, }
}

func ( []Scalar,  []string) (*Struct, error) {
	if len() != len() {
		return nil, xerrors.New("mismatching number of field names and child scalars")
	}

	 := make([]arrow.Field, len())
	for ,  := range  {
		[] = arrow.Field{Name: , Type: [].DataType(), Nullable: true}
	}
	return NewStructScalar(, arrow.StructOf(...)), nil
}

type Dictionary struct {
	scalar

	Value struct {
		Index Scalar
		Dict  arrow.Array
	}
}

func ( arrow.DataType) *Dictionary {
	 := &Dictionary{scalar: scalar{, false}}
	.Value.Index = MakeNullScalar(.(*arrow.DictionaryType).IndexType)
	.Value.Dict = nil
	return 
}

func ( Scalar,  arrow.Array) *Dictionary {
	 := &Dictionary{scalar: scalar{&arrow.DictionaryType{IndexType: .DataType(), ValueType: .DataType()}, .IsValid()}}
	.Value.Index = 
	.Value.Dict = 
	.Retain()
	return 
}

func ( *Dictionary) () []byte { return .Value.Index.(PrimitiveScalar).Data() }

func ( *Dictionary) () {
	if ,  := .Value.Index.(Releasable);  {
		.Retain()
	}
	if .Value.Dict != (arrow.Array)(nil) {
		.Value.Dict.Retain()
	}
}

func ( *Dictionary) () {
	if ,  := .Value.Index.(Releasable);  {
		.Release()
	}
	if .Value.Dict != (arrow.Array)(nil) {
		.Value.Dict.Release()
	}
}

func ( *Dictionary) () ( error) {
	,  := .Type.(*arrow.DictionaryType)
	if ! {
		return errors.New("arrow/scalar: dictionary scalar should have type Dictionary")
	}

	if .Value.Index == (Scalar)(nil) {
		return fmt.Errorf("%s scalar doesn't have an index value", )
	}

	if  = .Value.Index.Validate();  != nil {
		return fmt.Errorf("%s scalar fails validation for index value: %w", , )
	}

	if !arrow.TypeEqual(.Value.Index.DataType(), .IndexType) {
		return fmt.Errorf("%s scalar should have an index value of type %s, got %s",
			, .IndexType, .Value.Index.DataType())
	}

	if .IsValid() && !.Value.Index.IsValid() {
		return fmt.Errorf("non-null %s scalar has null index value", )
	}

	if !.IsValid() && .Value.Index.IsValid() {
		return fmt.Errorf("null %s scalar has non-null index value", )
	}

	if !.IsValid() {
		return
	}

	if .Value.Dict == (arrow.Array)(nil) {
		return fmt.Errorf("%s scalar doesn't have a dictionary value", )
	}

	if !arrow.TypeEqual(.Value.Dict.DataType(), .ValueType) {
		return fmt.Errorf("%s scalar's value type doesn't match dict type: got %s", , .Value.Dict.DataType())
	}

	return
}

func ( *Dictionary) () ( error) {
	if  = .Validate();  != nil {
		return
	}

	if !.Value.Index.IsValid() {
		return nil
	}

	 := .Value.Dict.Len() - 1
	switch idx := .Value.Index.value().(type) {
	case int8:
		if  < 0 || int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	case uint8:
		if int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	case int16:
		if  < 0 || int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	case uint16:
		if int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	case int32:
		if  < 0 || int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	case uint32:
		if int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	case int64:
		if  < 0 || int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	case uint64:
		if int() >  {
			 = fmt.Errorf("%s scalar index value out of bounds: %d", .DataType(), )
		}
	}

	return
}

func ( *Dictionary) () string {
	if !.Valid {
		return "null"
	}

	return .Value.Dict.String() + "[" + .Value.Index.String() + "]"
}

func ( *Dictionary) ( Scalar) bool {
	return .Value.Index.equals(.(*Dictionary).Value.Index) &&
		array.Equal(.Value.Dict, .(*Dictionary).Value.Dict)
}

func ( *Dictionary) (arrow.DataType) (Scalar, error) {
	return nil, fmt.Errorf("cast from scalar %s not implemented", .DataType())
}

func ( *Dictionary) () (Scalar, error) {
	 := .Type.(*arrow.DictionaryType)
	if !.IsValid() {
		return MakeNullScalar(.ValueType), nil
	}

	var  int
	switch .IndexType.ID() {
	case arrow.INT8:
		 = int(.Value.Index.value().(int8))
	case arrow.UINT8:
		 = int(.Value.Index.value().(uint8))
	case arrow.INT16:
		 = int(.Value.Index.value().(int16))
	case arrow.UINT16:
		 = int(.Value.Index.value().(uint16))
	case arrow.INT32:
		 = int(.Value.Index.value().(int32))
	case arrow.UINT32:
		 = int(.Value.Index.value().(uint32))
	case arrow.INT64:
		 = int(.Value.Index.value().(int64))
	case arrow.UINT64:
		 = int(.Value.Index.value().(uint64))
	default:
		return nil, fmt.Errorf("unimplemented dictionary type %s", .IndexType)
	}
	return GetScalar(.Value.Dict, )
}

func ( *Dictionary) () interface{} {
	return .Value.Index.value()
}

type Union interface {
	Scalar
	ChildValue() Scalar
	Release()
}

type SparseUnion struct {
	scalar

	TypeCode arrow.UnionTypeCode
	Value    []Scalar
	ChildID  int
}

func ( *SparseUnion) ( Scalar) bool {
	 := .(*SparseUnion)
	return Equals(.ChildValue(), .ChildValue())
}

func ( *SparseUnion) () interface{} { return .ChildValue() }

func ( *SparseUnion) () string {
	 := .Type.(*arrow.SparseUnionType)
	 := .ChildValue()
	return "union{" + .Fields()[.ChildIDs()[.TypeCode]].String() + " = " + .String() + "}"
}

func ( *SparseUnion) () {
	for ,  := range .Value {
		if ,  := .(Releasable);  {
			.Retain()
		}
	}
}

func ( *SparseUnion) () {
	for ,  := range .Value {
		if ,  := .(Releasable);  {
			.Release()
		}
	}
}

func ( *SparseUnion) () ( error) {
	 := .Type.(*arrow.SparseUnionType)
	if .NumFields() != len(.Value) {
		return fmt.Errorf("sparse union scalar value had %d fields but type has %d fields", .NumFields(), len(.Value))
	}

	if .TypeCode < 0 || int(.TypeCode) >= len(.ChildIDs()) || .ChildIDs()[.TypeCode] == arrow.InvalidUnionChildID {
		return fmt.Errorf("%s scalar has invalid type code %d", , .TypeCode)
	}

	for ,  := range .Fields() {
		 := .Value[]
		if !arrow.TypeEqual(.Type, .DataType()) {
			return fmt.Errorf("%s value for field %s had incorrect type of %s", , , .DataType())
		}
		if  = .Validate();  != nil {
			return 
		}
	}
	return
}

func ( *SparseUnion) () ( error) {
	 := .Type.(*arrow.SparseUnionType)
	if .NumFields() != len(.Value) {
		return fmt.Errorf("sparse union scalar value had %d fields but type has %d fields", .NumFields(), len(.Value))
	}

	if .TypeCode < 0 || int(.TypeCode) >= len(.ChildIDs()) || .ChildIDs()[.TypeCode] == arrow.InvalidUnionChildID {
		return fmt.Errorf("%s scalar has invalid type code %d", , .TypeCode)
	}

	for ,  := range .Fields() {
		 := .Value[]
		if !arrow.TypeEqual(.Type, .DataType()) {
			return fmt.Errorf("%s value for field %s had incorrect type of %s", , , .DataType())
		}
		if  = .ValidateFull();  != nil {
			return 
		}
	}
	return
}

func ( *SparseUnion) ( arrow.DataType) (Scalar, error) {
	if !.Valid {
		return MakeNullScalar(), nil
	}

	switch .ID() {
	case arrow.STRING:
		return NewStringScalar(.String()), nil
	case arrow.LARGE_STRING:
		return NewLargeStringScalar(.String()), nil
	}

	return nil, fmt.Errorf("cannot cast non-nil union to type other than string")
}

func ( *SparseUnion) () Scalar { return .Value[.ChildID] }

func ( []Scalar,  arrow.UnionTypeCode,  *arrow.SparseUnionType) *SparseUnion {
	 := &SparseUnion{
		scalar:   scalar{, true},
		TypeCode: ,
		Value:    ,
		ChildID:  .ChildIDs()[],
	}
	.Valid = .Value[.ChildID].IsValid()
	return 
}

func ( Scalar,  int,  *arrow.SparseUnionType) *SparseUnion {
	 := .TypeCodes()[]
	 := make([]Scalar, .NumFields())
	for ,  := range .Fields() {
		if  ==  {
			[] = 
		} else {
			[] = MakeNullScalar(.Type)
		}
	}
	return NewSparseUnionScalar(, , )
}

type DenseUnion struct {
	scalar

	TypeCode arrow.UnionTypeCode
	Value    Scalar
}

func ( *DenseUnion) ( Scalar) bool {
	 := .(*DenseUnion)
	return Equals(.Value, .Value)
}

func ( *DenseUnion) () interface{} { return .ChildValue() }

func ( *DenseUnion) () string {
	 := .Type.(*arrow.DenseUnionType)
	return "union{" + .Fields()[.ChildIDs()[.TypeCode]].String() + " = " + .Value.String() + "}"
}

func ( *DenseUnion) () {
	if ,  := .Value.(Releasable);  {
		.Retain()
	}
}

func ( *DenseUnion) () {
	if ,  := .Value.(Releasable);  {
		.Release()
	}
}

func ( *DenseUnion) () ( error) {
	 := .Type.(*arrow.DenseUnionType)
	if .TypeCode < 0 || int(.TypeCode) >= len(.ChildIDs()) || .ChildIDs()[.TypeCode] == arrow.InvalidUnionChildID {
		return fmt.Errorf("%s scalar has invalid type code %d", , .TypeCode)
	}
	 := .Fields()[.ChildIDs()[.TypeCode]].Type
	if !arrow.TypeEqual(, .Value.DataType()) {
		return fmt.Errorf("%s scalar with type code %d should have an underlying value of type %s, got %s",
			.Type, .TypeCode, , .Value.DataType())
	}
	return .Value.Validate()
}

func ( *DenseUnion) () error {
	 := .Type.(*arrow.DenseUnionType)
	if .TypeCode < 0 || int(.TypeCode) >= len(.ChildIDs()) || .ChildIDs()[.TypeCode] == arrow.InvalidUnionChildID {
		return fmt.Errorf("%s scalar has invalid type code %d", , .TypeCode)
	}
	 := .Fields()[.ChildIDs()[.TypeCode]].Type
	if !arrow.TypeEqual(, .Value.DataType()) {
		return fmt.Errorf("%s scalar with type code %d should have an underlying value of type %s, got %s",
			.Type, .TypeCode, , .Value.DataType())
	}
	return .Value.ValidateFull()
}

func ( *DenseUnion) ( arrow.DataType) (Scalar, error) {
	if !.Valid {
		return MakeNullScalar(), nil
	}

	switch .ID() {
	case arrow.STRING:
		return NewStringScalar(.String()), nil
	case arrow.LARGE_STRING:
		return NewLargeStringScalar(.String()), nil
	}

	return nil, fmt.Errorf("cannot cast non-nil union to type other than string")
}

func ( *DenseUnion) () Scalar { return .Value }

func ( Scalar,  arrow.UnionTypeCode,  *arrow.DenseUnionType) *DenseUnion {
	return &DenseUnion{scalar: scalar{, .IsValid()}, TypeCode: , Value: }
}

type RunEndEncoded struct {
	scalar

	Value Scalar
}

func ( Scalar,  *arrow.RunEndEncodedType) *RunEndEncoded {
	return &RunEndEncoded{scalar: scalar{, .IsValid()}, Value: }
}

func ( *RunEndEncoded) () {
	if ,  := .Value.(Releasable);  {
		.Release()
	}
}

func ( *RunEndEncoded) () interface{} { return .Value.value() }

func ( *RunEndEncoded) () ( error) {
	if  = .Value.Validate();  != nil {
		return
	}

	if  = validateOptional(&.scalar, .value(), "value");  != nil {
		return
	}

	if !.Valid {
		return
	}

	if .Type.ID() != arrow.RUN_END_ENCODED {
		return fmt.Errorf("%w: run-end-encoded scalar should not have type %s",
			arrow.ErrInvalid, .Type)
	}

	if !arrow.TypeEqual(.Value.DataType(), .Type.(*arrow.RunEndEncodedType).Encoded()) {
		return fmt.Errorf("%w: run-end-encoded scalar value type %s does not match type %s",
			arrow.ErrInvalid, .Value.DataType(), .Type)
	}
	return
}

func ( *RunEndEncoded) () error { return .Validate() }

func ( *RunEndEncoded) ( Scalar) bool {
	 := .(*RunEndEncoded)
	return Equals(.Value, .Value)
}

func ( *RunEndEncoded) () string {
	return .Value.String()
}

func ( *RunEndEncoded) ( arrow.DataType) (Scalar, error) {
	if !.Valid {
		return MakeNullScalar(), nil
	}

	if arrow.TypeEqual(.Type, ) {
		return , nil
	}

	if ,  := .(*arrow.RunEndEncodedType);  {
		,  := .Value.CastTo(.Encoded())
		if  != nil {
			return nil, 
		}

		return NewRunEndEncodedScalar(, ), nil
	}

	return .Value.CastTo()
}