|
43 | 43 | import java.util.Set; |
44 | 44 | import java.util.concurrent.ConcurrentHashMap; |
45 | 45 | import java.util.concurrent.TimeUnit; |
| 46 | +import java.util.concurrent.locks.Lock; |
| 47 | +import java.util.concurrent.locks.ReentrantLock; |
46 | 48 | import java.util.function.Predicate; |
47 | 49 | import java.util.stream.Collectors; |
48 | 50 | import java.util.stream.IntStream; |
@@ -1609,6 +1611,7 @@ private static class ReadFn<ParameterT, OutputT> extends DoFn<ParameterT, Output |
1609 | 1611 | private final int fetchSize; |
1610 | 1612 | private final boolean disableAutoCommit; |
1611 | 1613 |
|
| 1614 | + private Lock connectionLock = new ReentrantLock(); |
1612 | 1615 | private @Nullable DataSource dataSource; |
1613 | 1616 | private @Nullable Connection connection; |
1614 | 1617 | private @Nullable KV<@Nullable String, String> reportedLineage; |
@@ -1637,8 +1640,13 @@ private Connection getConnection() throws SQLException { |
1637 | 1640 | Connection connection = this.connection; |
1638 | 1641 | if (connection == null) { |
1639 | 1642 | DataSource validSource = checkStateNotNull(this.dataSource); |
1640 | | - connection = checkStateNotNull(validSource).getConnection(); |
1641 | | - this.connection = connection; |
| 1643 | + connectionLock.lock(); |
| 1644 | + try { |
| 1645 | + connection = validSource.getConnection(); |
| 1646 | + this.connection = connection; |
| 1647 | + } finally { |
| 1648 | + connectionLock.unlock(); |
| 1649 | + } |
1642 | 1650 |
|
1643 | 1651 | // report Lineage if not haven't done so |
1644 | 1652 | KV<@Nullable String, String> schemaWithTable = |
@@ -2663,6 +2671,7 @@ abstract Builder<T, V> setMaxBatchBufferingDuration( |
2663 | 2671 | Metrics.distribution(WriteFn.class, "milliseconds_per_batch"); |
2664 | 2672 |
|
2665 | 2673 | private final WriteFnSpec<T, V> spec; |
| 2674 | + private Lock connectionLock = new ReentrantLock(); |
2666 | 2675 | private @Nullable DataSource dataSource; |
2667 | 2676 | private @Nullable Connection connection; |
2668 | 2677 | private @Nullable PreparedStatement preparedStatement; |
@@ -2700,7 +2709,13 @@ private Connection getConnection() throws SQLException { |
2700 | 2709 | Connection connection = this.connection; |
2701 | 2710 | if (connection == null) { |
2702 | 2711 | DataSource validSource = checkStateNotNull(dataSource); |
2703 | | - connection = validSource.getConnection(); |
| 2712 | + connectionLock.lock(); |
| 2713 | + try { |
| 2714 | + connection = validSource.getConnection(); |
| 2715 | + } finally { |
| 2716 | + connectionLock.unlock(); |
| 2717 | + } |
| 2718 | + |
2704 | 2719 | connection.setAutoCommit(false); |
2705 | 2720 | preparedStatement = |
2706 | 2721 | connection.prepareStatement(checkStateNotNull(spec.getStatement()).get()); |
|
0 commit comments