// 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 array

import (
	
	
	
	
	
	

	
	
	
	
	
	
)

// Union is a convenience interface to encompass both Sparse and Dense
// union array types.
type Union interface {
	arrow.Array
	// NumFields returns the number of child fields in this union.
	// Equivalent to len(UnionType().Fields())
	NumFields() int
	// Validate returns an error if there are any issues with the lengths
	// or types of the children arrays mismatching with the Type of the
	// Union Array. nil is returned if there are no problems.
	Validate() error
	// ValidateFull runs the same checks that Validate() does, but additionally
	// checks that all childIDs are valid (>= 0 || ==InvalidID) and for
	// dense unions validates that all offsets are within the bounds of their
	// respective child.
	ValidateFull() error
	// TypeCodes returns the type id buffer for the union Array, equivalent to
	// Data().Buffers()[1]. Note: This will not account for any slice offset.
	TypeCodes() *memory.Buffer
	// RawTypeCodes returns a slice of UnionTypeCodes properly accounting for
	// any slice offset.
	RawTypeCodes() []arrow.UnionTypeCode
	// TypeCode returns the logical type code of the value at the requested index
	TypeCode(i int) arrow.UnionTypeCode
	// ChildID returns the index of the physical child containing the value
	// at the requested index. Equivalent to:
	//
	// 	arr.UnionType().ChildIDs()[arr.RawTypeCodes()[i+arr.Data().Offset()]]
	ChildID(i int) int
	// UnionType is a convenience function to retrieve the properly typed UnionType
	// instead of having to call DataType() and manually assert the type.
	UnionType() arrow.UnionType
	// Mode returns the union mode of the underlying Array, either arrow.SparseMode
	// or arrow.DenseMode.
	Mode() arrow.UnionMode
	// Field returns the requested child array for this union. Returns nil if a
	// nonexistent position is passed in.
	//
	// The appropriate child for an index can be retrieved with Field(ChildID(index))
	Field(pos int) arrow.Array
}

const kMaxElems = math.MaxInt32

type union struct {
	array

	unionType arrow.UnionType
	typecodes []arrow.UnionTypeCode

	children []arrow.Array
}

func ( *union) () {
	.array.Retain()
	for ,  := range .children {
		.Retain()
	}
}

func ( *union) () {
	.array.Release()
	for ,  := range .children {
		.Release()
	}
}

func ( *union) () int { return len(.unionType.Fields()) }

func ( *union) () arrow.UnionMode { return .unionType.Mode() }

func ( *union) () arrow.UnionType { return .unionType }

func ( *union) () *memory.Buffer {
	return .data.buffers[1]
}

func ( *union) () []arrow.UnionTypeCode {
	if .data.length > 0 {
		return .typecodes[.data.offset:]
	}
	return []arrow.UnionTypeCode{}
}

func ( *union) ( int) arrow.UnionTypeCode {
	return .typecodes[+.data.offset]
}

func ( *union) ( int) int {
	return .unionType.ChildIDs()[.typecodes[+.data.offset]]
}

func ( *union) ( *Data) {
	.unionType = .dtype.(arrow.UnionType)
	debug.Assert(len(.buffers) >= 2, "arrow/array: invalid number of union array buffers")

	if .length > 0 {
		.typecodes = arrow.Int8Traits.CastFromBytes(.buffers[1].Bytes())
	} else {
		.typecodes = []int8{}
	}
	.children = make([]arrow.Array, len(.childData))
	for ,  := range .childData {
		if .unionType.Mode() == arrow.SparseMode && (.offset != 0 || .Len() != .length) {
			 = NewSliceData(, int64(.offset), int64(.offset+.length))
			defer .Release()
		}
		.children[] = MakeFromData()
	}
	.array.setData()
}

func ( *union) ( int) ( arrow.Array) {
	if  < 0 ||  >= len(.children) {
		return nil
	}

	return .children[]
}

func ( *union) () error {
	 := .unionType.Fields()
	for ,  := range  {
		 := .data.childData[]
		if .unionType.Mode() == arrow.SparseMode && .Len() < .data.length+.data.offset {
			return fmt.Errorf("arrow/array: sparse union child array #%d has length smaller than expected for union array (%d < %d)",
				, .Len(), .data.length+.data.offset)
		}

		if !arrow.TypeEqual(.Type, .DataType()) {
			return fmt.Errorf("arrow/array: union child array #%d does not match type field %s vs %s",
				, .DataType(), .Type)
		}
	}
	return nil
}

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

	 := .unionType.ChildIDs()
	 := .unionType.TypeCodes()
	 := .RawTypeCodes()

	for  := 0;  < .data.length; ++ {
		 := []
		if  < 0 || [] == arrow.InvalidUnionChildID {
			return fmt.Errorf("arrow/array: union value at position %d has invalid type id %d", , )
		}
	}

	if .unionType.Mode() == arrow.DenseMode {
		// validate offsets

		// map logical typeid to child length
		var  [256]int64
		for  := range .unionType.Fields() {
			[[]] = int64(.data.childData[].Len())
		}

		// check offsets are in bounds
		var  [256]int64
		 := arrow.Int32Traits.CastFromBytes(.data.buffers[2].Bytes())[.data.offset:]
		for  := int64(0);  < int64(.data.length); ++ {
			 := []
			 := []
			switch {
			case  < 0:
				return fmt.Errorf("arrow/array: union value at position %d has negative offset %d", , )
			case  >= int32([]):
				return fmt.Errorf("arrow/array: union value at position %d has offset larger than child length (%d >= %d)",
					, , [])
			case  < int32([]):
				return fmt.Errorf("arrow/array: union value at position %d has non-monotonic offset %d", , )
			}
			[] = int64()
		}
	}

	return nil
}

// SparseUnion represents an array where each logical value is taken from
// a single child. A buffer of 8-bit type ids indicates which child a given
// logical value is to be taken from. This is represented as the ChildID,
// which is the index into the list of children.
//
// In a sparse union, each child array will have the same length as the
// union array itself, regardless of how many values in the union actually
// refer to it.
//
// Unlike most other arrays, unions do not have a top-level validity bitmap.
type SparseUnion struct {
	union
}

// NewSparseUnion constructs a union array using the given type, length, list of
// children and buffer of typeIDs with the given offset.
func ( *arrow.SparseUnionType,  int,  []arrow.Array,  *memory.Buffer,  int) *SparseUnion {
	 := make([]arrow.ArrayData, len())
	for ,  := range  {
		[] = .Data()
	}
	 := NewData(, , []*memory.Buffer{nil, }, , 0, )
	defer .Release()
	return NewSparseUnionData()
}

// NewSparseUnionData constructs a SparseUnion array from the given ArrayData object.
func ( arrow.ArrayData) *SparseUnion {
	 := &SparseUnion{}
	.refCount.Add(1)
	.setData(.(*Data))
	return 
}

// NewSparseUnionFromArrays constructs a new SparseUnion array with the provided
// values.
//
// typeIDs *must* be an INT8 array with no nulls
// len(codes) *must* be either 0 or equal to len(children). If len(codes) is 0,
// the type codes used will be sequentially numeric starting at 0.
func ( arrow.Array,  []arrow.Array,  ...arrow.UnionTypeCode) (*SparseUnion, error) {
	return NewSparseUnionFromArraysWithFieldCodes(, , []string{}, )
}

// NewSparseUnionFromArrayWithFields constructs a new SparseUnion array like
// NewSparseUnionFromArrays, but allows specifying the field names. Type codes
// will be auto-generated sequentially starting at 0.
//
// typeIDs *must* be an INT8 array with no nulls.
// len(fields) *must* either be 0 or equal to len(children). If len(fields) is 0,
// then the fields will be named sequentially starting at "0".
func ( arrow.Array,  []arrow.Array,  []string) (*SparseUnion, error) {
	return NewSparseUnionFromArraysWithFieldCodes(, , , []arrow.UnionTypeCode{})
}

// NewSparseUnionFromArraysWithFieldCodes combines the other constructors
// for constructing a new SparseUnion array with the provided field names
// and type codes, along with children and type ids.
//
// All the requirements mentioned in NewSparseUnionFromArrays and
// NewSparseUnionFromArraysWithFields apply.
func ( arrow.Array,  []arrow.Array,  []string,  []arrow.UnionTypeCode) (*SparseUnion, error) {
	switch {
	case .DataType().ID() != arrow.INT8:
		return nil, errors.New("arrow/array: union array type ids must be signed int8")
	case .NullN() != 0:
		return nil, errors.New("arrow/array: union type ids may not have nulls")
	case len() > 0 && len() != len():
		return nil, errors.New("arrow/array: field names must have the same length as children")
	case len() > 0 && len() != len():
		return nil, errors.New("arrow/array: type codes must have same length as children")
	}

	 := []*memory.Buffer{nil, .Data().Buffers()[1]}
	 := arrow.SparseUnionFromArrays(, , )

	 := make([]arrow.ArrayData, len())
	for ,  := range  {
		[] = .Data()
		if .Len() != .Len() {
			return nil, errors.New("arrow/array: sparse union array must have len(child) == len(typeids) for all children")
		}
	}

	 := NewData(, .Len(), , , 0, .Data().Offset())
	defer .Release()
	return NewSparseUnionData(), nil
}

func ( *SparseUnion) ( *Data) {
	.union.setData()
	debug.Assert(.data.dtype.ID() == arrow.SPARSE_UNION, "arrow/array: invalid data type for SparseUnion")
	debug.Assert(len(.data.buffers) == 2, "arrow/array: sparse unions should have exactly 2 buffers")
	debug.Assert(.data.buffers[0] == nil, "arrow/array: validity bitmap for sparse unions should be nil")
}

func ( *SparseUnion) ( int) interface{} {
	 := .RawTypeCodes()[]

	 := .ChildID()
	 := .Field()

	if .IsNull() {
		return nil
	}

	return []interface{}{, .GetOneForMarshal()}
}

func ( *SparseUnion) () ([]byte, error) {
	var  bytes.Buffer
	 := json.NewEncoder(&)

	.WriteByte('[')
	for  := 0;  < .Len(); ++ {
		if  != 0 {
			.WriteByte(',')
		}
		if  := .Encode(.GetOneForMarshal());  != nil {
			return nil, 
		}
	}
	.WriteByte(']')
	return .Bytes(), nil
}

func ( *SparseUnion) ( int) string {
	if .IsNull() {
		return NullValueStr
	}

	 := .GetOneForMarshal()
	if  == nil {
		// child is nil
		return NullValueStr
	}

	,  := json.Marshal()
	if  != nil {
		panic()
	}
	return string()
}

func ( *SparseUnion) () string {
	var  strings.Builder
	.WriteByte('[')

	 := .unionType.Fields()
	for  := 0;  < .Len(); ++ {
		if  > 0 {
			.WriteString(" ")
		}

		 := [.ChildID()]
		 := .Field(.ChildID())
		fmt.Fprintf(&, "{%s=%v}", .Name, .GetOneForMarshal())
	}
	.WriteByte(']')
	return .String()
}

// GetFlattenedField returns a child array, adjusting its validity bitmap
// where the union array type codes don't match.
//
// ie: the returned array will have a null in every index that it is
// not referenced by union.
func ( *SparseUnion) ( memory.Allocator,  int) (arrow.Array, error) {
	if  < 0 ||  >= .NumFields() {
		return nil, fmt.Errorf("arrow/array: index out of range: %d", )
	}

	 := .data.childData[]
	if .data.offset != 0 || .data.length != .Len() {
		 = NewSliceData(, int64(.data.offset), int64(.data.offset+.data.length))
		// NewSliceData doesn't break the slice reference for buffers
		// since we're going to replace the null bitmap buffer we need to break the
		// slice reference so that we don't affect a.children's references
		 := make([]*memory.Buffer, len(.Buffers()))
		copy(, .(*Data).buffers)
		.(*Data).buffers = 
	} else {
		 = .(*Data).Copy()
	}
	defer .Release()

	// synthesize a null bitmap based on the union discriminant
	// make sure the bitmap has extra bits corresponding to the child's offset
	 := memory.NewResizableBuffer()
	.Resize(.Len() + .Offset())

	var (
		       = .Buffers()[0]
		           = .Offset()
		              = .unionType.TypeCodes()[]
		                 = .RawTypeCodes()
		          int64 = 0
	)
	bitutils.GenerateBitsUnrolled(.Bytes(), int64(), int64(.data.length),
		func() bool {
			 := [] == 
			++
			return 
		})

	if  != nil {
		defer .Release()
		bitutil.BitmapAnd(.Bytes(), .Bytes(),
			int64(), int64(), .Bytes(),
			int64(), int64(.Len()))
	}
	.(*Data).buffers[0] = 
	.(*Data).nulls = .Len() - bitutil.CountSetBits(.Bytes(), , .Len())
	return MakeFromData(), nil
}

func arraySparseUnionEqual(,  *SparseUnion) bool {
	 := .unionType.ChildIDs()
	,  := .RawTypeCodes(), .RawTypeCodes()

	for  := 0;  < .data.length; ++ {
		 := []
		if  != [] {
			return false
		}

		 := []
		 := SliceEqual(.children[], int64(), int64(+1),
			.children[], int64(), int64(+1))
		if ! {
			return false
		}
	}
	return true
}

func arraySparseUnionApproxEqual(,  *SparseUnion,  equalOption) bool {
	 := .unionType.ChildIDs()
	,  := .RawTypeCodes(), .RawTypeCodes()

	for  := 0;  < .data.length; ++ {
		 := []
		if  != [] {
			return false
		}

		 := []
		 := sliceApproxEqual(.children[], int64(+.data.offset), int64(+.data.offset+1),
			.children[], int64(+.data.offset), int64(+.data.offset+1), )
		if ! {
			return false
		}
	}
	return true
}

// DenseUnion represents an array where each logical value is taken from
// a single child, at a specific offset. A buffer of 8-bit type ids
// indicates which child a given logical value is to be taken from and
// a buffer of 32-bit offsets indicating which physical position in the
// given child array has the logical value for that index.
//
// Unlike a sparse union, a dense union allows encoding only the child values
// which are actually referred to by the union array. This is counterbalanced
// by the additional footprint of the offsets buffer, and the additional
// indirection cost when looking up values.
//
// Unlike most other arrays, unions do not have a top-level validity bitmap.
type DenseUnion struct {
	union
	offsets []int32
}

// NewDenseUnion constructs a union array using the given type, length, list of
// children and buffers of typeIDs and offsets, with the given array offset.
func ( *arrow.DenseUnionType,  int,  []arrow.Array, ,  *memory.Buffer,  int) *DenseUnion {
	 := make([]arrow.ArrayData, len())
	for ,  := range  {
		[] = .Data()
	}

	 := NewData(, , []*memory.Buffer{nil, , }, , 0, )
	defer .Release()
	return NewDenseUnionData()
}

// NewDenseUnionData constructs a DenseUnion array from the given ArrayData object.
func ( arrow.ArrayData) *DenseUnion {
	 := &DenseUnion{}
	.refCount.Add(1)
	.setData(.(*Data))
	return 
}

// NewDenseUnionFromArrays constructs a new DenseUnion array with the provided
// values.
//
// typeIDs *must* be an INT8 array with no nulls
// offsets *must* be an INT32 array with no nulls
// len(codes) *must* be either 0 or equal to len(children). If len(codes) is 0,
// the type codes used will be sequentially numeric starting at 0.
func (,  arrow.Array,  []arrow.Array,  ...arrow.UnionTypeCode) (*DenseUnion, error) {
	return NewDenseUnionFromArraysWithFieldCodes(, , , []string{}, )
}

// NewDenseUnionFromArrayWithFields constructs a new DenseUnion array like
// NewDenseUnionFromArrays, but allows specifying the field names. Type codes
// will be auto-generated sequentially starting at 0.
//
// typeIDs *must* be an INT8 array with no nulls.
// offsets *must* be an INT32 array with no nulls.
// len(fields) *must* either be 0 or equal to len(children). If len(fields) is 0,
// then the fields will be named sequentially starting at "0".
func (,  arrow.Array,  []arrow.Array,  []string) (*DenseUnion, error) {
	return NewDenseUnionFromArraysWithFieldCodes(, , , , []arrow.UnionTypeCode{})
}

// NewDenseUnionFromArraysWithFieldCodes combines the other constructors
// for constructing a new DenseUnion array with the provided field names
// and type codes, along with children and type ids.
//
// All the requirements mentioned in NewDenseUnionFromArrays and
// NewDenseUnionFromArraysWithFields apply.
func (,  arrow.Array,  []arrow.Array,  []string,  []arrow.UnionTypeCode) (*DenseUnion, error) {
	switch {
	case .DataType().ID() != arrow.INT32:
		return nil, errors.New("arrow/array: union offsets must be signed int32")
	case .DataType().ID() != arrow.INT8:
		return nil, errors.New("arrow/array: union type_ids must be signed int8")
	case .NullN() != 0:
		return nil, errors.New("arrow/array: union typeIDs may not have nulls")
	case .NullN() != 0:
		return nil, errors.New("arrow/array: nulls are not allowed in offsets for NewDenseUnionFromArrays*")
	case len() > 0 && len() != len():
		return nil, errors.New("arrow/array: fields must be the same length as children")
	case len() > 0 && len() != len():
		return nil, errors.New("arrow/array: typecodes must have the same length as children")
	}

	 := arrow.DenseUnionFromArrays(, , )
	 := []*memory.Buffer{nil, .Data().Buffers()[1], .Data().Buffers()[1]}

	 := make([]arrow.ArrayData, len())
	for ,  := range  {
		[] = .Data()
	}

	 := NewData(, .Len(), , , 0, .Data().Offset())
	defer .Release()
	return NewDenseUnionData(), nil
}

func ( *DenseUnion) () *memory.Buffer { return .data.buffers[2] }

func ( *DenseUnion) ( int) int32 { return .offsets[+.data.offset] }

func ( *DenseUnion) () []int32 { return .offsets[.data.offset:] }

func ( *DenseUnion) ( *Data) {
	.union.setData()
	debug.Assert(.data.dtype.ID() == arrow.DENSE_UNION, "arrow/array: invalid data type for DenseUnion")
	debug.Assert(len(.data.buffers) == 3, "arrow/array: dense unions should have exactly 3 buffers")
	debug.Assert(.data.buffers[0] == nil, "arrow/array: validity bitmap for dense unions should be nil")

	if .length > 0 {
		.offsets = arrow.Int32Traits.CastFromBytes(.data.buffers[2].Bytes())
	} else {
		.offsets = []int32{}
	}
}

func ( *DenseUnion) ( int) interface{} {
	 := .RawTypeCodes()[]

	 := .ChildID()
	 := .Field()

	 := int(.RawValueOffsets()[])
	if .IsNull() {
		return nil
	}

	return []interface{}{, .GetOneForMarshal()}
}

func ( *DenseUnion) () ([]byte, error) {
	var  bytes.Buffer
	 := json.NewEncoder(&)

	.WriteByte('[')
	for  := 0;  < .Len(); ++ {
		if  != 0 {
			.WriteByte(',')
		}
		if  := .Encode(.GetOneForMarshal());  != nil {
			return nil, 
		}
	}
	.WriteByte(']')
	return .Bytes(), nil
}

func ( *DenseUnion) ( int) string {
	if .IsNull() {
		return NullValueStr
	}

	 := .GetOneForMarshal()
	if  == nil {
		// child in nil
		return NullValueStr
	}

	,  := json.Marshal()
	if  != nil {
		panic()
	}
	return string()
}

func ( *DenseUnion) () string {
	var  strings.Builder
	.WriteByte('[')

	 := .RawValueOffsets()

	 := .unionType.Fields()
	for  := 0;  < .Len(); ++ {
		if  > 0 {
			.WriteString(" ")
		}

		 := [.ChildID()]
		 := .Field(.ChildID())
		fmt.Fprintf(&, "{%s=%v}", .Name, .GetOneForMarshal(int([])))
	}
	.WriteByte(']')
	return .String()
}

func arrayDenseUnionEqual(,  *DenseUnion) bool {
	 := .unionType.ChildIDs()
	,  := .RawTypeCodes(), .RawTypeCodes()
	,  := .RawValueOffsets(), .RawValueOffsets()

	for  := 0;  < .data.length; ++ {
		 := []
		if  != [] {
			return false
		}

		 := []
		 := SliceEqual(.children[], int64([]), int64([]+1),
			.children[], int64([]), int64([]+1))
		if ! {
			return false
		}
	}
	return true
}

func arrayDenseUnionApproxEqual(,  *DenseUnion,  equalOption) bool {
	 := .unionType.ChildIDs()
	,  := .RawTypeCodes(), .RawTypeCodes()
	,  := .RawValueOffsets(), .RawValueOffsets()

	for  := 0;  < .data.length; ++ {
		 := []
		if  != [] {
			return false
		}

		 := []
		 := sliceApproxEqual(.children[], int64([]), int64([]+1),
			.children[], int64([]), int64([]+1), )
		if ! {
			return false
		}
	}
	return true
}

// UnionBuilder is a convenience interface for building Union arrays of
// either Dense or Sparse mode.
type UnionBuilder interface {
	Builder
	// AppendChild allows constructing the union type on the fly by making new
	// new array builder available to the union builder. The type code (index)
	// of the new child is returned, which should be passed to the Append method
	// when adding a new element to the union array.
	AppendChild(newChild Builder, fieldName string) (newCode arrow.UnionTypeCode)
	// Append adds an element to the UnionArray indicating which typecode the
	// new element should use. This *must* be followed up by an append to the
	// appropriate child builder.
	Append(arrow.UnionTypeCode)
	// Mode returns what kind of Union is being built, either arrow.SparseMode
	// or arrow.DenseMode
	Mode() arrow.UnionMode
	// Child returns the builder for the requested child index.
	// If an invalid index is requested (e.g. <0 or >len(children))
	// then this will panic.
	Child(idx int) Builder
}

type unionBuilder struct {
	builder

	childFields []arrow.Field
	codes       []arrow.UnionTypeCode
	mode        arrow.UnionMode

	children        []Builder
	typeIDtoBuilder []Builder
	typeIDtoChildID []int
	// for all typeID < denseTypeID, typeIDtoBuilder[typeID] != nil
	denseTypeID  arrow.UnionTypeCode
	typesBuilder *int8BufferBuilder
}

func newUnionBuilder( memory.Allocator,  []Builder,  arrow.UnionType) *unionBuilder {
	if  == nil {
		 = make([]Builder, 0)
	}
	 := unionBuilder{
		builder:         builder{mem: },
		mode:            .Mode(),
		codes:           .TypeCodes(),
		children:        ,
		typeIDtoChildID: make([]int, int(.MaxTypeCode())+1),     // convert to int as int8(127) +1 panics
		typeIDtoBuilder: make([]Builder, int(.MaxTypeCode())+1), // convert to int as int8(127) +1 panics
		childFields:     make([]arrow.Field, len()),
		typesBuilder:    newInt8BufferBuilder(),
	}
	.refCount.Add(1)

	.typeIDtoChildID[0] = arrow.InvalidUnionChildID
	for  := 1;  < len(.typeIDtoChildID);  *= 2 {
		copy(.typeIDtoChildID[:], .typeIDtoChildID[:])
	}

	debug.Assert(len() == len(.TypeCodes()), "mismatched typecodes and children")
	debug.Assert(len(.typeIDtoBuilder)-1 <= int(arrow.MaxUnionTypeCode), "too many typeids")

	copy(.childFields, .Fields())
	for ,  := range  {
		.Retain()
		 := .TypeCodes()[]
		.typeIDtoChildID[] = 
		.typeIDtoBuilder[] = 
	}

	return &
}

func ( *unionBuilder) () int {
	return len(.children)
}

func ( *unionBuilder) ( int) Builder {
	if  < 0 ||  > len(.children) {
		panic("arrow/array: invalid child index for union builder")
	}
	return .children[]
}

// Len returns the current number of elements in the builder.
func ( *unionBuilder) () int { return .typesBuilder.Len() }

func ( *unionBuilder) () arrow.UnionMode { return .mode }

func ( *unionBuilder) ( int,  func(int)) {
	// union has no null bitmap, ever so we can skip that handling
	if .length+ > .capacity {
		.capacity = bitutil.NextPowerOf2(.length + )
		(.capacity)
	}
}

func ( *unionBuilder) () {
	debug.Assert(.refCount.Load() > 0, "too many releases")

	if .refCount.Add(-1) == 0 {
		for ,  := range .children {
			.Release()
		}
		.typesBuilder.Release()
	}
}

func ( *unionBuilder) () arrow.DataType {
	 := make([]arrow.Field, len(.childFields))
	for ,  := range .childFields {
		[] = 
		[].Type = .children[].Type()
	}

	switch .mode {
	case arrow.SparseMode:
		return arrow.SparseUnionOf(, .codes)
	case arrow.DenseMode:
		return arrow.DenseUnionOf(, .codes)
	default:
		panic("invalid union builder mode")
	}
}

func ( *unionBuilder) ( Builder,  string) arrow.UnionTypeCode {
	.Retain()
	.children = append(.children, )
	 := .nextTypeID()

	.typeIDtoChildID[] = len(.children) - 1
	.typeIDtoBuilder[] = 
	.childFields = append(.childFields, arrow.Field{Name: , Nullable: true})
	.codes = append(.codes, )

	return 
}

func ( *unionBuilder) () arrow.UnionTypeCode {
	// find typeID such that typeIDtoBuilder[typeID] == nil
	// use that for the new child. Start searching at denseTypeID
	// since typeIDtoBuilder is densely packed up at least to denseTypeID
	for ; int(.denseTypeID) < len(.typeIDtoBuilder); .denseTypeID++ {
		if .typeIDtoBuilder[.denseTypeID] == nil {
			 := .denseTypeID
			.denseTypeID++
			return 
		}
	}

	debug.Assert(len(.typeIDtoBuilder) < int(arrow.MaxUnionTypeCode), "too many children typeids")
	// typeIDtoBuilder is already densely packed, so just append the new child
	.typeIDtoBuilder = append(.typeIDtoBuilder, nil)
	.typeIDtoChildID = append(.typeIDtoChildID, arrow.InvalidUnionChildID)
	 := .denseTypeID
	.denseTypeID++
	return 
}

func ( *unionBuilder) () *Data {
	 := .typesBuilder.Len()
	 := .typesBuilder.Finish()
	defer .Release()
	 := make([]arrow.ArrayData, len(.children))
	for ,  := range .children {
		[] = .newData()
		defer [].Release()
	}

	return NewData(.Type(), , []*memory.Buffer{nil, }, , 0, 0)
}

// SparseUnionBuilder is used to build a Sparse Union array using the Append
// methods. You can also add new types to the union on the fly by using
// AppendChild.
//
// Keep in mind: All children of a SparseUnion should be the same length
// as the union itself. If you add new children with AppendChild, ensure
// that they have the correct number of preceding elements that have been
// added to the builder beforehand.
type SparseUnionBuilder struct {
	*unionBuilder
}

// NewEmptySparseUnionBuilder is a helper to construct a SparseUnionBuilder
// without having to predefine the union types. It creates a builder with no
// children and AppendChild will have to be called before appending any
// elements to this builder.
func ( memory.Allocator) *SparseUnionBuilder {
	return &SparseUnionBuilder{
		unionBuilder: newUnionBuilder(, nil, arrow.SparseUnionOf([]arrow.Field{}, []arrow.UnionTypeCode{})),
	}
}

// NewSparseUnionBuilder constructs a new SparseUnionBuilder with the provided
// children and type codes. Builders will be constructed for each child
// using the fields in typ
func ( memory.Allocator,  *arrow.SparseUnionType) *SparseUnionBuilder {
	 := make([]Builder, .NumFields())
	for ,  := range .Fields() {
		[] = NewBuilder(, .Type)
		defer [].Release()
	}
	return NewSparseUnionBuilderWithBuilders(, , )
}

// NewSparseUnionWithBuilders returns a new SparseUnionBuilder using the
// provided type and builders.
func ( memory.Allocator,  *arrow.SparseUnionType,  []Builder) *SparseUnionBuilder {
	return &SparseUnionBuilder{
		unionBuilder: newUnionBuilder(, , ),
	}
}

func ( *SparseUnionBuilder) ( int) {
	.reserve(, .Resize)
}

func ( *SparseUnionBuilder) ( int) {
	.typesBuilder.resize()
}

// AppendNull will append a null to the first child and an empty value
// (implementation-defined) to the rest of the children.
func ( *SparseUnionBuilder) () {
	 := .codes[0]
	.typesBuilder.AppendValue()
	.typeIDtoBuilder[].AppendNull()
	for ,  := range .codes[1:] {
		.typeIDtoBuilder[].AppendEmptyValue()
	}
}

// AppendNulls is identical to calling AppendNull() n times, except
// it will pre-allocate with reserve for all the nulls beforehand.
func ( *SparseUnionBuilder) ( int) {
	 := .codes[0]
	.Reserve()
	for ,  := range .codes {
		.typeIDtoBuilder[].Reserve()
	}
	for  := 0;  < ; ++ {
		.typesBuilder.AppendValue()
		.typeIDtoBuilder[].AppendNull()
		for ,  := range .codes[1:] {
			.typeIDtoBuilder[].AppendEmptyValue()
		}
	}
}

// AppendEmptyValue appends an empty value (implementation defined)
// to each child, and appends the type of the first typecode to the typeid
// buffer.
func ( *SparseUnionBuilder) () {
	.typesBuilder.AppendValue(.codes[0])
	for ,  := range .codes {
		.typeIDtoBuilder[].AppendEmptyValue()
	}
}

// AppendEmptyValues is identical to calling AppendEmptyValue() n times,
// except it pre-allocates first so it is more efficient.
func ( *SparseUnionBuilder) ( int) {
	.Reserve()
	 := .codes[0]
	for ,  := range .codes {
		.typeIDtoBuilder[].Reserve()
	}
	for  := 0;  < ; ++ {
		.typesBuilder.AppendValue()
		for ,  := range .codes {
			.typeIDtoBuilder[].AppendEmptyValue()
		}
	}
}

// Append appends an element to the UnionArray and must be followed up
// by an append to the appropriate child builder. The parameter should
// be the type id of the child to which the next value will be appended.
//
// After appending to the corresponding child builder, all other child
// builders should have a null or empty value appended to them (although
// this is not enforced and any value is theoretically allowed and will be
// ignored).
func ( *SparseUnionBuilder) ( arrow.UnionTypeCode) {
	.typesBuilder.AppendValue()
}

func ( *SparseUnionBuilder) () arrow.Array {
	return .NewSparseUnionArray()
}

func ( *SparseUnionBuilder) () ( *SparseUnion) {
	 := .newData()
	 = NewSparseUnionData()
	.Release()
	return
}

func ( *SparseUnionBuilder) ( []byte) ( error) {
	 := json.NewDecoder(bytes.NewReader())
	,  := .Token()
	if  != nil {
		return 
	}

	if ,  := .(json.Delim); ! ||  != '[' {
		return fmt.Errorf("sparse union builder must unpack from json array, found %s", )
	}
	return .Unmarshal()
}

func ( *SparseUnionBuilder) ( *json.Decoder) error {
	for .More() {
		if  := .UnmarshalOne();  != nil {
			return 
		}
	}
	return nil
}

func ( *SparseUnionBuilder) ( string) error {
	if  == NullValueStr {
		.AppendNull()
		return nil
	}
	 := json.NewDecoder(strings.NewReader())
	return .UnmarshalOne()
}

func ( *SparseUnionBuilder) ( *json.Decoder) error {
	,  := .Token()
	if  != nil {
		return 
	}

	switch  {
	case json.Delim('['):
		// should be [type_id, Value]
		,  := .Token()
		if  != nil {
			return 
		}

		var  int8

		switch tid := .(type) {
		case json.Number:
			,  := .Int64()
			if  != nil {
				return 
			}
			 = int8()
		case float64:
			if  != float64(int64()) {
				return &json.UnmarshalTypeError{
					Offset: .InputOffset(),
					Type:   reflect.TypeOf(int8(0)),
					Struct: fmt.Sprint(.Type()),
					Value:  "float",
				}
			}
			 = int8()
		}

		 := .typeIDtoChildID[]
		if  == arrow.InvalidUnionChildID {
			return &json.UnmarshalTypeError{
				Offset: .InputOffset(),
				Value:  "invalid type code",
			}
		}

		for ,  := range .children {
			if  !=  {
				.AppendNull()
			}
		}

		.Append()
		if  := .children[].UnmarshalOne();  != nil {
			return 
		}

		,  := .Token()
		if  != nil {
			return 
		}

		if  != json.Delim(']') {
			return &json.UnmarshalTypeError{
				Offset: .InputOffset(),
				Value:  "union value array should have exactly 2 elements",
			}
		}
	case nil:
		.AppendNull()
	default:
		return &json.UnmarshalTypeError{
			Offset: .InputOffset(),
			Value:  fmt.Sprint(),
			Struct: fmt.Sprint(.Type()),
		}
	}
	return nil
}

// DenseUnionBuilder is used to build a Dense Union array using the Append
// methods. You can also add new types to the union on the fly by using
// AppendChild.
type DenseUnionBuilder struct {
	*unionBuilder

	offsetsBuilder *int32BufferBuilder
}

// NewEmptyDenseUnionBuilder is a helper to construct a DenseUnionBuilder
// without having to predefine the union types. It creates a builder with no
// children and AppendChild will have to be called before appending any
// elements to this builder.
func ( memory.Allocator) *DenseUnionBuilder {
	return &DenseUnionBuilder{
		unionBuilder:   newUnionBuilder(, nil, arrow.DenseUnionOf([]arrow.Field{}, []arrow.UnionTypeCode{})),
		offsetsBuilder: newInt32BufferBuilder(),
	}
}

// NewDenseUnionBuilder constructs a new DenseUnionBuilder with the provided
// children and type codes. Builders will be constructed for each child
// using the fields in typ
func ( memory.Allocator,  *arrow.DenseUnionType) *DenseUnionBuilder {
	 := make([]Builder, 0, .NumFields())
	defer func() {
		for ,  := range  {
			.Release()
		}
	}()

	for ,  := range .Fields() {
		 = append(, NewBuilder(, .Type))
	}
	return NewDenseUnionBuilderWithBuilders(, , )
}

// NewDenseUnionWithBuilders returns a new DenseUnionBuilder using the
// provided type and builders.
func ( memory.Allocator,  *arrow.DenseUnionType,  []Builder) *DenseUnionBuilder {
	return &DenseUnionBuilder{
		unionBuilder:   newUnionBuilder(, , ),
		offsetsBuilder: newInt32BufferBuilder(),
	}
}

func ( *DenseUnionBuilder) ( int) {
	.reserve(, .Resize)
}

func ( *DenseUnionBuilder) ( int) {
	.typesBuilder.resize()
	.offsetsBuilder.resize( * arrow.Int32SizeBytes)
}

// AppendNull will only append a null value arbitrarily to the first child
// and use that offset for this element of the array.
func ( *DenseUnionBuilder) () {
	 := .codes[0]
	 := .typeIDtoBuilder[]
	.typesBuilder.AppendValue()
	.offsetsBuilder.AppendValue(int32(.Len()))
	.AppendNull()
}

// AppendNulls will only append a single null arbitrarily to the first child
// and use the same offset multiple times to point to it. The result is that
// for a DenseUnion this is more efficient than calling AppendNull multiple
// times in a loop
func ( *DenseUnionBuilder) ( int) {
	// only append 1 null to the child builder, use the same offset twice
	 := .codes[0]
	 := .typeIDtoBuilder[]
	.Reserve()
	for  := 0;  < ; ++ {
		.typesBuilder.AppendValue()
		.offsetsBuilder.AppendValue(int32(.Len()))
	}
	// only append a single null to the child builder, the offsets all refer to the same value
	.AppendNull()
}

// AppendEmptyValue only appends an empty value arbitrarily to the first child,
// and then uses that offset to identify the value.
func ( *DenseUnionBuilder) () {
	 := .codes[0]
	 := .typeIDtoBuilder[]
	.typesBuilder.AppendValue()
	.offsetsBuilder.AppendValue(int32(.Len()))
	.AppendEmptyValue()
}

// AppendEmptyValues, like AppendNulls, will only append a single empty value
// (implementation defined) to the first child arbitrarily, and then point
// at that value using the offsets n times. That makes this more efficient
// than calling AppendEmptyValue multiple times.
func ( *DenseUnionBuilder) ( int) {
	// only append 1 null to the child builder, use the same offset twice
	 := .codes[0]
	 := .typeIDtoBuilder[]
	.Reserve()
	for  := 0;  < ; ++ {
		.typesBuilder.AppendValue()
		.offsetsBuilder.AppendValue(int32(.Len()))
	}
	// only append a single empty value to the child builder, the offsets all
	// refer to the same value
	.AppendEmptyValue()
}

// Append appends the necessary offset and type code to the builder
// and must be followed up with an append to the appropriate child builder
func ( *DenseUnionBuilder) ( arrow.UnionTypeCode) {
	.typesBuilder.AppendValue()
	 := .typeIDtoBuilder[]
	if .Len() == kMaxElems {
		panic("a dense UnionArray cannot contain more than 2^31 - 1 elements from a single child")
	}

	.offsetsBuilder.AppendValue(int32(.Len()))
}

func ( *DenseUnionBuilder) () {
	debug.Assert(.refCount.Load() > 0, "too many releases")

	if .refCount.Add(-1) == 0 {
		for ,  := range .children {
			.Release()
		}
		.typesBuilder.Release()
		.offsetsBuilder.Release()
	}
}

func ( *DenseUnionBuilder) () *Data {
	 := .unionBuilder.newData()
	.buffers = append(.buffers, .offsetsBuilder.Finish())
	return 
}

func ( *DenseUnionBuilder) () arrow.Array {
	return .NewDenseUnionArray()
}

func ( *DenseUnionBuilder) () ( *DenseUnion) {
	 := .newData()
	 = NewDenseUnionData()
	.Release()
	return
}

func ( *DenseUnionBuilder) ( []byte) ( error) {
	 := json.NewDecoder(bytes.NewReader())
	,  := .Token()
	if  != nil {
		return 
	}

	if ,  := .(json.Delim); ! ||  != '[' {
		return fmt.Errorf("dense union builder must unpack from json array, found %s", )
	}
	return .Unmarshal()
}

func ( *DenseUnionBuilder) ( *json.Decoder) error {
	for .More() {
		if  := .UnmarshalOne();  != nil {
			return 
		}
	}
	return nil
}

func ( *DenseUnionBuilder) ( string) error {
	if  == NullValueStr {
		.AppendNull()
		return nil
	}
	 := json.NewDecoder(strings.NewReader())
	return .UnmarshalOne()
}

func ( *DenseUnionBuilder) ( *json.Decoder) error {
	,  := .Token()
	if  != nil {
		return 
	}

	switch  {
	case json.Delim('['):
		// should be [type_id, Value]
		,  := .Token()
		if  != nil {
			return 
		}

		var  int8

		switch tid := .(type) {
		case json.Number:
			,  := .Int64()
			if  != nil {
				return 
			}
			 = int8()
		case float64:
			if  != float64(int64()) {
				return &json.UnmarshalTypeError{
					Offset: .InputOffset(),
					Type:   reflect.TypeOf(int8(0)),
					Struct: fmt.Sprint(.Type()),
					Value:  "float",
				}
			}
			 = int8()
		}

		 := .typeIDtoChildID[]
		if  == arrow.InvalidUnionChildID {
			return &json.UnmarshalTypeError{
				Offset: .InputOffset(),
				Value:  "invalid type code",
			}
		}

		.Append()
		if  := .children[].UnmarshalOne();  != nil {
			return 
		}

		,  := .Token()
		if  != nil {
			return 
		}

		if  != json.Delim(']') {
			return &json.UnmarshalTypeError{
				Offset: .InputOffset(),
				Value:  "union value array should have exactly 2 elements",
			}
		}
	case nil:
		.AppendNull()
	default:
		return &json.UnmarshalTypeError{
			Offset: .InputOffset(),
			Value:  fmt.Sprint(),
			Struct: fmt.Sprint(.Type()),
		}
	}
	return nil
}

var (
	_ arrow.Array  = (*SparseUnion)(nil)
	_ arrow.Array  = (*DenseUnion)(nil)
	_ Union        = (*SparseUnion)(nil)
	_ Union        = (*DenseUnion)(nil)
	_ Builder      = (*SparseUnionBuilder)(nil)
	_ Builder      = (*DenseUnionBuilder)(nil)
	_ UnionBuilder = (*SparseUnionBuilder)(nil)
	_ UnionBuilder = (*DenseUnionBuilder)(nil)
)