Skip to content
This repository has been archived by the owner on Jan 28, 2021. It is now read-only.

Close iterators #655

Merged
merged 1 commit into from
Apr 4, 2019
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
7 changes: 5 additions & 2 deletions sql/analyzer/pushdown.go
Original file line number Diff line number Diff line change
Expand Up @@ -329,7 +329,10 @@ func (i *releaseIter) Next() (sql.Row, error) {
return row, nil
}

func (i *releaseIter) Close() error {
func (i *releaseIter) Close() (err error) {
i.once.Do(i.release)
return nil
if i.child != nil {
err = i.child.Close()
}
return err
}
2 changes: 2 additions & 0 deletions sql/index/pilosa/driver.go
Original file line number Diff line number Diff line change
Expand Up @@ -251,6 +251,7 @@ func (d *Driver) savePartition(
}

idx.mapping.close()
kviter.Close()
}()

for colID = offset; err == nil; colID++ {
Expand Down Expand Up @@ -344,6 +345,7 @@ func (d *Driver) Save(
return err
}

defer iter.Close()
pilosaIndex := idx.index
var rows uint64
for {
Expand Down
17 changes: 9 additions & 8 deletions sql/plan/cross_join.go
Original file line number Diff line number Diff line change
Expand Up @@ -155,17 +155,18 @@ func (i *crossJoinIterator) Next() (sql.Row, error) {
}
}

func (i *crossJoinIterator) Close() error {
if err := i.l.Close(); err != nil {
if i.r != nil {
_ = i.r.Close()
}
return err
func (i *crossJoinIterator) Close() (err error) {
if i.l != nil {
err = i.l.Close()
}

if i.r != nil {
return i.r.Close()
if err == nil {
err = i.r.Close()
juanjux marked this conversation as resolved.
Show resolved Hide resolved
} else {
i.r.Close()
}
}

return nil
return err
}
26 changes: 17 additions & 9 deletions sql/plan/exchange.go
Original file line number Diff line number Diff line change
Expand Up @@ -183,10 +183,6 @@ func (it *exchangeRowIter) iterPartitions(ch chan<- sql.Partition) {
}

close(ch)

if err := it.partitions.Close(); err != nil {
it.err <- err
}
}()

for {
Expand Down Expand Up @@ -278,13 +274,25 @@ func (it *exchangeRowIter) Next() (sql.Row, error) {
}
}

func (it *exchangeRowIter) Close() error {
if it.quit == nil {
return nil
func (it *exchangeRowIter) Close() (err error) {
if it.quit != nil {
close(it.quit)
it.quit = nil
}

// TODO(kuba): in my opinion we should close err channel here,
// but becasue we use it in another go routine, I'll leave this block commented.
//
// if it.err != nil {
// close(it.err)
// it.err = nil
// }

if it.partitions != nil {
err = it.partitions.Close()
}

close(it.quit)
return nil
return err
}

type exchangePartition struct {
Expand Down
1 change: 1 addition & 0 deletions sql/plan/exchange_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -95,6 +95,7 @@ func TestExchangePanicRecover(t *testing.T) {
it := &partitionPanic{}
ex := newExchangeRowIter(ctx, 1, it, nil)
ex.start()
it.Close()

require.True(t, it.closed)
}
Expand Down
21 changes: 12 additions & 9 deletions sql/plan/innerjoin.go
Original file line number Diff line number Diff line change
Expand Up @@ -355,19 +355,22 @@ func (i *innerJoinIter) Next() (sql.Row, error) {
}
}

func (i *innerJoinIter) Close() error {
if err := i.l.Close(); err != nil {
if i.r != nil {
_ = i.r.Close()
func (i *innerJoinIter) Close() (err error) {
i.right = nil

if i.l != nil {
if err = i.l.Close(); err != nil {
if i.r != nil {
_ = i.r.Close()
}
return err
}
return err

}

if i.r != nil {
return i.r.Close()
err = i.r.Close()
}

i.right = nil

return nil
return err
}
7 changes: 5 additions & 2 deletions sql/plan/process.go
Original file line number Diff line number Diff line change
Expand Up @@ -183,9 +183,12 @@ func (i *trackedIndexKeyValueIter) done() {
}
}

func (i *trackedIndexKeyValueIter) Close() error {
func (i *trackedIndexKeyValueIter) Close() (err error) {
i.done()
return nil
if i.iter != nil {
err = i.iter.Close()
}
return err
}

func (i *trackedIndexKeyValueIter) Next() ([]interface{}, []byte, error) {
Expand Down