diff --git a/sql/core/src/main/scala/org/apache/spark/sql/execution/columnar/InMemoryRelation.scala b/sql/core/src/main/scala/org/apache/spark/sql/execution/columnar/InMemoryRelation.scala index bfcbc9a06d273..eabbc7fc74f50 100644 --- a/sql/core/src/main/scala/org/apache/spark/sql/execution/columnar/InMemoryRelation.scala +++ b/sql/core/src/main/scala/org/apache/spark/sql/execution/columnar/InMemoryRelation.scala @@ -273,35 +273,22 @@ case class CachedRDDBuilder( serializer.supportsColumnarInput(cachedPlan.output) } - def cachedColumnBuffers: RDD[CachedBatch] = { + def cachedColumnBuffers: RDD[CachedBatch] = synchronized { if (_cachedColumnBuffers == null) { - synchronized { - if (_cachedColumnBuffers == null) { - _cachedColumnBuffers = buildBuffers() - } - } + _cachedColumnBuffers = buildBuffers() } _cachedColumnBuffers } - def clearCache(blocking: Boolean = false): Unit = { + def clearCache(blocking: Boolean = false): Unit = synchronized { if (_cachedColumnBuffers != null) { - synchronized { - if (_cachedColumnBuffers != null) { - _cachedColumnBuffers.unpersist(blocking) - _cachedColumnBuffers = null - } - } + _cachedColumnBuffers.unpersist(blocking) + _cachedColumnBuffers = null } } - def isCachedColumnBuffersLoaded: Boolean = { - if (_cachedColumnBuffers != null) { - synchronized { - return _cachedColumnBuffers != null && isCachedRDDLoaded - } - } - false + def isCachedColumnBuffersLoaded: Boolean = synchronized { + _cachedColumnBuffers != null && isCachedRDDLoaded } private def isCachedRDDLoaded: Boolean = {